You are viewing a plain text version of this content. The canonical link for it is here.
Posted to hdfs-commits@hadoop.apache.org by su...@apache.org on 2011/06/02 19:39:15 UTC

svn commit: r1130680 - in /hadoop/hdfs/branches/branch-0.22: ./ src/java/org/apache/hadoop/hdfs/protocol/ src/java/org/apache/hadoop/hdfs/server/common/ src/java/org/apache/hadoop/hdfs/server/datanode/ src/java/org/apache/hadoop/hdfs/server/namenode/ s...

Author: suresh
Date: Thu Jun  2 17:39:14 2011
New Revision: 1130680

URL: http://svn.apache.org/viewvc?rev=1130680&view=rev
Log:
HDFS-1936. Layout version change from HDFS-1822 causes upgrade failure. Contributed by Suresh Srinivas.


Added:
    hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/protocol/LayoutVersion.java
    hadoop/hdfs/branches/branch-0.22/src/test/hdfs/org/apache/hadoop/hdfs/protocol/
    hadoop/hdfs/branches/branch-0.22/src/test/hdfs/org/apache/hadoop/hdfs/protocol/TestLayoutVersion.java
Modified:
    hadoop/hdfs/branches/branch-0.22/CHANGES.txt
    hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/protocol/FSConstants.java
    hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/common/Storage.java
    hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/datanode/DataStorage.java
    hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSEditLogLoader.java
    hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSImage.java
    hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSImageFormat.java
    hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/tools/offlineImageViewer/ImageLoaderCurrent.java
    hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/tools/offlineImageViewer/OfflineImageViewer.java
    hadoop/hdfs/branches/branch-0.22/src/test/hdfs/org/apache/hadoop/hdfs/UpgradeUtilities.java

Modified: hadoop/hdfs/branches/branch-0.22/CHANGES.txt
URL: http://svn.apache.org/viewvc/hadoop/hdfs/branches/branch-0.22/CHANGES.txt?rev=1130680&r1=1130679&r2=1130680&view=diff
==============================================================================
--- hadoop/hdfs/branches/branch-0.22/CHANGES.txt (original)
+++ hadoop/hdfs/branches/branch-0.22/CHANGES.txt Thu Jun  2 17:39:14 2011
@@ -552,6 +552,9 @@ Release 0.22.0 - Unreleased
     HDFS-977. DataNode.createInterDataNodeProtocolProxy() guards a log
     at the wrong level. (Harsh J Chouraria via todd)
 
+    HDFS-1936. Layout version change from HDFS-1822 causes upgrade failure.
+    (suresh)
+
 Release 0.21.1 - Unreleased
 
   IMPROVEMENTS

Modified: hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/protocol/FSConstants.java
URL: http://svn.apache.org/viewvc/hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/protocol/FSConstants.java?rev=1130680&r1=1130679&r2=1130680&view=diff
==============================================================================
--- hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/protocol/FSConstants.java (original)
+++ hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/protocol/FSConstants.java Thu Jun  2 17:39:14 2011
@@ -87,11 +87,9 @@ public interface FSConstants {
    */
   public static final String HDFS_URI_SCHEME = "hdfs";
 
-  // Version is reflected in the dfs image and edit log files.
-  // Version is reflected in the data storage file.
-  // Versions are negative.
-  // Decrement LAYOUT_VERSION to define a new version.
-  public static final int LAYOUT_VERSION = -33;
-  // Current version: 33
-  // -31, -32 are reserved for 0.20.203, 0.20.204
+  /**
+   * Please see {@link LayoutVersion} on adding new layout version.
+   */
+  public static final int LAYOUT_VERSION = 
+    LayoutVersion.getCurrentLayoutVersion();
 }

Added: hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/protocol/LayoutVersion.java
URL: http://svn.apache.org/viewvc/hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/protocol/LayoutVersion.java?rev=1130680&view=auto
==============================================================================
--- hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/protocol/LayoutVersion.java (added)
+++ hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/protocol/LayoutVersion.java Thu Jun  2 17:39:14 2011
@@ -0,0 +1,184 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.hdfs.protocol;
+
+import java.util.EnumSet;
+import java.util.HashMap;
+import java.util.Map;
+
+import org.apache.hadoop.classification.InterfaceAudience;
+
+/**
+ * This class tracks changes in the layout version of HDFS.
+ * 
+ * Layout version is changed for following reasons:
+ * <ol>
+ * <li>The layout of how namenode or datanode stores information 
+ * on disk changes.</li>
+ * <li>A new operation code is added to the editlog.</li>
+ * <li>Modification such as format of a record, content of a record 
+ * in editlog or fsimage.</li>
+ * </ol>
+ * <br>
+ * <b>How to update layout version:<br></b>
+ * When a change requires new layout version, please add an entry into
+ * {@link Feature} with a short enum name, new layout version and description
+ * of the change. Please see {@link Feature} for further details.
+ * <br>
+ */
+@InterfaceAudience.Private
+public class LayoutVersion {
+ 
+  /**
+   * Enums for features that change the layout version.
+   * <br><br>
+   * To add a new layout version:
+   * <ul>
+   * <li>Define a new enum constant with a short enum name, the new layout version 
+   * and description of the added feature.</li>
+   * <li>When adding a layout version with an ancestor that is not same as
+   * its immediate predecessor, use the constructor where a spacific ancestor
+   * can be passed.
+   * </li>
+   * </ul>
+   */
+  public static enum Feature {
+    NAMESPACE_QUOTA(-16, "Support for namespace quotas"),
+    FILE_ACCESS_TIME(-17, "Support for access time on files"),
+    DISKSPACE_QUOTA(-18, "Support for disk space quotas"),
+    STICKY_BIT(-19, "Support for sticky bits"),
+    APPEND_RBW_DIR(-20, "Datanode has \"rbw\" subdirectory for append"),
+    ATOMIC_RENAME(-21, "Support for atomic rename"),
+    CONCAT(-22, "Support for concat operation"),
+    SYMLINKS(-23, "Support for symbolic links"),
+    DELEGATION_TOKEN(-24, "Support for delegation tokens for security"),
+    FSIMAGE_COMPRESSION(-25, "Support for fsimage compression"),
+    FSIMAGE_CHECKSUM(-26, "Support checksum for fsimage"),
+    REMOVE_REL13_DISK_LAYOUT_SUPPORT(-27, "Remove support for 0.13 disk layout"),
+    UNUSED_28(-28, "Support checksum for editlog"),
+    UNUSED_29(-29, "Skipped version"),
+    UNUSED_30(-30, "Store only last part of path in fsimage"),
+    RESERVED_REL20_203(-31, -19, "Reserved for release 0.20.203"),
+    RESERVED_REL20_204(-32, "Reserved for release 0.20.204"),
+    RESERVED_REL22(-33, -27, "Reserved for release 0.22");
+    
+    final int lv;
+    final int ancestorLV;
+    final String description;
+    
+    /**
+     * Feature that is added at {@code currentLV}. 
+     * @param lv new layout version with the addition of this feature
+     * @param description description of the feature
+     */
+    Feature(final int lv, final String description) {
+      this(lv, lv + 1, description);
+    }
+
+    /**
+     * Feature that is added at {@code currentLV}.
+     * @param lv new layout version with the addition of this feature
+     * @param ancestorLV layout version from which the new lv is derived
+     *          from.
+     * @param description description of the feature
+     */
+    Feature(final int lv, final int ancestorLV,
+        final String description) {
+      this.lv = lv;
+      this.ancestorLV = ancestorLV;
+      this.description = description;
+    }
+  }
+  
+  // Build layout version and corresponding feature matrix
+  static final Map<Integer, EnumSet<Feature>>map = 
+    new HashMap<Integer, EnumSet<Feature>>();
+  
+  // Static initialization 
+  static {
+    initMap();
+  }
+  
+  /**
+   * Initialize the map of a layout version and EnumSet of {@link Feature}s 
+   * supported.
+   */
+  private static void initMap() {
+    // Go through all the enum constants and build a map of
+    // LayoutVersion <-> EnumSet of all supported features in that LayoutVersion
+    for (Feature f : Feature.values()) {
+      EnumSet<Feature> ancestorSet = map.get(f.ancestorLV);
+      if (ancestorSet == null) {
+        ancestorSet = EnumSet.noneOf(Feature.class); // Empty enum set
+        map.put(f.ancestorLV, ancestorSet);
+      }
+      EnumSet<Feature> featureSet = EnumSet.copyOf(ancestorSet);
+      featureSet.add(f);
+      map.put(f.lv, featureSet);
+    }
+    
+    // Special initialization for 0.20.203 and 0.20.204
+    // to add Feature#DELEGATION_TOKEN
+    specialInit(Feature.RESERVED_REL20_203.lv, Feature.DELEGATION_TOKEN);
+    specialInit(Feature.RESERVED_REL20_204.lv, Feature.DELEGATION_TOKEN);
+  }
+  
+  private static void specialInit(int lv, Feature f) {
+    EnumSet<Feature> set = map.get(lv);
+    set.add(f);
+  }
+  
+  /**
+   * Gets formatted string that describes {@link LayoutVersion} information.
+   */
+  public static String getString() {
+    final StringBuilder buf = new StringBuilder();
+    buf.append("Feature List:\n");
+    for (Feature f : Feature.values()) {
+      buf.append(f).append(" introduced in layout version ")
+          .append(f.lv).append(" (").
+      append(f.description).append(")\n");
+    }
+    
+    buf.append("\n\nLayoutVersion and supported features:\n");
+    for (Feature f : Feature.values()) {
+      buf.append(f.lv).append(": ").append(map.get(f.lv))
+          .append("\n");
+    }
+    return buf.toString();
+  }
+  
+  /**
+   * Returns true if a given feature is supported in the given layout version
+   * @param f Feature
+   * @param lv LayoutVersion
+   * @return true if {@code f} is supported in layout version {@code lv}
+   */
+  public static boolean supports(final Feature f, final int lv) {
+    final EnumSet<Feature> set =  map.get(lv);
+    return set != null && set.contains(f);
+  }
+  
+  /**
+   * Get the current layout version
+   */
+  public static int getCurrentLayoutVersion() {
+    Feature[] values = Feature.values();
+    return values[values.length - 1].lv;
+  }
+}

Modified: hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/common/Storage.java
URL: http://svn.apache.org/viewvc/hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/common/Storage.java?rev=1130680&r1=1130679&r2=1130680&view=diff
==============================================================================
--- hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/common/Storage.java (original)
+++ hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/common/Storage.java Thu Jun  2 17:39:14 2011
@@ -33,6 +33,8 @@ import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.hdfs.protocol.FSConstants;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion.Feature;
 import org.apache.hadoop.hdfs.server.common.HdfsConstants.NodeType;
 import org.apache.hadoop.hdfs.server.common.HdfsConstants.StartupOption;
 import org.apache.hadoop.fs.FileUtil;
@@ -75,9 +77,6 @@ public abstract class Storage extends St
    * any upgrade code that uses this constant should also be removed. */
   public static final int PRE_GENERATIONSTAMP_LAYOUT_VERSION = -13;
   
-  // last layout version that did not support persistent rbw replicas
-  public static final int PRE_RBW_LAYOUT_VERSION = -19;
-  
   /** Layout versions of 0.20.203 release */
   public static final int[] LAYOUT_VERSIONS_203 = {-19, -31};
 

Modified: hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/datanode/DataStorage.java
URL: http://svn.apache.org/viewvc/hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/datanode/DataStorage.java?rev=1130680&r1=1130679&r2=1130680&view=diff
==============================================================================
--- hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/datanode/DataStorage.java (original)
+++ hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/datanode/DataStorage.java Thu Jun  2 17:39:14 2011
@@ -34,6 +34,8 @@ import java.util.regex.Pattern;
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.fs.FileUtil.HardLink;
 import org.apache.hadoop.hdfs.protocol.FSConstants;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion.Feature;
 import org.apache.hadoop.hdfs.server.common.GenerationStamp;
 import org.apache.hadoop.hdfs.server.common.InconsistentFSStateException;
 import org.apache.hadoop.hdfs.server.common.Storage;
@@ -311,7 +313,7 @@ public class DataStorage extends Storage
    * @throws IOException if the directory is not empty or it can not be removed
    */
   private void cleanupDetachDir(File detachDir) throws IOException {
-    if (layoutVersion >= PRE_RBW_LAYOUT_VERSION &&
+    if (!LayoutVersion.supports(Feature.APPEND_RBW_DIR, layoutVersion) &&
         detachDir.exists() && detachDir.isDirectory() ) {
       
         if (detachDir.list().length != 0 ) {
@@ -404,7 +406,7 @@ public class DataStorage extends Storage
   private void linkAllBlocks(File fromDir, File toDir) throws IOException {
     // do the link
     int diskLayoutVersion = this.getLayoutVersion();
-    if (diskLayoutVersion < PRE_RBW_LAYOUT_VERSION) { // RBW version
+    if (LayoutVersion.supports(Feature.APPEND_RBW_DIR, diskLayoutVersion)) {
       // hardlink finalized blocks in tmpDir/finalized
       linkBlocks(new File(fromDir, STORAGE_DIR_FINALIZED), 
           new File(toDir, STORAGE_DIR_FINALIZED), diskLayoutVersion);

Modified: hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSEditLogLoader.java
URL: http://svn.apache.org/viewvc/hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSEditLogLoader.java?rev=1130680&r1=1130679&r2=1130680&view=diff
==============================================================================
--- hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSEditLogLoader.java (original)
+++ hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSEditLogLoader.java Thu Jun  2 17:39:14 2011
@@ -30,6 +30,8 @@ import org.apache.hadoop.hdfs.protocol.B
 import org.apache.hadoop.hdfs.protocol.DatanodeID;
 import org.apache.hadoop.hdfs.protocol.FSConstants;
 import org.apache.hadoop.hdfs.protocol.HdfsFileStatus;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion.Feature;
 import org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier;
 import static org.apache.hadoop.hdfs.server.common.Util.now;
 import org.apache.hadoop.hdfs.server.common.GenerationStamp;
@@ -151,7 +153,7 @@ public class FSEditLogLoader {
           path = FSImageSerialization.readString(in);
           short replication = fsNamesys.adjustReplication(readShort(in));
           mtime = readLong(in);
-          if (logVersion <= -17) {
+          if (LayoutVersion.supports(Feature.FILE_ACCESS_TIME, logVersion)) {
             atime = readLong(in);
           }
           if (logVersion < -7) {
@@ -237,10 +239,6 @@ public class FSEditLogLoader {
           break;
         } 
         case Ops.OP_CONCAT_DELETE: {
-          if (logVersion > -22) {
-            throw new IOException("Unexpected opcode " + opcode
-                + " for version " + logVersion);
-          }
           numOpConcatDelete++;
           int length = in.readInt();
           if (length < 3) { // trg, srcs.., timestam
@@ -299,7 +297,7 @@ public class FSEditLogLoader {
           // The disk format stores atimes for directories as well.
           // However, currently this is not being updated/used because of
           // performance reasons.
-          if (logVersion <= -17) {
+          if (LayoutVersion.supports(Feature.FILE_ACCESS_TIME, logVersion)) {
             atime = readLong(in);
           }
 
@@ -330,38 +328,24 @@ public class FSEditLogLoader {
         }
         case Ops.OP_SET_PERMISSIONS: {
           numOpSetPerm++;
-          if (logVersion > -11)
-            throw new IOException("Unexpected opcode " + opcode
-                                  + " for version " + logVersion);
           fsDir.unprotectedSetPermission(
               FSImageSerialization.readString(in), FsPermission.read(in));
           break;
         }
         case Ops.OP_SET_OWNER: {
           numOpSetOwner++;
-          if (logVersion > -11)
-            throw new IOException("Unexpected opcode " + opcode
-                                  + " for version " + logVersion);
           fsDir.unprotectedSetOwner(FSImageSerialization.readString(in),
               FSImageSerialization.readString_EmptyAsNull(in),
               FSImageSerialization.readString_EmptyAsNull(in));
           break;
         }
         case Ops.OP_SET_NS_QUOTA: {
-          if (logVersion > -16) {
-            throw new IOException("Unexpected opcode " + opcode
-                + " for version " + logVersion);
-          }
           fsDir.unprotectedSetQuota(FSImageSerialization.readString(in), 
                                     readLongWritable(in), 
                                     FSConstants.QUOTA_DONT_SET);
           break;
         }
         case Ops.OP_CLEAR_NS_QUOTA: {
-          if (logVersion > -16) {
-            throw new IOException("Unexpected opcode " + opcode
-                + " for version " + logVersion);
-          }
           fsDir.unprotectedSetQuota(FSImageSerialization.readString(in),
                                     FSConstants.QUOTA_RESET,
                                     FSConstants.QUOTA_DONT_SET);
@@ -404,10 +388,6 @@ public class FSEditLogLoader {
           break;
         }
         case Ops.OP_RENAME: {
-          if (logVersion > -21) {
-            throw new IOException("Unexpected opcode " + opcode
-                + " for version " + logVersion);
-          }
           numOpRename++;
           int length = in.readInt();
           if (length != 3) {
@@ -424,10 +404,6 @@ public class FSEditLogLoader {
           break;
         }
         case Ops.OP_GET_DELEGATION_TOKEN: {
-          if (logVersion > -24) {
-            throw new IOException("Unexpected opcode " + opcode
-                + " for version " + logVersion);
-          }
           numOpGetDelegationToken++;
           DelegationTokenIdentifier delegationTokenId = 
               new DelegationTokenIdentifier();
@@ -438,10 +414,6 @@ public class FSEditLogLoader {
           break;
         }
         case Ops.OP_RENEW_DELEGATION_TOKEN: {
-          if (logVersion > -24) {
-            throw new IOException("Unexpected opcode " + opcode
-                + " for version " + logVersion);
-          }
           numOpRenewDelegationToken++;
           DelegationTokenIdentifier delegationTokenId = 
               new DelegationTokenIdentifier();
@@ -452,10 +424,6 @@ public class FSEditLogLoader {
           break;
         }
         case Ops.OP_CANCEL_DELEGATION_TOKEN: {
-          if (logVersion > -24) {
-            throw new IOException("Unexpected opcode " + opcode
-                + " for version " + logVersion);
-          }
           numOpCancelDelegationToken++;
           DelegationTokenIdentifier delegationTokenId = 
               new DelegationTokenIdentifier();
@@ -465,10 +433,6 @@ public class FSEditLogLoader {
           break;
         }
         case Ops.OP_UPDATE_MASTER_KEY: {
-          if (logVersion > -24) {
-            throw new IOException("Unexpected opcode " + opcode
-                + " for version " + logVersion);
-          }
           numOpUpdateMasterKey++;
           DelegationKey delegationKey = new DelegationKey();
           delegationKey.readFields(in);

Modified: hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSImage.java
URL: http://svn.apache.org/viewvc/hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSImage.java?rev=1130680&r1=1130679&r2=1130680&view=diff
==============================================================================
--- hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSImage.java (original)
+++ hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSImage.java Thu Jun  2 17:39:14 2011
@@ -43,6 +43,8 @@ import org.apache.hadoop.classification.
 import org.apache.hadoop.classification.InterfaceStability;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.hdfs.protocol.FSConstants;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion.Feature;
 import org.apache.hadoop.hdfs.server.common.InconsistentFSStateException;
 import org.apache.hadoop.hdfs.server.common.Storage;
 import org.apache.hadoop.hdfs.server.common.StorageInfo;
@@ -677,7 +679,8 @@ public class FSImage extends Storage {
         sDUV == null? getLayoutVersion() : Integer.parseInt(sDUV));
     
     String sMd5 = props.getProperty(MESSAGE_DIGEST_PROPERTY);
-    if (layoutVersion <= -26) {
+    if (LayoutVersion.supports(Feature.FSIMAGE_CHECKSUM, layoutVersion)) {
+    
       if (sMd5 == null) {
         throw new InconsistentFSStateException(sd.getRoot(),
             "file " + STORAGE_FILE_VERSION + " does not have MD5 image digest.");
@@ -860,8 +863,8 @@ public class FSImage extends Storage {
     RandomAccessFile oldFile = new RandomAccessFile(oldF, "rws");
     try {
       oldFile.seek(0);
-      int odlVersion = oldFile.readInt();
-      if (odlVersion < LAST_PRE_UPGRADE_LAYOUT_VERSION)
+      int oldVersion = oldFile.readInt();
+      if (oldVersion < LAST_PRE_UPGRADE_LAYOUT_VERSION)
         return false;
     } finally {
       oldFile.close();

Modified: hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSImageFormat.java
URL: http://svn.apache.org/viewvc/hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSImageFormat.java?rev=1130680&r1=1130679&r2=1130680&view=diff
==============================================================================
--- hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSImageFormat.java (original)
+++ hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/server/namenode/FSImageFormat.java Thu Jun  2 17:39:14 2011
@@ -40,6 +40,8 @@ import org.apache.hadoop.fs.permission.P
 import org.apache.hadoop.hdfs.DFSUtil;
 import org.apache.hadoop.hdfs.protocol.Block;
 import org.apache.hadoop.hdfs.protocol.FSConstants;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion.Feature;
 import org.apache.hadoop.hdfs.server.common.GenerationStamp;
 import org.apache.hadoop.io.MD5Hash;
 import org.apache.hadoop.io.Text;
@@ -167,7 +169,7 @@ class FSImageFormat {
 
         // read compression related info
         FSImageCompression compression;
-        if (imgVersion <= -25) {  // -25: 1st version providing compression option
+        if (LayoutVersion.supports(Feature.FSIMAGE_COMPRESSION, imgVersion)) {
           compression = FSImageCompression.readCompressionHeader(conf, in);
         } else {
           compression = FSImageCompression.createNoopCompression();
@@ -193,7 +195,7 @@ class FSImageFormat {
           replication = in.readShort();
           replication = targetNamesystem.adjustReplication(replication);
           modificationTime = in.readLong();
-          if (imgVersion <= -17) {
+          if (LayoutVersion.supports(Feature.FILE_ACCESS_TIME, imgVersion)) {
             atime = in.readLong();
           }
           if (imgVersion <= -8) {
@@ -232,17 +234,19 @@ class FSImageFormat {
           
           // get quota only when the node is a directory
           long nsQuota = -1L;
-          if (imgVersion <= -16 && blocks == null  && numBlocks == -1) {
+          if (LayoutVersion.supports(Feature.NAMESPACE_QUOTA, imgVersion)
+              && blocks == null && numBlocks == -1) {
             nsQuota = in.readLong();
           }
           long dsQuota = -1L;
-          if (imgVersion <= -18 && blocks == null && numBlocks == -1) {
+          if (LayoutVersion.supports(Feature.DISKSPACE_QUOTA, imgVersion)
+              && blocks == null && numBlocks == -1) {
             dsQuota = in.readLong();
           }
 
           // Read the symlink only when the node is a symlink
           String symlink = "";
-          if (imgVersion <= -23 && numBlocks == -2) {
+          if (numBlocks == -2) {
             symlink = Text.readString(in);
           }
           
@@ -338,7 +342,7 @@ class FSImageFormat {
 
     private void loadSecretManagerState(DataInputStream in, 
         FSNamesystem fs) throws IOException {
-      if (imgVersion > -23) {
+      if (!LayoutVersion.supports(Feature.DELEGATION_TOKEN, imgVersion)) {
         //SecretManagerState is not available.
         //This must not happen if security is turned on.
         return; 
@@ -348,7 +352,7 @@ class FSImageFormat {
 
 
     private long readNumFiles(DataInputStream in) throws IOException {
-      if (imgVersion <= -16) {
+      if (LayoutVersion.supports(Feature.NAMESPACE_QUOTA, imgVersion)) {
         return in.readLong();
       } else {
         return in.readInt();

Modified: hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/tools/offlineImageViewer/ImageLoaderCurrent.java
URL: http://svn.apache.org/viewvc/hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/tools/offlineImageViewer/ImageLoaderCurrent.java?rev=1130680&r1=1130679&r2=1130680&view=diff
==============================================================================
--- hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/tools/offlineImageViewer/ImageLoaderCurrent.java (original)
+++ hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/tools/offlineImageViewer/ImageLoaderCurrent.java Thu Jun  2 17:39:14 2011
@@ -26,6 +26,8 @@ import java.util.Date;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.permission.FsPermission;
 import org.apache.hadoop.hdfs.protocol.DatanodeInfo.AdminStates;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion.Feature;
 import org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier;
 import org.apache.hadoop.hdfs.server.namenode.FSImage;
 import org.apache.hadoop.hdfs.server.namenode.FSImageSerialization;
@@ -156,7 +158,7 @@ class ImageLoaderCurrent implements Imag
 
       v.visit(ImageElement.GENERATION_STAMP, in.readLong());
 
-      if (imageVersion <= -25) {
+      if (LayoutVersion.supports(Feature.FSIMAGE_COMPRESSION, imageVersion)) {
         boolean isCompressed = in.readBoolean();
         v.visit(ImageElement.IS_COMPRESSED, imageVersion);
         if (isCompressed) {
@@ -176,7 +178,7 @@ class ImageLoaderCurrent implements Imag
 
       processINodesUC(in, v, skipBlocks);
 
-      if (imageVersion <= -24) {
+      if (LayoutVersion.supports(Feature.DELEGATION_TOKEN, imageVersion)) {
         processDelegationTokens(in, v);
       }
       
@@ -340,7 +342,7 @@ class ImageLoaderCurrent implements Imag
       v.visit(ImageElement.INODE_PATH, FSImageSerialization.readString(in));
       v.visit(ImageElement.REPLICATION, in.readShort());
       v.visit(ImageElement.MODIFICATION_TIME, formatDate(in.readLong()));
-      if(imageVersion <= -17) // added in version -17
+      if(LayoutVersion.supports(Feature.FILE_ACCESS_TIME, imageVersion))
         v.visit(ImageElement.ACCESS_TIME, formatDate(in.readLong()));
       v.visit(ImageElement.BLOCK_SIZE, in.readLong());
       int numBlocks = in.readInt();
@@ -350,10 +352,10 @@ class ImageLoaderCurrent implements Imag
       // File or directory
       if (numBlocks > 0 || numBlocks == -1) {
         v.visit(ImageElement.NS_QUOTA, numBlocks == -1 ? in.readLong() : -1);
-        if(imageVersion <= -18) // added in version -18
+        if (LayoutVersion.supports(Feature.DISKSPACE_QUOTA, imageVersion))
           v.visit(ImageElement.DS_QUOTA, numBlocks == -1 ? in.readLong() : -1);
       }
-      if (imageVersion <= -23 && numBlocks == -2) {
+      if (numBlocks == -2) {
         v.visit(ImageElement.SYMLINK, Text.readString(in));
       }
 

Modified: hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/tools/offlineImageViewer/OfflineImageViewer.java
URL: http://svn.apache.org/viewvc/hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/tools/offlineImageViewer/OfflineImageViewer.java?rev=1130680&r1=1130679&r2=1130680&view=diff
==============================================================================
--- hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/tools/offlineImageViewer/OfflineImageViewer.java (original)
+++ hadoop/hdfs/branches/branch-0.22/src/java/org/apache/hadoop/hdfs/tools/offlineImageViewer/OfflineImageViewer.java Thu Jun  2 17:39:14 2011
@@ -47,8 +47,7 @@ public class OfflineImageViewer {
     "saving the results in OUTPUTFILE.\n" +
     "\n" +
     "The oiv utility will attempt to parse correctly formed image files\n" +
-    "and will abort fail with mal-formed image files. Currently the\n" +
-    "supports FSImage layout versions -16 through -23.\n" +
+    "and will abort fail with mal-formed image files.\n" +
     "\n" +
     "The tool works offline and does not require a running cluster in\n" +
     "order to process an image file.\n" +

Modified: hadoop/hdfs/branches/branch-0.22/src/test/hdfs/org/apache/hadoop/hdfs/UpgradeUtilities.java
URL: http://svn.apache.org/viewvc/hadoop/hdfs/branches/branch-0.22/src/test/hdfs/org/apache/hadoop/hdfs/UpgradeUtilities.java?rev=1130680&r1=1130679&r2=1130680&view=diff
==============================================================================
--- hadoop/hdfs/branches/branch-0.22/src/test/hdfs/org/apache/hadoop/hdfs/UpgradeUtilities.java (original)
+++ hadoop/hdfs/branches/branch-0.22/src/test/hdfs/org/apache/hadoop/hdfs/UpgradeUtilities.java Thu Jun  2 17:39:14 2011
@@ -34,7 +34,9 @@ import org.apache.hadoop.fs.FileUtil;
 import org.apache.hadoop.fs.LocalFileSystem;
 import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.hdfs.protocol.FSConstants;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion;
 import org.apache.hadoop.hdfs.protocol.FSConstants.SafeModeAction;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion.Feature;
 import org.apache.hadoop.hdfs.server.common.HdfsConstants.NodeType;
 import org.apache.hadoop.hdfs.server.common.HdfsConstants.StartupOption;
 

Added: hadoop/hdfs/branches/branch-0.22/src/test/hdfs/org/apache/hadoop/hdfs/protocol/TestLayoutVersion.java
URL: http://svn.apache.org/viewvc/hadoop/hdfs/branches/branch-0.22/src/test/hdfs/org/apache/hadoop/hdfs/protocol/TestLayoutVersion.java?rev=1130680&view=auto
==============================================================================
--- hadoop/hdfs/branches/branch-0.22/src/test/hdfs/org/apache/hadoop/hdfs/protocol/TestLayoutVersion.java (added)
+++ hadoop/hdfs/branches/branch-0.22/src/test/hdfs/org/apache/hadoop/hdfs/protocol/TestLayoutVersion.java Thu Jun  2 17:39:14 2011
@@ -0,0 +1,73 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.hdfs.protocol;
+
+import java.util.EnumSet;
+
+import static org.junit.Assert.*;
+import org.junit.Test;
+import org.apache.hadoop.hdfs.protocol.LayoutVersion.Feature;
+
+/**
+ * Test for {@link LayoutVersion}
+ */
+public class TestLayoutVersion {
+  
+  /**
+   * Tests to make sure a given layout version supports all the
+   * features from the ancestor
+   */
+  @Test
+  public void testFeaturesFromAncestorSupported() {
+    for (Feature f : Feature.values()) {
+      validateFeatureList(f);
+    }
+  }
+  
+  /**
+   * Test to make sure 0.20.203 supports delegation token
+   */
+  @Test
+  public void testRelease203() {
+    assertTrue(LayoutVersion.supports(Feature.DELEGATION_TOKEN, 
+        Feature.RESERVED_REL20_203.lv));
+  }
+  
+  /**
+   * Test to make sure 0.20.204 supports delegation token
+   */
+  @Test
+  public void testRelease204() {
+    assertTrue(LayoutVersion.supports(Feature.DELEGATION_TOKEN, 
+        Feature.RESERVED_REL20_204.lv));
+  }
+  
+  /**
+   * Given feature {@code f}, ensures the layout version of that feature
+   * supports all the features supported by it's ancestor.
+   */
+  private void validateFeatureList(Feature f) {
+    int lv = f.lv;
+    int ancestorLV = f.ancestorLV;
+    EnumSet<Feature> ancestorSet = LayoutVersion.map.get(ancestorLV);
+    assertNotNull(ancestorSet);
+    for (Feature  feature : ancestorSet) {
+      assertTrue(LayoutVersion.supports(feature, lv));
+    }
+  }
+}