Browse Source

HDFS-2170. Address remaining TODOs in HDFS-1073 branch. Contributed by Todd Lipcon.

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/branches/HDFS-1073@1148589 13f79535-47bb-0310-9956-ffa450edef68
Todd Lipcon 14 years ago
parent
commit
581672c812

+ 1 - 0
hdfs/CHANGES.HDFS-1073.txt

@@ -78,3 +78,4 @@ HDFS-2135. Fix regression of HDFS-1955 in HDFS-1073 branch. (todd)
 HDFS-2160. Fix CreateEditsLog test tool in HDFS-1073 branch. (todd)
 HDFS-2168. Reenable TestEditLog.testFailedOpen and fix exposed bug. (todd)
 HDFS-2169. Clean up TestCheckpoint and remove TODOs (todd)
+HDFS-2170. Address remaining TODOs in HDFS-1073 branch. (todd)

+ 1 - 0
hdfs/src/java/org/apache/hadoop/hdfs/server/namenode/FSEditLog.java

@@ -806,6 +806,7 @@ public class FSEditLog  {
     numTransactions = totalTimeTransactions = numTransactionsBatchedInSync = 0;
 
     // TODO no need to link this back to storage anymore!
+    // See HDFS-2174.
     storage.attemptRestoreRemovedStorage();
     
     mapJournalsAndReportErrors(new JournalClosure() {

+ 0 - 1
hdfs/src/java/org/apache/hadoop/hdfs/server/namenode/FSImage.java

@@ -848,7 +848,6 @@ public class FSImage implements Closeable {
       throw new IOException(
         "Failed to save in any storage directories while saving namespace.");
     }
-    // TODO Double-check for regressions against HDFS-1505 and HDFS-1921.
 
     renameCheckpoint(txid);
     

+ 31 - 1
hdfs/src/test/hdfs/org/apache/hadoop/hdfs/server/namenode/FSImageTestUtil.java

@@ -24,6 +24,7 @@ import java.io.FileInputStream;
 import java.io.FileNotFoundException;
 import java.io.FileOutputStream;
 import java.io.IOException;
+import java.io.RandomAccessFile;
 import java.math.BigInteger;
 import java.net.URI;
 import java.security.MessageDigest;
@@ -63,7 +64,12 @@ import static org.mockito.Mockito.mock;
  */
 public abstract class FSImageTestUtil {
   
-  
+  /**
+   * The position in the fsimage header where the txid is
+   * written.
+   */
+  private static final long IMAGE_TXID_POS = 24;
+
   /**
    * This function returns a md5 hash of a file.
    * 
@@ -74,6 +80,30 @@ public abstract class FSImageTestUtil {
     return MD5FileUtils.computeMd5ForFile(file).toString();
   }
   
+  /**
+   * Calculate the md5sum of an image after zeroing out the transaction ID
+   * field in the header. This is useful for tests that want to verify
+   * that two checkpoints have identical namespaces.
+   */
+  public static String getImageFileMD5IgnoringTxId(File imageFile)
+      throws IOException {
+    File tmpFile = File.createTempFile("hadoop_imagefile_tmp", "fsimage");
+    tmpFile.deleteOnExit();
+    try {
+      Files.copy(imageFile, tmpFile);
+      RandomAccessFile raf = new RandomAccessFile(tmpFile, "rw");
+      try {
+        raf.seek(IMAGE_TXID_POS);
+        raf.writeLong(0);
+      } finally {
+        IOUtils.closeStream(raf);
+      }
+      return getFileMD5(tmpFile);
+    } finally {
+      tmpFile.delete();
+    }
+  }
+  
   public static StorageDirectory mockStorageDirectory(
       File currentDir, NameNodeDirType type) {
     // Mock the StorageDirectory interface to just point to this file

+ 0 - 2
hdfs/src/test/hdfs/org/apache/hadoop/hdfs/server/namenode/TestEditLogRace.java

@@ -303,8 +303,6 @@ public class TestEditLogRace {
         
         // The checkpoint id should be 1 less than the last written ID, since
         // the log roll writes the "BEGIN" transaction to the new log.
-        // TODO: consider making enterSafeMode actually close the edit log
-        // at that point?
         assertEquals(fsimage.getStorage().getMostRecentCheckpointTxId(),
                      editLog.getLastWrittenTxId() - 1);
 

+ 9 - 12
hdfs/src/test/hdfs/org/apache/hadoop/hdfs/server/namenode/TestParallelImageWrite.java

@@ -30,17 +30,12 @@ import org.apache.hadoop.hdfs.MiniDFSCluster;
 import org.apache.hadoop.hdfs.DFSTestUtil;
 import org.apache.hadoop.hdfs.HdfsConfiguration;
 import org.apache.hadoop.hdfs.protocol.FSConstants.SafeModeAction;
-import org.apache.hadoop.util.PureJavaCrc32;
 import org.apache.hadoop.hdfs.server.namenode.NNStorage.NameNodeDirType;
-import org.apache.hadoop.hdfs.server.namenode.NNStorage.NameNodeFile;
 
 import java.util.Collections;
-import java.util.Iterator;
 import java.util.List;
-import java.util.ArrayList;
 
 import java.io.File;
-import java.io.FileInputStream;
 
 /**
  * A JUnit test for checking if restarting DFS preserves integrity.
@@ -85,6 +80,10 @@ public class TestParallelImageWrite extends TestCase {
       if (cluster != null) { cluster.shutdown(); }
     }
     try {
+      // Force the NN to save its images on startup so long as
+      // there are any uncheckpointed txns
+      conf.setInt(DFSConfigKeys.DFS_NAMENODE_CHECKPOINT_TXNS_KEY, 1);
+
       // Here we restart the MiniDFScluster without formatting namenode
       cluster = new MiniDFSCluster.Builder(conf).format(false)
           .numDataNodes(NUM_DATANODES).build();
@@ -111,12 +110,9 @@ public class TestParallelImageWrite extends TestCase {
       fsn.setSafeMode(SafeModeAction.SAFEMODE_ENTER);
       cluster.getNameNode().saveNamespace();
       final String checkAfterModify = checkImages(fsn, numNamenodeDirs);
-      /**
-       * TODO the following assertion is no longer valid since the fsimage
-       * includes a transaction ID in its header.
-      assertFalse("Modified namespace doesn't change fsimage contents",
-          !checkAfterRestart.equals(checkAfterModify));
-       */
+      assertFalse("Modified namespace should change fsimage contents. " +
+          "was: " + checkAfterRestart + " now: " + checkAfterModify,
+          checkAfterRestart.equals(checkAfterModify));
       fsn.setSafeMode(SafeModeAction.SAFEMODE_LEAVE);
       files.cleanup(fs, dir);
     } finally {
@@ -155,7 +151,8 @@ public class TestParallelImageWrite extends TestCase {
     // Return the hash of the newest image file
     StorageDirectory firstSd = stg.dirIterator(NameNodeDirType.IMAGE).next();
     File latestImage = FSImageTestUtil.findLatestImageFile(firstSd);
-    String md5 = FSImageTestUtil.getFileMD5(latestImage);
+    String md5 = FSImageTestUtil.getImageFileMD5IgnoringTxId(latestImage);
+    System.err.println("md5 of " + latestImage + ": " + md5);
     return md5;
   }
 }

+ 3 - 1
hdfs/src/test/hdfs/org/apache/hadoop/hdfs/server/namenode/TestSaveNamespace.java

@@ -152,7 +152,9 @@ public class TestSaveNamespace {
           storage, 1);
       doThrow(new RuntimeException("Injected"))
         .when(dir).write();
-      shouldFail = true; // TODO: unfortunately this fails -- should be improved
+      // TODO: unfortunately this fails -- should be improved.
+      // See HDFS-2173.
+      shouldFail = true;
       break;
     }