source: proiecte/HadoopJUnit/hadoop-0.20.1/src/test/org/apache/hadoop/hdfs/TestDFSStartupVersions.java @ 120

Last change on this file since 120 was 120, checked in by (none), 14 years ago

Added the mail files for the Hadoop JUNit Project

  • Property svn:executable set to *
File size: 9.5 KB
Line 
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 */
18package org.apache.hadoop.hdfs;
19
20import java.io.File;
21import junit.framework.TestCase;
22import org.apache.commons.logging.Log;
23import org.apache.commons.logging.LogFactory;
24import org.apache.hadoop.conf.Configuration;
25import org.apache.hadoop.hdfs.protocol.FSConstants;
26import org.apache.hadoop.hdfs.server.common.HdfsConstants.NodeType;
27import org.apache.hadoop.hdfs.server.common.HdfsConstants.StartupOption;
28
29import static org.apache.hadoop.hdfs.server.common.HdfsConstants.NodeType.NAME_NODE;
30import static org.apache.hadoop.hdfs.server.common.HdfsConstants.NodeType.DATA_NODE;
31
32import org.apache.hadoop.hdfs.server.common.HdfsConstants;
33import org.apache.hadoop.hdfs.server.common.Storage;
34import org.apache.hadoop.hdfs.server.common.StorageInfo;
35import org.apache.hadoop.fs.Path;
36
37/**
38 * This test ensures the appropriate response (successful or failure) from
39 * a Datanode when the system is started with differing version combinations.
40 */
41public class TestDFSStartupVersions extends TestCase {
42 
43  private static final Log LOG = LogFactory.getLog(
44                                                   "org.apache.hadoop.hdfs.TestDFSStartupVersions");
45  private static Path TEST_ROOT_DIR = new Path(
46                                               System.getProperty("test.build.data","/tmp").toString().replace(' ', '+'));
47  private MiniDFSCluster cluster = null;
48 
49  /**
50   * Writes an INFO log message containing the parameters.
51   */
52  void log(String label, NodeType nodeType, Integer testCase, StorageInfo version) {
53    String testCaseLine = "";
54    if (testCase != null) {
55      testCaseLine = " testCase="+testCase;
56    }
57    LOG.info("============================================================");
58    LOG.info("***TEST*** " + label + ":"
59             + testCaseLine
60             + " nodeType="+nodeType
61             + " layoutVersion="+version.getLayoutVersion()
62             + " namespaceID="+version.getNamespaceID()
63             + " fsscTime="+version.getCTime());
64  }
65 
66  /**
67   * Initialize the versions array.  This array stores all combinations
68   * of cross product:
69   *  {oldLayoutVersion,currentLayoutVersion,futureLayoutVersion} X
70   *    {currentNamespaceId,incorrectNamespaceId} X
71   *      {pastFsscTime,currentFsscTime,futureFsscTime}
72   */
73  private StorageInfo[] initializeVersions() throws Exception {
74    int layoutVersionOld = Storage.LAST_UPGRADABLE_LAYOUT_VERSION;
75    int layoutVersionCur = UpgradeUtilities.getCurrentLayoutVersion();
76    int layoutVersionNew = Integer.MIN_VALUE;
77    int namespaceIdCur = UpgradeUtilities.getCurrentNamespaceID(null);
78    int namespaceIdOld = Integer.MIN_VALUE;
79    long fsscTimeOld = Long.MIN_VALUE;
80    long fsscTimeCur = UpgradeUtilities.getCurrentFsscTime(null);
81    long fsscTimeNew = Long.MAX_VALUE;
82   
83    return new StorageInfo[] {
84      new StorageInfo(layoutVersionOld, namespaceIdCur, fsscTimeOld), // 0
85      new StorageInfo(layoutVersionOld, namespaceIdCur, fsscTimeCur), // 1
86      new StorageInfo(layoutVersionOld, namespaceIdCur, fsscTimeNew), // 2
87      new StorageInfo(layoutVersionOld, namespaceIdOld, fsscTimeOld), // 3
88      new StorageInfo(layoutVersionOld, namespaceIdOld, fsscTimeCur), // 4
89      new StorageInfo(layoutVersionOld, namespaceIdOld, fsscTimeNew), // 5
90      new StorageInfo(layoutVersionCur, namespaceIdCur, fsscTimeOld), // 6
91      new StorageInfo(layoutVersionCur, namespaceIdCur, fsscTimeCur), // 7
92      new StorageInfo(layoutVersionCur, namespaceIdCur, fsscTimeNew), // 8
93      new StorageInfo(layoutVersionCur, namespaceIdOld, fsscTimeOld), // 9
94      new StorageInfo(layoutVersionCur, namespaceIdOld, fsscTimeCur), // 10
95      new StorageInfo(layoutVersionCur, namespaceIdOld, fsscTimeNew), // 11
96      new StorageInfo(layoutVersionNew, namespaceIdCur, fsscTimeOld), // 12
97      new StorageInfo(layoutVersionNew, namespaceIdCur, fsscTimeCur), // 13
98      new StorageInfo(layoutVersionNew, namespaceIdCur, fsscTimeNew), // 14
99      new StorageInfo(layoutVersionNew, namespaceIdOld, fsscTimeOld), // 15
100      new StorageInfo(layoutVersionNew, namespaceIdOld, fsscTimeCur), // 16
101      new StorageInfo(layoutVersionNew, namespaceIdOld, fsscTimeNew), // 17
102    };
103  }
104 
105  /**
106   * Determines if the given Namenode version and Datanode version
107   * are compatible with each other. Compatibility in this case mean
108   * that the Namenode and Datanode will successfully start up and
109   * will work together. The rules for compatibility,
110   * taken from the DFS Upgrade Design, are as follows:
111   * <pre>
112   * 1. The data-node does regular startup (no matter which options
113   *    it is started with) if
114   *       softwareLV == storedLV AND
115   *       DataNode.FSSCTime == NameNode.FSSCTime
116   * 2. The data-node performs an upgrade if it is started without any
117   *    options and
118   *       |softwareLV| > |storedLV| OR
119   *       (softwareLV == storedLV AND
120   *        DataNode.FSSCTime < NameNode.FSSCTime)
121   * 3. NOT TESTED: The data-node rolls back if it is started with
122   *    the -rollback option and
123   *       |softwareLV| >= |previous.storedLV| AND
124   *       DataNode.previous.FSSCTime <= NameNode.FSSCTime
125   * 4. In all other cases the startup fails.
126   * </pre>
127   */
128  boolean isVersionCompatible(StorageInfo namenodeVer, StorageInfo datanodeVer) {
129    // check #0
130    if (namenodeVer.getNamespaceID() != datanodeVer.getNamespaceID()) {
131      LOG.info("namespaceIDs are not equal: isVersionCompatible=false");
132      return false;
133    }
134    // check #1
135    int softwareLV = FSConstants.LAYOUT_VERSION;  // will also be Namenode's LV
136    int storedLV = datanodeVer.getLayoutVersion();
137    if (softwareLV == storedLV && 
138        datanodeVer.getCTime() == namenodeVer.getCTime()) 
139      {
140        LOG.info("layoutVersions and cTimes are equal: isVersionCompatible=true");
141        return true;
142      }
143    // check #2
144    long absSoftwareLV = Math.abs((long)softwareLV);
145    long absStoredLV = Math.abs((long)storedLV);
146    if (absSoftwareLV > absStoredLV ||
147        (softwareLV == storedLV &&
148         datanodeVer.getCTime() < namenodeVer.getCTime())) 
149      {
150        LOG.info("softwareLayoutVersion is newer OR namenode cTime is newer: isVersionCompatible=true");
151        return true;
152      }
153    // check #4
154    LOG.info("default case: isVersionCompatible=false");
155    return false;
156  }
157 
158  /**
159   * This test ensures the appropriate response (successful or failure) from
160   * a Datanode when the system is started with differing version combinations.
161   * <pre>
162   * For each 3-tuple in the cross product
163   *   ({oldLayoutVersion,currentLayoutVersion,futureLayoutVersion},
164   *    {currentNamespaceId,incorrectNamespaceId},
165   *    {pastFsscTime,currentFsscTime,futureFsscTime})
166   *      1. Startup Namenode with version file containing
167   *         (currentLayoutVersion,currentNamespaceId,currentFsscTime)
168   *      2. Attempt to startup Datanode with version file containing
169   *         this iterations version 3-tuple
170   * </pre>
171   */
172  public void testVersions() throws Exception {
173    UpgradeUtilities.initialize();
174    Configuration conf = UpgradeUtilities.initializeStorageStateConf(1, 
175                                                      new Configuration());
176    StorageInfo[] versions = initializeVersions();
177    UpgradeUtilities.createStorageDirs(
178                                       NAME_NODE, conf.getStrings("dfs.name.dir"), "current");
179    cluster = new MiniDFSCluster(conf, 0, StartupOption.REGULAR);
180    StorageInfo nameNodeVersion = new StorageInfo(
181                                                  UpgradeUtilities.getCurrentLayoutVersion(),
182                                                  UpgradeUtilities.getCurrentNamespaceID(cluster),
183                                                  UpgradeUtilities.getCurrentFsscTime(cluster));
184    log("NameNode version info", NAME_NODE, null, nameNodeVersion);
185    for (int i = 0; i < versions.length; i++) {
186      File[] storage = UpgradeUtilities.createStorageDirs(
187                                                          DATA_NODE, conf.getStrings("dfs.data.dir"), "current");
188      log("DataNode version info", DATA_NODE, i, versions[i]);
189      UpgradeUtilities.createVersionFile(DATA_NODE, storage, versions[i]);
190      try {
191        cluster.startDataNodes(conf, 1, false, StartupOption.REGULAR, null);
192      } catch (Exception ignore) {
193        // Ignore.  The asserts below will check for problems.
194        // ignore.printStackTrace();
195      }
196      assertTrue(cluster.getNameNode() != null);
197      assertEquals(isVersionCompatible(nameNodeVersion, versions[i]),
198                   cluster.isDataNodeUp());
199      cluster.shutdownDataNodes();
200    }
201  }
202 
203  protected void tearDown() throws Exception {
204    LOG.info("Shutting down MiniDFSCluster");
205    if (cluster != null) cluster.shutdown();
206  }
207 
208  public static void main(String[] args) throws Exception {
209    new TestDFSStartupVersions().testVersions();
210  }
211 
212}
213
Note: See TracBrowser for help on using the repository browser.