Просмотр исходного кода

HDFS-10499. TestNameNodeMetadataConsistency#testGenerationStampInFuture Fails Intermittently. Contributed by Yiqun Lin.

Brahma Reddy Battula 8 лет назад
Родитель
Сommit
808f5d8985

+ 26 - 19
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestNameNodeMetadataConsistency.java

@@ -27,6 +27,9 @@ import org.apache.hadoop.hdfs.MiniDFSCluster.DataNodeProperties;
 import org.apache.hadoop.hdfs.protocol.ExtendedBlock;
 import org.apache.hadoop.hdfs.server.blockmanagement.BlockInfo;
 import org.apache.hadoop.hdfs.server.blockmanagement.BlockManagerTestUtil;
+import org.apache.hadoop.test.GenericTestUtils;
+
+import com.google.common.base.Supplier;
 
 import org.junit.After;
 import org.junit.Before;
@@ -34,7 +37,6 @@ import org.junit.Test;
 
 import java.io.IOException;
 import java.io.OutputStream;
-import java.util.concurrent.TimeUnit;
 
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertTrue;
@@ -73,8 +75,7 @@ public class TestNameNodeMetadataConsistency {
    * safe mode while it is in startup mode.
    */
   @Test
-  public void testGenerationStampInFuture() throws
-      IOException, InterruptedException {
+  public void testGenerationStampInFuture() throws Exception {
     cluster.waitActive();
 
     FileSystem fs = cluster.getFileSystem();
@@ -105,9 +106,7 @@ public class TestNameNodeMetadataConsistency {
         cluster.getNameNode().getNamesystem().getBlockManager());
 
     cluster.restartDataNode(dnProps);
-    waitTil(TimeUnit.SECONDS.toMillis(SCAN_WAIT));
-    cluster.triggerBlockReports();
-    waitTil(TimeUnit.SECONDS.toMillis(SCAN_WAIT));
+    waitForNumBytes(TEST_DATA_IN_FUTURE.length());
 
     // Make sure that we find all written bytes in future block
     assertEquals(TEST_DATA_IN_FUTURE.length(),
@@ -122,9 +121,7 @@ public class TestNameNodeMetadataConsistency {
    * hence we should not have positive count of Blocks in future.
    */
   @Test
-  public void testEnsureGenStampsIsStartupOnly() throws
-      IOException, InterruptedException {
-
+  public void testEnsureGenStampsIsStartupOnly() throws Exception {
     String testData = " This is test data";
     cluster.restartDataNodes();
     cluster.restartNameNodes();
@@ -153,21 +150,31 @@ public class TestNameNodeMetadataConsistency {
     cluster.getNameNode().getNamesystem().writeUnlock();
 
     cluster.restartDataNode(dnProps);
-    waitTil(TimeUnit.SECONDS.toMillis(SCAN_WAIT));
-    cluster.triggerBlockReports();
-    waitTil(TimeUnit.SECONDS.toMillis(SCAN_WAIT));
-
+    waitForNumBytes(0);
 
     // Make sure that there are no bytes in future since isInStartupSafe
     // mode is not true.
     assertEquals(0, cluster.getNameNode().getBytesWithFutureGenerationStamps());
   }
 
-  private void waitTil(long waitPeriod) {
-    try {
-      Thread.sleep(waitPeriod);
-    } catch (InterruptedException e) {
-      e.printStackTrace();
-    }
+  private void waitForNumBytes(final int numBytes) throws Exception {
+    GenericTestUtils.waitFor(new Supplier<Boolean>() {
+
+      @Override
+      public Boolean get() {
+        try {
+          cluster.triggerBlockReports();
+          // Compare the number of bytes
+          if (cluster.getNameNode().getBytesWithFutureGenerationStamps()
+              == numBytes) {
+            return true;
+          }
+        } catch (Exception e) {
+          // Ignore the exception
+        }
+
+        return false;
+      }
+    }, SCAN_WAIT * 1000, 60000);
   }
 }