Browse Source

HADOOP-2392 TestRegionServerExit has new failure mode since HADOOP-2338

git-svn-id: https://svn.apache.org/repos/asf/lucene/hadoop/trunk@602790 13f79535-47bb-0310-9956-ffa450edef68
Jim Kellerman 17 years ago
parent
commit
83c8efb12e

+ 1 - 1
src/contrib/hbase/CHANGES.txt

@@ -103,7 +103,7 @@ Trunk (unreleased changes)
    HADOOP-2333 Client side retries happen at the wrong level
    HADOOP-2357 Compaction cleanup; less deleting + prevent possible file leaks
    HADOOP-2377 Holding open MapFile.Readers is expensive, so use less of them
-
+   HADOOP-2392 TestRegionServerExit has new failure mode since HADOOP-2338
 
 Release 0.15.1
 Branch 0.15

+ 22 - 6
src/contrib/hbase/src/java/org/apache/hadoop/hbase/HMaster.java

@@ -1111,17 +1111,15 @@ public class HMaster extends Thread implements HConstants, HMasterInterface,
     } catch (Throwable t) {
       LOG.fatal("Unhandled exception", t);
     }
+    // The region servers won't all exit until we stop scanning the meta regions
+    stopScanners();
+    
+    // Wait for all the remaining region servers to report in.
     letRegionServersShutdown();
 
     /*
      * Clean up and close up shop
      */
-    synchronized(rootScannerLock) {
-      rootScannerThread.interrupt();    // Wake root scanner
-    }
-    synchronized(metaScannerLock) {
-      metaScannerThread.interrupt();    // Wake meta scanner
-    }
     if (this.infoServer != null) {
       LOG.info("Stopping infoServer");
       try {
@@ -1192,6 +1190,23 @@ public class HMaster extends Thread implements HConstants, HMasterInterface,
       LOG.error("Failed startup", e);
     }
   }
+  
+  /*
+   * Stop the root and meta scanners so that the region servers serving meta
+   * regions can shut down.
+   */
+  private void stopScanners() {
+    synchronized(rootScannerLock) {
+      if (rootScannerThread.isAlive()) {
+        rootScannerThread.interrupt();  // Wake root scanner
+      }
+    }
+    synchronized(metaScannerLock) {
+      if (metaScannerThread.isAlive()) {
+        metaScannerThread.interrupt();  // Wake meta scanner
+      }
+    }
+  }
 
   /*
    * Wait on regionservers to report in
@@ -1348,6 +1363,7 @@ public class HMaster extends Thread implements HConstants, HMasterInterface,
           // proceed with shutdown
           LOG.info("All user tables quiesced. Proceeding with shutdown");
           closed.set(true);
+          stopScanners();
           synchronized(toDoQueue) {
             toDoQueue.clear();                         // Empty the queue
             delayedToDoQueue.clear();                  // Empty shut down queue

+ 0 - 6
src/contrib/hbase/src/java/org/apache/hadoop/hbase/HStore.java

@@ -2142,12 +2142,6 @@ class HStore implements HConstants {
           }
           if (dataFilter.filterAllRemaining()) {
             moreToFollow = false;
-            LOG.debug("page limit");
-          }
-        }
-        if (LOG.isDebugEnabled()) {
-          if (this.dataFilter != null) {
-            LOG.debug("ROWKEY = " + chosenRow + ", FILTERED = " + filtered);
           }
         }
         

+ 3 - 8
src/contrib/hbase/src/test/org/apache/hadoop/hbase/mapred/TestTableMapReduce.java

@@ -401,9 +401,6 @@ public class TestTableMapReduce extends MultiRegionTable {
 
         } else {
           first = new String(firstValue, HConstants.UTF8_ENCODING);
-          if (LOG.isDebugEnabled()) {
-            LOG.debug("row=" + key.getRow() + ": first value=" + first);
-          }
         }
         
         String second = "";
@@ -414,10 +411,6 @@ public class TestTableMapReduce extends MultiRegionTable {
           fail();
           
         } else {
-          if (LOG.isDebugEnabled()) {
-            LOG.debug("row=" + key.getRow() + ": second value=" +
-                new String(secondValue, HConstants.UTF8_ENCODING));
-          }
           byte[] secondReversed = new byte[secondValue.length];
           for (int i = 0, j = secondValue.length - 1; j >= 0; j--, i++) {
             secondReversed[i] = secondValue[j];
@@ -426,7 +419,9 @@ public class TestTableMapReduce extends MultiRegionTable {
         }
         if (first.compareTo(second) != 0) {
           if (LOG.isDebugEnabled()) {
-            LOG.debug("second key is not the reverse of first");
+            LOG.debug("second key is not the reverse of first. row=" +
+                key.getRow() + ", first value=" + first + ", second value=" +
+                second);
           }
           fail();
         }

+ 20 - 22
src/contrib/hbase/src/test/org/apache/hadoop/hbase/shell/algebra/TestTableJoinMapReduce.java

@@ -35,14 +35,19 @@ import org.apache.hadoop.hbase.HScannerInterface;
 import org.apache.hadoop.hbase.HStoreKey;
 import org.apache.hadoop.hbase.HTable;
 import org.apache.hadoop.hbase.HTableDescriptor;
+import org.apache.hadoop.hbase.StaticTestEnvironment;
 import org.apache.hadoop.hbase.MasterNotRunningException;
 import org.apache.hadoop.hbase.MiniHBaseCluster;
 import org.apache.hadoop.hbase.MultiRegionTable;
+import org.apache.hadoop.hbase.mapred.TableReduce;
 import org.apache.hadoop.io.Text;
 import org.apache.hadoop.mapred.JobClient;
 import org.apache.hadoop.mapred.JobConf;
 import org.apache.hadoop.mapred.MiniMRCluster;
 
+/**
+ * HBase shell join test
+ */
 public class TestTableJoinMapReduce extends MultiRegionTable {
   @SuppressWarnings("hiding")
   private static final Log LOG = LogFactory.getLog(TestTableJoinMapReduce.class
@@ -64,7 +69,6 @@ public class TestTableJoinMapReduce extends MultiRegionTable {
   @Override
   public void setUp() throws Exception {
     super.setUp();
-    conf.setLong("hbase.hregion.max.filesize", 256 * 1024);
     dfsCluster = new MiniDFSCluster(conf, 1, true, (String[]) null);
     try {
       fs = dfsCluster.getFileSystem();
@@ -73,10 +77,7 @@ public class TestTableJoinMapReduce extends MultiRegionTable {
       // Start up HBase cluster
       hCluster = new MiniHBaseCluster(conf, 1, dfsCluster);
     } catch (Exception e) {
-      if (dfsCluster != null) {
-        dfsCluster.shutdown();
-        dfsCluster = null;
-      }
+      StaticTestEnvironment.shutdownDfs(dfsCluster);
       throw e;
     }
   }
@@ -90,21 +91,11 @@ public class TestTableJoinMapReduce extends MultiRegionTable {
     if (hCluster != null) {
       hCluster.shutdown();
     }
-
-    if (dfsCluster != null) {
-      dfsCluster.shutdown();
-    }
-
-    if (fs != null) {
-      try {
-        fs.close();
-      } catch (IOException e) {
-        LOG.info("During tear down got a " + e.getMessage());
-      }
-    }
+    StaticTestEnvironment.shutdownDfs(dfsCluster);
   }
 
   public void testTableJoinMapReduce() {
+    HTable table = null;
     try {
       HTableDescriptor desc = new HTableDescriptor(FIRST_RELATION);
       String[] columns = FIRST_COLUMNS.split(" ");
@@ -115,7 +106,7 @@ public class TestTableJoinMapReduce extends MultiRegionTable {
       admin.createTable(desc);
 
       // insert random data into the input table
-      HTable table = new HTable(conf, new Text(FIRST_RELATION));
+      table = new HTable(conf, new Text(FIRST_RELATION));
       for (int j = 0; j < 5; j++) {
         long lockid = table.startUpdate(new Text("rowKey" + j));
         table.put(lockid, new Text("a:"), Integer.toString(j).getBytes(
@@ -131,6 +122,10 @@ public class TestTableJoinMapReduce extends MultiRegionTable {
       e.printStackTrace();
     } catch (IOException e) {
       e.printStackTrace();
+    } finally {
+      if (table != null) {
+        table.close();
+      }
     }
 
     try {
@@ -143,8 +138,7 @@ public class TestTableJoinMapReduce extends MultiRegionTable {
       admin.createTable(desc);
 
       // insert random data into the input table
-      HTable table = new HTable(conf, new Text(SECOND_RELATION));
-
+      table = new HTable(conf, new Text(SECOND_RELATION));
       for (int j = 0; j < 3; j++) {
         long lockid = table.startUpdate(new Text("joinKey-" + j));
         table.put(lockid, new Text("d:"), ("s-" + Integer.toString(j))
@@ -158,6 +152,10 @@ public class TestTableJoinMapReduce extends MultiRegionTable {
       e.printStackTrace();
     } catch (IOException e) {
       e.printStackTrace();
+    } finally {
+      if (table != null) {
+        table.close();
+      }
     }
 
     try {
@@ -186,7 +184,7 @@ public class TestTableJoinMapReduce extends MultiRegionTable {
 
       IndexJoinMap.initJob(FIRST_RELATION, SECOND_RELATION, FIRST_COLUMNS,
           SECOND_COLUMNS, JOIN_EXPRESSION, IndexJoinMap.class, jobConf);
-      IndexJoinReduce.initJob(OUTPUT_TABLE, IndexJoinReduce.class, jobConf);
+      TableReduce.initJob(OUTPUT_TABLE, IndexJoinReduce.class, jobConf);
 
       JobClient.runJob(jobConf);
 
@@ -197,7 +195,7 @@ public class TestTableJoinMapReduce extends MultiRegionTable {
     }
 
     try {
-      verify(conf, OUTPUT_TABLE);
+    verify(conf, OUTPUT_TABLE);
     } catch (IOException e) {
       e.printStackTrace();
     }