001/**
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hdfs.protocol;
019
020import java.util.Comparator;
021import java.util.Map;
022import java.util.SortedSet;
023import java.util.TreeSet;
024
025import org.apache.hadoop.classification.InterfaceAudience;
026
027/**
028 * This class tracks changes in the layout version of HDFS.
029 * 
030 * Layout version is changed for following reasons:
031 * <ol>
032 * <li>The layout of how namenode or datanode stores information 
033 * on disk changes.</li>
034 * <li>A new operation code is added to the editlog.</li>
035 * <li>Modification such as format of a record, content of a record 
036 * in editlog or fsimage.</li>
037 * </ol>
038 * <br>
039 * <b>How to update layout version:<br></b>
040 * When a change requires new layout version, please add an entry into
041 * {@link Feature} with a short enum name, new layout version and description
042 * of the change. Please see {@link Feature} for further details.
043 * <br>
044 */
045@InterfaceAudience.Private
046public class LayoutVersion {
047  /**
048   * Version in which HDFS-2991 was fixed. This bug caused OP_ADD to
049   * sometimes be skipped for append() calls. If we see such a case when
050   * loading the edits, but the version is known to have that bug, we
051   * workaround the issue. Otherwise we should consider it a corruption
052   * and bail.
053   */
054  public static final int BUGFIX_HDFS_2991_VERSION = -40;
055
056  /**
057   * The interface to be implemented by NameNode and DataNode layout features 
058   */
059  public interface LayoutFeature {
060    public FeatureInfo getInfo();
061  }
062
063  /**
064   * Enums for features that change the layout version before rolling
065   * upgrade is supported.
066   * <br><br>
067   * To add a new layout version:
068   * <ul>
069   * <li>Define a new enum constant with a short enum name, the new layout version 
070   * and description of the added feature.</li>
071   * <li>When adding a layout version with an ancestor that is not same as
072   * its immediate predecessor, use the constructor where a specific ancestor
073   * can be passed.
074   * </li>
075   * </ul>
076   */
077  public static enum Feature implements LayoutFeature {
078    NAMESPACE_QUOTA(-16, "Support for namespace quotas"),
079    FILE_ACCESS_TIME(-17, "Support for access time on files"),
080    DISKSPACE_QUOTA(-18, "Support for disk space quotas"),
081    STICKY_BIT(-19, "Support for sticky bits"),
082    APPEND_RBW_DIR(-20, "Datanode has \"rbw\" subdirectory for append"),
083    ATOMIC_RENAME(-21, "Support for atomic rename"),
084    CONCAT(-22, "Support for concat operation"),
085    SYMLINKS(-23, "Support for symbolic links"),
086    DELEGATION_TOKEN(-24, "Support for delegation tokens for security"),
087    FSIMAGE_COMPRESSION(-25, "Support for fsimage compression"),
088    FSIMAGE_CHECKSUM(-26, "Support checksum for fsimage"),
089    REMOVE_REL13_DISK_LAYOUT_SUPPORT(-27, "Remove support for 0.13 disk layout"),
090    EDITS_CHECKSUM(-28, "Support checksum for editlog"),
091    UNUSED(-29, "Skipped version"),
092    FSIMAGE_NAME_OPTIMIZATION(-30, "Store only last part of path in fsimage"),
093    RESERVED_REL20_203(-31, -19, "Reserved for release 0.20.203", true,
094        DELEGATION_TOKEN),
095    RESERVED_REL20_204(-32, -31, "Reserved for release 0.20.204", true),
096    RESERVED_REL22(-33, -27, "Reserved for release 0.22", true),
097    RESERVED_REL23(-34, -30, "Reserved for release 0.23", true),
098    FEDERATION(-35, "Support for namenode federation"),
099    LEASE_REASSIGNMENT(-36, "Support for persisting lease holder reassignment"),
100    STORED_TXIDS(-37, "Transaction IDs are stored in edits log and image files"),
101    TXID_BASED_LAYOUT(-38, "File names in NN Storage are based on transaction IDs"), 
102    EDITLOG_OP_OPTIMIZATION(-39,
103        "Use LongWritable and ShortWritable directly instead of ArrayWritable of UTF8"),
104    OPTIMIZE_PERSIST_BLOCKS(-40,
105        "Serialize block lists with delta-encoded variable length ints, " +
106        "add OP_UPDATE_BLOCKS"),
107    RESERVED_REL1_2_0(-41, -32, "Reserved for release 1.2.0", true, CONCAT),
108    ADD_INODE_ID(-42, -40, "Assign a unique inode id for each inode", false),
109    SNAPSHOT(-43, "Support for snapshot feature"),
110    RESERVED_REL1_3_0(-44, -41, "Reserved for release 1.3.0", true,
111                ADD_INODE_ID, SNAPSHOT, FSIMAGE_NAME_OPTIMIZATION),
112    OPTIMIZE_SNAPSHOT_INODES(-45, -43,
113        "Reduce snapshot inode memory footprint", false),
114    SEQUENTIAL_BLOCK_ID(-46, "Allocate block IDs sequentially and store " +
115        "block IDs in the edits log and image files"),
116    EDITLOG_SUPPORT_RETRYCACHE(-47, "Record ClientId and CallId in editlog to " 
117        + "enable rebuilding retry cache in case of HA failover"),
118    EDITLOG_ADD_BLOCK(-48, "Add new editlog that only records allocation of "
119        + "the new block instead of the entire block list"),
120    ADD_DATANODE_AND_STORAGE_UUIDS(-49, "Replace StorageID with DatanodeUuid."
121        + " Use distinct StorageUuid per storage directory."),
122    ADD_LAYOUT_FLAGS(-50, "Add support for layout flags."),
123    CACHING(-51, "Support for cache pools and path-based caching"),
124    // Hadoop 2.4.0
125    PROTOBUF_FORMAT(-52, "Use protobuf to serialize FSImage"),
126    EXTENDED_ACL(-53, "Extended ACL"),
127    RESERVED_REL2_4_0(-54, -51, "Reserved for release 2.4.0", true,
128        PROTOBUF_FORMAT, EXTENDED_ACL);
129
130    private final FeatureInfo info;
131
132    /**
133     * Feature that is added at layout version {@code lv} - 1. 
134     * @param lv new layout version with the addition of this feature
135     * @param description description of the feature
136     */
137    Feature(final int lv, final String description) {
138      this(lv, lv + 1, description, false);
139    }
140
141    /**
142     * Feature that is added at layout version {@code ancestoryLV}.
143     * @param lv new layout version with the addition of this feature
144     * @param ancestorLV layout version from which the new lv is derived from.
145     * @param description description of the feature
146     * @param reserved true when this is a layout version reserved for previous
147     *        version
148     * @param features set of features that are to be enabled for this version
149     */
150    Feature(final int lv, final int ancestorLV, final String description,
151        boolean reserved, Feature... features) {
152      info = new FeatureInfo(lv, ancestorLV, description, reserved, features);
153    }
154    
155    @Override
156    public FeatureInfo getInfo() {
157      return info;
158    }
159  }
160  
161  /** Feature information. */
162  public static class FeatureInfo {
163    private final int lv;
164    private final int ancestorLV;
165    private final Integer minCompatLV;
166    private final String description;
167    private final boolean reserved;
168    private final LayoutFeature[] specialFeatures;
169
170    public FeatureInfo(final int lv, final int ancestorLV, final String description,
171        boolean reserved, LayoutFeature... specialFeatures) {
172      this(lv, ancestorLV, null, description, reserved, specialFeatures);
173    }
174
175    public FeatureInfo(final int lv, final int ancestorLV, Integer minCompatLV,
176        final String description, boolean reserved,
177        LayoutFeature... specialFeatures) {
178      this.lv = lv;
179      this.ancestorLV = ancestorLV;
180      this.minCompatLV = minCompatLV;
181      this.description = description;
182      this.reserved = reserved;
183      this.specialFeatures = specialFeatures;
184    }
185    
186    /** 
187     * Accessor method for feature layout version 
188     * @return int lv value
189     */
190    public int getLayoutVersion() {
191      return lv;
192    }
193
194    /** 
195     * Accessor method for feature ancestor layout version 
196     * @return int ancestor LV value
197     */
198    public int getAncestorLayoutVersion() {
199      return ancestorLV;
200    }
201
202    /**
203     * Accessor method for feature minimum compatible layout version.  If the
204     * feature does not define a minimum compatible layout version, then this
205     * method returns the feature's own layout version.  This would indicate
206     * that the feature cannot provide compatibility with any prior layout
207     * version.
208     *
209     * @return int minimum compatible LV value
210     */
211    public int getMinimumCompatibleLayoutVersion() {
212      return minCompatLV != null ? minCompatLV : lv;
213    }
214
215    /**
216     * Accessor method for feature description 
217     * @return String feature description 
218     */
219    public String getDescription() {
220      return description;
221    }
222    
223    public boolean isReservedForOldRelease() {
224      return reserved;
225    }
226    
227    public LayoutFeature[] getSpecialFeatures() {
228      return specialFeatures;
229    }
230  }
231
232  static class LayoutFeatureComparator implements Comparator<LayoutFeature> {
233    @Override
234    public int compare(LayoutFeature arg0, LayoutFeature arg1) {
235      return arg0.getInfo().getLayoutVersion()
236          - arg1.getInfo().getLayoutVersion();
237    }
238  }
239 
240  public static void updateMap(Map<Integer, SortedSet<LayoutFeature>> map,
241      LayoutFeature[] features) {
242    // Go through all the enum constants and build a map of
243    // LayoutVersion <-> Set of all supported features in that LayoutVersion
244    SortedSet<LayoutFeature> existingFeatures = new TreeSet<LayoutFeature>(
245        new LayoutFeatureComparator());
246    for (SortedSet<LayoutFeature> s : map.values()) {
247      existingFeatures.addAll(s);
248    }
249    LayoutFeature prevF = existingFeatures.isEmpty() ? null :
250        existingFeatures.first();
251    for (LayoutFeature f : features) {
252      final FeatureInfo info = f.getInfo();
253      int minCompatLV = info.getMinimumCompatibleLayoutVersion();
254      if (prevF != null &&
255          minCompatLV > prevF.getInfo().getMinimumCompatibleLayoutVersion()) {
256        throw new AssertionError(String.format(
257            "Features must be listed in order of minimum compatible layout " +
258            "version.  Check features %s and %s.", prevF, f));
259      }
260      prevF = f;
261      SortedSet<LayoutFeature> ancestorSet = map.get(info.getAncestorLayoutVersion());
262      if (ancestorSet == null) {
263        // Empty set
264        ancestorSet = new TreeSet<LayoutFeature>(new LayoutFeatureComparator());
265        map.put(info.getAncestorLayoutVersion(), ancestorSet);
266      }
267      SortedSet<LayoutFeature> featureSet = new TreeSet<LayoutFeature>(ancestorSet);
268      if (info.getSpecialFeatures() != null) {
269        for (LayoutFeature specialFeature : info.getSpecialFeatures()) {
270          featureSet.add(specialFeature);
271        }
272      }
273      featureSet.add(f);
274      map.put(info.getLayoutVersion(), featureSet);
275    }
276  }
277  
278  /**
279   * Gets formatted string that describes {@link LayoutVersion} information.
280   */
281  public String getString(Map<Integer, SortedSet<LayoutFeature>> map,
282      LayoutFeature[] values) {
283    final StringBuilder buf = new StringBuilder();
284    buf.append("Feature List:\n");
285    for (LayoutFeature f : values) {
286      final FeatureInfo info = f.getInfo();
287      buf.append(f).append(" introduced in layout version ")
288          .append(info.getLayoutVersion()).append(" (")
289          .append(info.getDescription()).append(")\n");
290    }
291
292    buf.append("\n\nLayoutVersion and supported features:\n");
293    for (LayoutFeature f : values) {
294      final FeatureInfo info = f.getInfo();
295      buf.append(info.getLayoutVersion()).append(": ")
296          .append(map.get(info.getLayoutVersion())).append("\n");
297    }
298    return buf.toString();
299  }
300  
301  /**
302   * Returns true if a given feature is supported in the given layout version
303   * @param map layout feature map
304   * @param f Feature
305   * @param lv LayoutVersion
306   * @return true if {@code f} is supported in layout version {@code lv}
307   */
308  public static boolean supports(Map<Integer, SortedSet<LayoutFeature>> map,
309      final LayoutFeature f, final int lv) {
310    final SortedSet<LayoutFeature> set =  map.get(lv);
311    return set != null && set.contains(f);
312  }
313  
314  /**
315   * Get the current layout version
316   */
317  public static int getCurrentLayoutVersion(LayoutFeature[] features) {
318    return getLastNonReservedFeature(features).getInfo().getLayoutVersion();
319  }
320
321  /**
322   * Gets the minimum compatible layout version.
323   *
324   * @param features all features to check
325   * @return minimum compatible layout version
326   */
327  public static int getMinimumCompatibleLayoutVersion(
328      LayoutFeature[] features) {
329    return getLastNonReservedFeature(features).getInfo()
330        .getMinimumCompatibleLayoutVersion();
331  }
332
333  static LayoutFeature getLastNonReservedFeature(LayoutFeature[] features) {
334    for (int i = features.length -1; i >= 0; i--) {
335      final FeatureInfo info = features[i].getInfo();
336      if (!info.isReservedForOldRelease()) {
337        return features[i];
338      }
339    }
340    throw new AssertionError("All layout versions are reserved.");
341  }
342}