Storage.java
上传用户:quxuerui
上传日期:2018-01-08
资源大小:41811k
文件大小:25k
源码类别:

网格计算

开发平台:

Java

  1. /**
  2.  * Licensed to the Apache Software Foundation (ASF) under one
  3.  * or more contributor license agreements.  See the NOTICE file
  4.  * distributed with this work for additional information
  5.  * regarding copyright ownership.  The ASF licenses this file
  6.  * to you under the Apache License, Version 2.0 (the
  7.  * "License"); you may not use this file except in compliance
  8.  * with the License.  You may obtain a copy of the License at
  9.  *
  10.  *     http://www.apache.org/licenses/LICENSE-2.0
  11.  *
  12.  * Unless required by applicable law or agreed to in writing, software
  13.  * distributed under the License is distributed on an "AS IS" BASIS,
  14.  * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
  15.  * See the License for the specific language governing permissions and
  16.  * limitations under the License.
  17.  */
  18. package org.apache.hadoop.hdfs.server.common;
  19. import java.io.File;
  20. import java.io.FileInputStream;
  21. import java.io.FileOutputStream;
  22. import java.io.IOException;
  23. import java.io.RandomAccessFile;
  24. import java.nio.channels.FileLock;
  25. import java.nio.channels.OverlappingFileLockException;
  26. import java.util.ArrayList;
  27. import java.util.List;
  28. import java.util.Iterator;
  29. import java.util.Properties;
  30. import org.apache.commons.logging.Log;
  31. import org.apache.commons.logging.LogFactory;
  32. import org.apache.hadoop.hdfs.protocol.FSConstants;
  33. import org.apache.hadoop.hdfs.server.common.HdfsConstants.NodeType;
  34. import org.apache.hadoop.hdfs.server.common.HdfsConstants.StartupOption;
  35. import org.apache.hadoop.fs.FileUtil;
  36. import org.apache.hadoop.util.StringUtils;
  37. import org.apache.hadoop.util.VersionInfo;
  38. /**
  39.  * Storage information file.
  40.  * <p>
  41.  * Local storage information is stored in a separate file VERSION.
  42.  * It contains type of the node, 
  43.  * the storage layout version, the namespace id, and 
  44.  * the fs state creation time.
  45.  * <p>
  46.  * Local storage can reside in multiple directories. 
  47.  * Each directory should contain the same VERSION file as the others.
  48.  * During startup Hadoop servers (name-node and data-nodes) read their local 
  49.  * storage information from them.
  50.  * <p>
  51.  * The servers hold a lock for each storage directory while they run so that 
  52.  * other nodes were not able to startup sharing the same storage.
  53.  * The locks are released when the servers stop (normally or abnormally).
  54.  * 
  55.  */
  56. public abstract class Storage extends StorageInfo {
  57.   public static final Log LOG = LogFactory.getLog(Storage.class.getName());
  58.   // Constants
  59.   
  60.   // last layout version that did not suppot upgrades
  61.   protected static final int LAST_PRE_UPGRADE_LAYOUT_VERSION = -3;
  62.   
  63.   // this corresponds to Hadoop-0.14.
  64.   public static final int LAST_UPGRADABLE_LAYOUT_VERSION = -7;
  65.   protected static final String LAST_UPGRADABLE_HADOOP_VERSION = "Hadoop-0.14";
  66.   /* this should be removed when LAST_UPGRADABLE_LV goes beyond -13.
  67.    * any upgrade code that uses this constant should also be removed. */
  68.   public static final int PRE_GENERATIONSTAMP_LAYOUT_VERSION = -13;
  69.   
  70.   private   static final String STORAGE_FILE_LOCK     = "in_use.lock";
  71.   protected static final String STORAGE_FILE_VERSION  = "VERSION";
  72.   public static final String STORAGE_DIR_CURRENT   = "current";
  73.   private   static final String STORAGE_DIR_PREVIOUS  = "previous";
  74.   private   static final String STORAGE_TMP_REMOVED   = "removed.tmp";
  75.   private   static final String STORAGE_TMP_PREVIOUS  = "previous.tmp";
  76.   private   static final String STORAGE_TMP_FINALIZED = "finalized.tmp";
  77.   private   static final String STORAGE_TMP_LAST_CKPT = "lastcheckpoint.tmp";
  78.   private   static final String STORAGE_PREVIOUS_CKPT = "previous.checkpoint";
  79.   
  80.   public enum StorageState {
  81.     NON_EXISTENT,
  82.     NOT_FORMATTED,
  83.     COMPLETE_UPGRADE,
  84.     RECOVER_UPGRADE,
  85.     COMPLETE_FINALIZE,
  86.     COMPLETE_ROLLBACK,
  87.     RECOVER_ROLLBACK,
  88.     COMPLETE_CHECKPOINT,
  89.     RECOVER_CHECKPOINT,
  90.     NORMAL;
  91.   }
  92.   
  93.   /**
  94.    * An interface to denote storage directory type
  95.    * Implementations can define a type for storage directory by implementing
  96.    * this interface.
  97.    */
  98.   public interface StorageDirType {
  99.     public StorageDirType getStorageDirType();
  100.     public boolean isOfType(StorageDirType type);
  101.   }
  102.   
  103.   private NodeType storageType;    // Type of the node using this storage 
  104.   protected List<StorageDirectory> storageDirs = new ArrayList<StorageDirectory>();
  105.   
  106.   private class DirIterator implements Iterator<StorageDirectory> {
  107.     StorageDirType dirType;
  108.     int prevIndex; // for remove()
  109.     int nextIndex; // for next()
  110.     
  111.     DirIterator(StorageDirType dirType) {
  112.       this.dirType = dirType;
  113.       this.nextIndex = 0;
  114.       this.prevIndex = 0;
  115.     }
  116.     
  117.     public boolean hasNext() {
  118.       if (storageDirs.isEmpty() || nextIndex >= storageDirs.size())
  119.         return false;
  120.       if (dirType != null) {
  121.         while (nextIndex < storageDirs.size()) {
  122.           if (getStorageDir(nextIndex).getStorageDirType().isOfType(dirType))
  123.             break;
  124.           nextIndex++;
  125.         }
  126.         if (nextIndex >= storageDirs.size())
  127.          return false;
  128.       }
  129.       return true;
  130.     }
  131.     
  132.     public StorageDirectory next() {
  133.       StorageDirectory sd = getStorageDir(nextIndex);
  134.       prevIndex = nextIndex;
  135.       nextIndex++;
  136.       if (dirType != null) {
  137.         while (nextIndex < storageDirs.size()) {
  138.           if (getStorageDir(nextIndex).getStorageDirType().isOfType(dirType))
  139.             break;
  140.           nextIndex++;
  141.         }
  142.       }
  143.       return sd;
  144.     }
  145.     
  146.     public void remove() {
  147.       nextIndex = prevIndex; // restore previous state
  148.       storageDirs.remove(prevIndex); // remove last returned element
  149.       hasNext(); // reset nextIndex to correct place
  150.     }
  151.   }
  152.   
  153.   /**
  154.    * Return default iterator
  155.    * This iterator returns all entires of storageDirs
  156.    */
  157.   public Iterator<StorageDirectory> dirIterator() {
  158.     return dirIterator(null);
  159.   }
  160.   
  161.   /**
  162.    * Return iterator based on Storage Directory Type
  163.    * This iterator selects entires of storageDirs of type dirType and returns
  164.    * them via the Iterator
  165.    */
  166.   public Iterator<StorageDirectory> dirIterator(StorageDirType dirType) {
  167.     return new DirIterator(dirType);
  168.   }
  169.   
  170.   /**
  171.    * One of the storage directories.
  172.    */
  173.   public class StorageDirectory {
  174.     File              root; // root directory
  175.     FileLock          lock; // storage lock
  176.     StorageDirType dirType; // storage dir type
  177.     
  178.     public StorageDirectory(File dir) {
  179.       // default dirType is null
  180.       this(dir, null);
  181.     }
  182.     
  183.     public StorageDirectory(File dir, StorageDirType dirType) {
  184.       this.root = dir;
  185.       this.lock = null;
  186.       this.dirType = dirType;
  187.     }
  188.     
  189.     /**
  190.      * Get root directory of this storage
  191.      */
  192.     public File getRoot() {
  193.       return root;
  194.     }
  195.     /**
  196.      * Get storage directory type
  197.      */
  198.     public StorageDirType getStorageDirType() {
  199.       return dirType;
  200.     }
  201.     
  202.     /**
  203.      * Read version file.
  204.      * 
  205.      * @throws IOException if file cannot be read or contains inconsistent data
  206.      */
  207.     public void read() throws IOException {
  208.       read(getVersionFile());
  209.     }
  210.     
  211.     public void read(File from) throws IOException {
  212.       RandomAccessFile file = new RandomAccessFile(from, "rws");
  213.       FileInputStream in = null;
  214.       try {
  215.         in = new FileInputStream(file.getFD());
  216.         file.seek(0);
  217.         Properties props = new Properties();
  218.         props.load(in);
  219.         getFields(props, this);
  220.       } finally {
  221.         if (in != null) {
  222.           in.close();
  223.         }
  224.         file.close();
  225.       }
  226.     }
  227.     /**
  228.      * Write version file.
  229.      * 
  230.      * @throws IOException
  231.      */
  232.     public void write() throws IOException {
  233.       corruptPreUpgradeStorage(root);
  234.       write(getVersionFile());
  235.     }
  236.     public void write(File to) throws IOException {
  237.       Properties props = new Properties();
  238.       setFields(props, this);
  239.       RandomAccessFile file = new RandomAccessFile(to, "rws");
  240.       FileOutputStream out = null;
  241.       try {
  242.         file.seek(0);
  243.         out = new FileOutputStream(file.getFD());
  244.         /*
  245.          * If server is interrupted before this line, 
  246.          * the version file will remain unchanged.
  247.          */
  248.         props.store(out, null);
  249.         /*
  250.          * Now the new fields are flushed to the head of the file, but file 
  251.          * length can still be larger then required and therefore the file can 
  252.          * contain whole or corrupted fields from its old contents in the end.
  253.          * If server is interrupted here and restarted later these extra fields
  254.          * either should not effect server behavior or should be handled
  255.          * by the server correctly.
  256.          */
  257.         file.setLength(out.getChannel().position());
  258.       } finally {
  259.         if (out != null) {
  260.           out.close();
  261.         }
  262.         file.close();
  263.       }
  264.     }
  265.     /**
  266.      * Clear and re-create storage directory.
  267.      * <p>
  268.      * Removes contents of the current directory and creates an empty directory.
  269.      * 
  270.      * This does not fully format storage directory. 
  271.      * It cannot write the version file since it should be written last after  
  272.      * all other storage type dependent files are written.
  273.      * Derived storage is responsible for setting specific storage values and
  274.      * writing the version file to disk.
  275.      * 
  276.      * @throws IOException
  277.      */
  278.     public void clearDirectory() throws IOException {
  279.       File curDir = this.getCurrentDir();
  280.       if (curDir.exists())
  281.         if (!(FileUtil.fullyDelete(curDir)))
  282.           throw new IOException("Cannot remove current directory: " + curDir);
  283.       if (!curDir.mkdirs())
  284.         throw new IOException("Cannot create directory " + curDir);
  285.     }
  286.     public File getCurrentDir() {
  287.       return new File(root, STORAGE_DIR_CURRENT);
  288.     }
  289.     public File getVersionFile() {
  290.       return new File(new File(root, STORAGE_DIR_CURRENT), STORAGE_FILE_VERSION);
  291.     }
  292.     public File getPreviousVersionFile() {
  293.       return new File(new File(root, STORAGE_DIR_PREVIOUS), STORAGE_FILE_VERSION);
  294.     }
  295.     public File getPreviousDir() {
  296.       return new File(root, STORAGE_DIR_PREVIOUS);
  297.     }
  298.     public File getPreviousTmp() {
  299.       return new File(root, STORAGE_TMP_PREVIOUS);
  300.     }
  301.     public File getRemovedTmp() {
  302.       return new File(root, STORAGE_TMP_REMOVED);
  303.     }
  304.     public File getFinalizedTmp() {
  305.       return new File(root, STORAGE_TMP_FINALIZED);
  306.     }
  307.     public File getLastCheckpointTmp() {
  308.       return new File(root, STORAGE_TMP_LAST_CKPT);
  309.     }
  310.     public File getPreviousCheckpoint() {
  311.       return new File(root, STORAGE_PREVIOUS_CKPT);
  312.     }
  313.     /**
  314.      * Check consistency of the storage directory
  315.      * 
  316.      * @param startOpt a startup option.
  317.      *  
  318.      * @return state {@link StorageState} of the storage directory 
  319.      * @throws {@link InconsistentFSStateException} if directory state is not 
  320.      * consistent and cannot be recovered 
  321.      */
  322.     public StorageState analyzeStorage(StartupOption startOpt) throws IOException {
  323.       assert root != null : "root is null";
  324.       String rootPath = root.getCanonicalPath();
  325.       try { // check that storage exists
  326.         if (!root.exists()) {
  327.           // storage directory does not exist
  328.           if (startOpt != StartupOption.FORMAT) {
  329.             LOG.info("Storage directory " + rootPath + " does not exist.");
  330.             return StorageState.NON_EXISTENT;
  331.           }
  332.           LOG.info(rootPath + " does not exist. Creating ...");
  333.           if (!root.mkdirs())
  334.             throw new IOException("Cannot create directory " + rootPath);
  335.         }
  336.         // or is inaccessible
  337.         if (!root.isDirectory()) {
  338.           LOG.info(rootPath + "is not a directory.");
  339.           return StorageState.NON_EXISTENT;
  340.         }
  341.         if (!root.canWrite()) {
  342.           LOG.info("Cannot access storage directory " + rootPath);
  343.           return StorageState.NON_EXISTENT;
  344.         }
  345.       } catch(SecurityException ex) {
  346.         LOG.info("Cannot access storage directory " + rootPath, ex);
  347.         return StorageState.NON_EXISTENT;
  348.       }
  349.       this.lock(); // lock storage if it exists
  350.       if (startOpt == HdfsConstants.StartupOption.FORMAT)
  351.         return StorageState.NOT_FORMATTED;
  352.       if (startOpt != HdfsConstants.StartupOption.IMPORT) {
  353.         //make sure no conversion is required
  354.         checkConversionNeeded(this);
  355.       }
  356.       // check whether current directory is valid
  357.       File versionFile = getVersionFile();
  358.       boolean hasCurrent = versionFile.exists();
  359.       // check which directories exist
  360.       boolean hasPrevious = getPreviousDir().exists();
  361.       boolean hasPreviousTmp = getPreviousTmp().exists();
  362.       boolean hasRemovedTmp = getRemovedTmp().exists();
  363.       boolean hasFinalizedTmp = getFinalizedTmp().exists();
  364.       boolean hasCheckpointTmp = getLastCheckpointTmp().exists();
  365.       if (!(hasPreviousTmp || hasRemovedTmp
  366.           || hasFinalizedTmp || hasCheckpointTmp)) {
  367.         // no temp dirs - no recovery
  368.         if (hasCurrent)
  369.           return StorageState.NORMAL;
  370.         if (hasPrevious)
  371.           throw new InconsistentFSStateException(root,
  372.                               "version file in current directory is missing.");
  373.         return StorageState.NOT_FORMATTED;
  374.       }
  375.       if ((hasPreviousTmp?1:0) + (hasRemovedTmp?1:0)
  376.           + (hasFinalizedTmp?1:0) + (hasCheckpointTmp?1:0) > 1)
  377.         // more than one temp dirs
  378.         throw new InconsistentFSStateException(root,
  379.                                                "too many temporary directories.");
  380.       // # of temp dirs == 1 should either recover or complete a transition
  381.       if (hasCheckpointTmp) {
  382.         return hasCurrent ? StorageState.COMPLETE_CHECKPOINT
  383.                           : StorageState.RECOVER_CHECKPOINT;
  384.       }
  385.       if (hasFinalizedTmp) {
  386.         if (hasPrevious)
  387.           throw new InconsistentFSStateException(root,
  388.                                                  STORAGE_DIR_PREVIOUS + " and " + STORAGE_TMP_FINALIZED
  389.                                                  + "cannot exist together.");
  390.         return StorageState.COMPLETE_FINALIZE;
  391.       }
  392.       if (hasPreviousTmp) {
  393.         if (hasPrevious)
  394.           throw new InconsistentFSStateException(root,
  395.                                                  STORAGE_DIR_PREVIOUS + " and " + STORAGE_TMP_PREVIOUS
  396.                                                  + " cannot exist together.");
  397.         if (hasCurrent)
  398.           return StorageState.COMPLETE_UPGRADE;
  399.         return StorageState.RECOVER_UPGRADE;
  400.       }
  401.       
  402.       assert hasRemovedTmp : "hasRemovedTmp must be true";
  403.       if (!(hasCurrent ^ hasPrevious))
  404.         throw new InconsistentFSStateException(root,
  405.                                                "one and only one directory " + STORAGE_DIR_CURRENT 
  406.                                                + " or " + STORAGE_DIR_PREVIOUS 
  407.                                                + " must be present when " + STORAGE_TMP_REMOVED
  408.                                                + " exists.");
  409.       if (hasCurrent)
  410.         return StorageState.COMPLETE_ROLLBACK;
  411.       return StorageState.RECOVER_ROLLBACK;
  412.     }
  413.     /**
  414.      * Complete or recover storage state from previously failed transition.
  415.      * 
  416.      * @param curState specifies what/how the state should be recovered
  417.      * @throws IOException
  418.      */
  419.     public void doRecover(StorageState curState) throws IOException {
  420.       File curDir = getCurrentDir();
  421.       String rootPath = root.getCanonicalPath();
  422.       switch(curState) {
  423.       case COMPLETE_UPGRADE:  // mv previous.tmp -> previous
  424.         LOG.info("Completing previous upgrade for storage directory " 
  425.                  + rootPath + ".");
  426.         rename(getPreviousTmp(), getPreviousDir());
  427.         return;
  428.       case RECOVER_UPGRADE:   // mv previous.tmp -> current
  429.         LOG.info("Recovering storage directory " + rootPath
  430.                  + " from previous upgrade.");
  431.         if (curDir.exists())
  432.           deleteDir(curDir);
  433.         rename(getPreviousTmp(), curDir);
  434.         return;
  435.       case COMPLETE_ROLLBACK: // rm removed.tmp
  436.         LOG.info("Completing previous rollback for storage directory "
  437.                  + rootPath + ".");
  438.         deleteDir(getRemovedTmp());
  439.         return;
  440.       case RECOVER_ROLLBACK:  // mv removed.tmp -> current
  441.         LOG.info("Recovering storage directory " + rootPath
  442.                  + " from previous rollback.");
  443.         rename(getRemovedTmp(), curDir);
  444.         return;
  445.       case COMPLETE_FINALIZE: // rm finalized.tmp
  446.         LOG.info("Completing previous finalize for storage directory "
  447.                  + rootPath + ".");
  448.         deleteDir(getFinalizedTmp());
  449.         return;
  450.       case COMPLETE_CHECKPOINT: // mv lastcheckpoint.tmp -> previous.checkpoint
  451.         LOG.info("Completing previous checkpoint for storage directory " 
  452.                  + rootPath + ".");
  453.         File prevCkptDir = getPreviousCheckpoint();
  454.         if (prevCkptDir.exists())
  455.           deleteDir(prevCkptDir);
  456.         rename(getLastCheckpointTmp(), prevCkptDir);
  457.         return;
  458.       case RECOVER_CHECKPOINT:  // mv lastcheckpoint.tmp -> current
  459.         LOG.info("Recovering storage directory " + rootPath
  460.                  + " from failed checkpoint.");
  461.         if (curDir.exists())
  462.           deleteDir(curDir);
  463.         rename(getLastCheckpointTmp(), curDir);
  464.         return;
  465.       default:
  466.         throw new IOException("Unexpected FS state: " + curState);
  467.       }
  468.     }
  469.     /**
  470.      * Lock storage to provide exclusive access.
  471.      * 
  472.      * <p> Locking is not supported by all file systems.
  473.      * E.g., NFS does not consistently support exclusive locks.
  474.      * 
  475.      * <p> If locking is supported we guarantee exculsive access to the
  476.      * storage directory. Otherwise, no guarantee is given.
  477.      * 
  478.      * @throws IOException if locking fails
  479.      */
  480.     public void lock() throws IOException {
  481.       this.lock = tryLock();
  482.       if (lock == null) {
  483.         String msg = "Cannot lock storage " + this.root 
  484.           + ". The directory is already locked.";
  485.         LOG.info(msg);
  486.         throw new IOException(msg);
  487.       }
  488.     }
  489.     /**
  490.      * Attempts to acquire an exclusive lock on the storage.
  491.      * 
  492.      * @return A lock object representing the newly-acquired lock or
  493.      * <code>null</code> if storage is already locked.
  494.      * @throws IOException if locking fails.
  495.      */
  496.     FileLock tryLock() throws IOException {
  497.       File lockF = new File(root, STORAGE_FILE_LOCK);
  498.       lockF.deleteOnExit();
  499.       RandomAccessFile file = new RandomAccessFile(lockF, "rws");
  500.       FileLock res = null;
  501.       try {
  502.         res = file.getChannel().tryLock();
  503.       } catch(OverlappingFileLockException oe) {
  504.         file.close();
  505.         return null;
  506.       } catch(IOException e) {
  507.         LOG.info(StringUtils.stringifyException(e));
  508.         file.close();
  509.         throw e;
  510.       }
  511.       return res;
  512.     }
  513.     /**
  514.      * Unlock storage.
  515.      * 
  516.      * @throws IOException
  517.      */
  518.     public void unlock() throws IOException {
  519.       if (this.lock == null)
  520.         return;
  521.       this.lock.release();
  522.       lock.channel().close();
  523.       lock = null;
  524.     }
  525.   }
  526.   /**
  527.    * Create empty storage info of the specified type
  528.    */
  529.   protected Storage(NodeType type) {
  530.     super();
  531.     this.storageType = type;
  532.   }
  533.   
  534.   protected Storage(NodeType type, int nsID, long cT) {
  535.     super(FSConstants.LAYOUT_VERSION, nsID, cT);
  536.     this.storageType = type;
  537.   }
  538.   
  539.   protected Storage(NodeType type, StorageInfo storageInfo) {
  540.     super(storageInfo);
  541.     this.storageType = type;
  542.   }
  543.   
  544.   public int getNumStorageDirs() {
  545.     return storageDirs.size();
  546.   }
  547.   
  548.   public StorageDirectory getStorageDir(int idx) {
  549.     return storageDirs.get(idx);
  550.   }
  551.   
  552.   protected void addStorageDir(StorageDirectory sd) {
  553.     storageDirs.add(sd);
  554.   }
  555.   
  556.   public abstract boolean isConversionNeeded(StorageDirectory sd) throws IOException;
  557.   /*
  558.    * Coversion is no longer supported. So this should throw exception if
  559.    * conversion is needed.
  560.    */
  561.   private void checkConversionNeeded(StorageDirectory sd) throws IOException {
  562.     if (isConversionNeeded(sd)) {
  563.       //throw an exception
  564.       checkVersionUpgradable(0);
  565.     }
  566.   }
  567.   /**
  568.    * Checks if the upgrade from the given old version is supported. If
  569.    * no upgrade is supported, it throws IncorrectVersionException.
  570.    * 
  571.    * @param oldVersion
  572.    */
  573.   protected static void checkVersionUpgradable(int oldVersion) 
  574.                                      throws IOException {
  575.     if (oldVersion > LAST_UPGRADABLE_LAYOUT_VERSION) {
  576.       String msg = "*********** Upgrade is not supported from this older" +
  577.                    " version of storage to the current version." + 
  578.                    " Please upgrade to " + LAST_UPGRADABLE_HADOOP_VERSION +
  579.                    " or a later version and then upgrade to current" +
  580.                    " version. Old layout version is " + 
  581.                    (oldVersion == 0 ? "'too old'" : (""+oldVersion)) +
  582.                    " and latest layout version this software version can" +
  583.                    " upgrade from is " + LAST_UPGRADABLE_LAYOUT_VERSION +
  584.                    ". ************";
  585.       LOG.error(msg);
  586.       throw new IOException(msg); 
  587.     }
  588.     
  589.   }
  590.   
  591.   /**
  592.    * Get common storage fields.
  593.    * Should be overloaded if additional fields need to be get.
  594.    * 
  595.    * @param props
  596.    * @throws IOException
  597.    */
  598.   protected void getFields(Properties props, 
  599.                            StorageDirectory sd 
  600.                            ) throws IOException {
  601.     String sv, st, sid, sct;
  602.     sv = props.getProperty("layoutVersion");
  603.     st = props.getProperty("storageType");
  604.     sid = props.getProperty("namespaceID");
  605.     sct = props.getProperty("cTime");
  606.     if (sv == null || st == null || sid == null || sct == null)
  607.       throw new InconsistentFSStateException(sd.root,
  608.                                              "file " + STORAGE_FILE_VERSION + " is invalid.");
  609.     int rv = Integer.parseInt(sv);
  610.     NodeType rt = NodeType.valueOf(st);
  611.     int rid = Integer.parseInt(sid);
  612.     long rct = Long.parseLong(sct);
  613.     if (!storageType.equals(rt) ||
  614.         !((namespaceID == 0) || (rid == 0) || namespaceID == rid))
  615.       throw new InconsistentFSStateException(sd.root,
  616.                                              "is incompatible with others.");
  617.     if (rv < FSConstants.LAYOUT_VERSION) // future version
  618.       throw new IncorrectVersionException(rv, "storage directory " 
  619.                                           + sd.root.getCanonicalPath());
  620.     layoutVersion = rv;
  621.     storageType = rt;
  622.     namespaceID = rid;
  623.     cTime = rct;
  624.   }
  625.   
  626.   /**
  627.    * Set common storage fields.
  628.    * Should be overloaded if additional fields need to be set.
  629.    * 
  630.    * @param props
  631.    * @throws IOException
  632.    */
  633.   protected void setFields(Properties props, 
  634.                            StorageDirectory sd 
  635.                            ) throws IOException {
  636.     props.setProperty("layoutVersion", String.valueOf(layoutVersion));
  637.     props.setProperty("storageType", storageType.toString());
  638.     props.setProperty("namespaceID", String.valueOf(namespaceID));
  639.     props.setProperty("cTime", String.valueOf(cTime));
  640.   }
  641.   public static void rename(File from, File to) throws IOException {
  642.     if (!from.renameTo(to))
  643.       throw new IOException("Failed to rename " 
  644.                             + from.getCanonicalPath() + " to " + to.getCanonicalPath());
  645.   }
  646.   protected static void deleteDir(File dir) throws IOException {
  647.     if (!FileUtil.fullyDelete(dir))
  648.       throw new IOException("Failed to delete " + dir.getCanonicalPath());
  649.   }
  650.   
  651.   /**
  652.    * Write all data storage files.
  653.    * @throws IOException
  654.    */
  655.   public void writeAll() throws IOException {
  656.     this.layoutVersion = FSConstants.LAYOUT_VERSION;
  657.     for (Iterator<StorageDirectory> it = storageDirs.iterator(); it.hasNext();) {
  658.       it.next().write();
  659.     }
  660.   }
  661.   /**
  662.    * Unlock all storage directories.
  663.    * @throws IOException
  664.    */
  665.   public void unlockAll() throws IOException {
  666.     for (Iterator<StorageDirectory> it = storageDirs.iterator(); it.hasNext();) {
  667.       it.next().unlock();
  668.     }
  669.   }
  670.   /**
  671.    * Check whether underlying file system supports file locking.
  672.    * 
  673.    * @return <code>true</code> if exclusive locks are supported or
  674.    *         <code>false</code> otherwise.
  675.    * @throws IOException
  676.    * @see StorageDirectory#lock()
  677.    */
  678.   public boolean isLockSupported(int idx) throws IOException {
  679.     StorageDirectory sd = storageDirs.get(idx);
  680.     FileLock firstLock = null;
  681.     FileLock secondLock = null;
  682.     try {
  683.       firstLock = sd.lock;
  684.       if(firstLock == null) {
  685.         firstLock = sd.tryLock();
  686.         if(firstLock == null)
  687.           return true;
  688.       }
  689.       secondLock = sd.tryLock();
  690.       if(secondLock == null)
  691.         return true;
  692.     } finally {
  693.       if(firstLock != null && firstLock != sd.lock) {
  694.         firstLock.release();
  695.         firstLock.channel().close();
  696.       }
  697.       if(secondLock != null) {
  698.         secondLock.release();
  699.         secondLock.channel().close();
  700.       }
  701.     }
  702.     return false;
  703.   }
  704.   public static String getBuildVersion() {
  705.     return VersionInfo.getRevision();
  706.   }
  707.   public static String getRegistrationID(StorageInfo storage) {
  708.     return "NS-" + Integer.toString(storage.getNamespaceID())
  709.       + "-" + Integer.toString(storage.getLayoutVersion())
  710.       + "-" + Long.toString(storage.getCTime());
  711.   }
  712.   // Pre-upgrade version compatibility
  713.   protected abstract void corruptPreUpgradeStorage(File rootDir) throws IOException;
  714.   protected void writeCorruptedData(RandomAccessFile file) throws IOException {
  715.     final String messageForPreUpgradeVersion =
  716.       "nThis file is INTENTIONALLY CORRUPTED so that versionsn"
  717.       + "of Hadoop prior to 0.13 (which are incompatiblen"
  718.       + "with this directory layout) will fail to start.n";
  719.   
  720.     file.seek(0);
  721.     file.writeInt(FSConstants.LAYOUT_VERSION);
  722.     org.apache.hadoop.io.UTF8.writeString(file, "");
  723.     file.writeBytes(messageForPreUpgradeVersion);
  724.     file.getFD().sync();
  725.   }
  726. }