You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by jm...@apache.org on 2015/02/22 21:56:01 UTC

[28/50] [abbrv] hbase git commit: HBASE-13004 Make possible to explain why HBaseTestingUtility.waitFor fails (Andrey Stepachev)

HBASE-13004 Make possible to explain why HBaseTestingUtility.waitFor fails (Andrey Stepachev)


Project: http://git-wip-us.apache.org/repos/asf/hbase/repo
Commit: http://git-wip-us.apache.org/repos/asf/hbase/commit/a3f495f2
Tree: http://git-wip-us.apache.org/repos/asf/hbase/tree/a3f495f2
Diff: http://git-wip-us.apache.org/repos/asf/hbase/diff/a3f495f2

Branch: refs/heads/hbase-11339
Commit: a3f495f20d492ab491f2af4759b2aef5d5c3bd74
Parents: 3d692cf
Author: tedyu <yu...@gmail.com>
Authored: Tue Feb 10 07:33:52 2015 -0800
Committer: tedyu <yu...@gmail.com>
Committed: Tue Feb 10 07:33:52 2015 -0800

----------------------------------------------------------------------
 .../java/org/apache/hadoop/hbase/Waiter.java    |  40 +++-
 .../hadoop/hbase/HBaseTestingUtility.java       | 206 ++++++++++++-------
 .../security/access/TestAccessController.java   |   8 +-
 .../hbase/snapshot/SnapshotTestingUtils.java    |   8 +-
 4 files changed, 175 insertions(+), 87 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/hbase/blob/a3f495f2/hbase-common/src/test/java/org/apache/hadoop/hbase/Waiter.java
----------------------------------------------------------------------
diff --git a/hbase-common/src/test/java/org/apache/hadoop/hbase/Waiter.java b/hbase-common/src/test/java/org/apache/hadoop/hbase/Waiter.java
index 3453baf..1110f20 100644
--- a/hbase-common/src/test/java/org/apache/hadoop/hbase/Waiter.java
+++ b/hbase-common/src/test/java/org/apache/hadoop/hbase/Waiter.java
@@ -21,13 +21,13 @@ package org.apache.hadoop.hbase;
 
 import java.text.MessageFormat;
 
-import junit.framework.Assert;
-
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.hbase.classification.InterfaceAudience;
 
+import static org.junit.Assert.fail;
+
 /**
  * A class that provides a standard waitFor pattern
  * See details at https://issues.apache.org/jira/browse/HBASE-7384
@@ -98,6 +98,21 @@ public final class Waiter {
   }
 
   /**
+   * A mixin interface, can be used with {@link Waiter} to explain failed state.
+   */
+  @InterfaceAudience.Private
+  public interface ExplainingPredicate<E extends Exception> extends Predicate<E> {
+
+    /**
+     * Perform a predicate evaluation.
+     *
+     * @return explanation of failed state
+     */
+    String explainFailure() throws E;
+
+  }
+
+  /**
    * Makes the current thread sleep for the duration equal to the specified time in milliseconds
    * multiplied by the {@link #getWaitForRatio(Configuration)}.
    * @param conf the configuration
@@ -190,9 +205,13 @@ public final class Waiter {
           LOG.warn(MessageFormat.format("Waiting interrupted after [{0}] msec",
             System.currentTimeMillis() - started));
         } else if (failIfTimeout) {
-          Assert.fail(MessageFormat.format("Waiting timed out after [{0}] msec", adjustedTimeout));
+          String msg = getExplanation(predicate);
+          fail(MessageFormat
+              .format("Waiting timed out after [{0}] msec" + msg, adjustedTimeout));
         } else {
-          LOG.warn(MessageFormat.format("Waiting timed out after [{0}] msec", adjustedTimeout));
+          String msg = getExplanation(predicate);
+          LOG.warn(
+              MessageFormat.format("Waiting timed out after [{0}] msec" + msg, adjustedTimeout));
         }
       }
       return (eval || interrupted) ? (System.currentTimeMillis() - started) : -1;
@@ -201,4 +220,17 @@ public final class Waiter {
     }
   }
 
+  public static String getExplanation(Predicate explain) {
+    if (explain instanceof ExplainingPredicate) {
+      try {
+        return " " + ((ExplainingPredicate) explain).explainFailure();
+      } catch (Exception e) {
+        LOG.error("Failed to get explanation, ", e);
+        return e.getMessage();
+      }
+    } else {
+      return "";
+    }
+  }
+
 }

http://git-wip-us.apache.org/repos/asf/hbase/blob/a3f495f2/hbase-server/src/test/java/org/apache/hadoop/hbase/HBaseTestingUtility.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/HBaseTestingUtility.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/HBaseTestingUtility.java
index 8613276..37114b9 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/HBaseTestingUtility.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/HBaseTestingUtility.java
@@ -17,10 +17,7 @@
  */
 package org.apache.hadoop.hbase;
 
-import static org.junit.Assert.assertEquals;
-import static org.junit.Assert.assertTrue;
-import static org.junit.Assert.fail;
-
+import javax.annotation.Nullable;
 import java.io.File;
 import java.io.IOException;
 import java.io.OutputStream;
@@ -44,6 +41,7 @@ import java.util.Set;
 import java.util.TreeSet;
 import java.util.UUID;
 import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicReference;
 
 import org.apache.commons.lang.RandomStringUtils;
 import org.apache.commons.logging.Log;
@@ -53,6 +51,7 @@ import org.apache.commons.logging.impl.Log4JLogger;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.Path;
+import org.apache.hadoop.hbase.Waiter.ExplainingPredicate;
 import org.apache.hadoop.hbase.Waiter.Predicate;
 import org.apache.hadoop.hbase.classification.InterfaceAudience;
 import org.apache.hadoop.hbase.classification.InterfaceStability;
@@ -71,6 +70,7 @@ import org.apache.hadoop.hbase.client.Result;
 import org.apache.hadoop.hbase.client.ResultScanner;
 import org.apache.hadoop.hbase.client.Scan;
 import org.apache.hadoop.hbase.client.Table;
+import org.apache.hadoop.hbase.client.TableState;
 import org.apache.hadoop.hbase.fs.HFileSystem;
 import org.apache.hadoop.hbase.io.compress.Compression;
 import org.apache.hadoop.hbase.io.compress.Compression.Algorithm;
@@ -122,6 +122,10 @@ import org.apache.zookeeper.WatchedEvent;
 import org.apache.zookeeper.ZooKeeper;
 import org.apache.zookeeper.ZooKeeper.States;
 
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertTrue;
+import static org.junit.Assert.fail;
+
 /**
  * Facility for testing HBase. Replacement for
  * old HBaseTestCase and HBaseClusterTestCase functionality.
@@ -2899,12 +2903,12 @@ public class HBaseTestingUtility extends HBaseCommonTestingUtility {
    */
   public void waitTableAvailable(TableName table)
       throws InterruptedException, IOException {
-    waitTableAvailable(getHBaseAdmin(), table.getName(), 30000);
+    waitTableAvailable(table.getName(), 30000);
   }
 
-  public void waitTableAvailable(Admin admin, byte[] table)
+  public void waitTableAvailable(TableName table, long timeoutMillis)
       throws InterruptedException, IOException {
-    waitTableAvailable(admin, table, 30000);
+    waitFor(timeoutMillis, predicateTableAvailable(table));
   }
 
   /**
@@ -2916,23 +2920,54 @@ public class HBaseTestingUtility extends HBaseCommonTestingUtility {
    */
   public void waitTableAvailable(byte[] table, long timeoutMillis)
   throws InterruptedException, IOException {
-    waitTableAvailable(getHBaseAdmin(), table, timeoutMillis);
+    waitFor(timeoutMillis, predicateTableAvailable(TableName.valueOf(table)));
+  }
+
+  public String explainTableAvailability(TableName tableName) throws IOException {
+    String msg = explainTableState(tableName, TableState.State.ENABLED) + ", ";
+    if (getHBaseCluster().getMaster().isAlive()) {
+      Map<HRegionInfo, ServerName> assignments =
+          getHBaseCluster().getMaster().getAssignmentManager().getRegionStates()
+              .getRegionAssignments();
+      final List<Pair<HRegionInfo, ServerName>> metaLocations =
+          MetaTableAccessor.getTableRegionsAndLocations(connection, tableName);
+      for (Pair<HRegionInfo, ServerName> metaLocation : metaLocations) {
+        HRegionInfo hri = metaLocation.getFirst();
+        ServerName sn = metaLocation.getSecond();
+        if (!assignments.containsKey(hri)) {
+          msg += ", region " + hri
+              + " not assigned, but found in meta, it expected to be on " + sn;
+
+        } else if (sn == null) {
+          msg += ",  region " + hri
+              + " assigned,  but has no server in meta";
+        } else if (!sn.equals(assignments.get(hri))) {
+          msg += ",  region " + hri
+              + " assigned,  but has different servers in meta and AM ( " +
+              sn + " <> " + assignments.get(hri);
+        }
+      }
+    }
+    return msg;
   }
 
-  public void waitTableAvailable(Admin admin, byte[] table, long timeoutMillis)
-  throws InterruptedException, IOException {
-    long startWait = System.currentTimeMillis();
-    while (!admin.isTableAvailable(TableName.valueOf(table))) {
-      assertTrue("Timed out waiting for table to become available " +
-        Bytes.toStringBinary(table),
-        System.currentTimeMillis() - startWait < timeoutMillis);
-      Thread.sleep(200);
+  public String explainTableState(final TableName table, TableState.State state)
+      throws IOException {
+    TableState.State tableState =
+        getHBaseCluster().getMaster().getTableStateManager().getTableState(table);
+    if (tableState == null) {
+      return "TableState: No state for table " + table;
+    } else if (!tableState.equals(state)) {
+      return "TableState: Not " + state + " state, but " + tableState;
+    } else {
+      return "TableState: OK";
     }
   }
 
   /**
    * Waits for a table to be 'enabled'.  Enabled means that table is set as 'enabled' and the
    * regions have been all assigned.  Will timeout after default period (30 seconds)
+   * Tolerates nonexistent table.
    * @param table Table to wait on.
    * @param table
    * @throws InterruptedException
@@ -2940,12 +2975,7 @@ public class HBaseTestingUtility extends HBaseCommonTestingUtility {
    */
   public void waitTableEnabled(TableName table)
       throws InterruptedException, IOException {
-    waitTableEnabled(getHBaseAdmin(), table.getName(), 30000);
-  }
-
-  public void waitTableEnabled(Admin admin, byte[] table)
-      throws InterruptedException, IOException {
-    waitTableEnabled(admin, table, 30000);
+    waitTableEnabled(table, 30000);
   }
 
   /**
@@ -2959,30 +2989,12 @@ public class HBaseTestingUtility extends HBaseCommonTestingUtility {
    */
   public void waitTableEnabled(byte[] table, long timeoutMillis)
   throws InterruptedException, IOException {
-    waitTableEnabled(getHBaseAdmin(), table, timeoutMillis);
+    waitTableEnabled(TableName.valueOf(table), timeoutMillis);
   }
 
-  public void waitTableEnabled(Admin admin, byte[] table, long timeoutMillis)
-  throws InterruptedException, IOException {
-    TableName tableName = TableName.valueOf(table);
-    long startWait = System.currentTimeMillis();
-    waitTableAvailable(admin, table, timeoutMillis);
-    while (!admin.isTableEnabled(tableName)) {
-      assertTrue("Timed out waiting for table to become available and enabled " +
-         Bytes.toStringBinary(table),
-         System.currentTimeMillis() - startWait < timeoutMillis);
-      Thread.sleep(200);
-    }
-    // Finally make sure all regions are fully open and online out on the cluster. Regions may be
-    // in the hbase:meta table and almost open on all regionservers but there setting the region
-    // online in the regionserver is the very last thing done and can take a little while to happen.
-    // Below we do a get.  The get will retry if a NotServeringRegionException or a
-    // RegionOpeningException.  It is crass but when done all will be online.
-    try {
-      Canary.sniff(admin, tableName);
-    } catch (Exception e) {
-      throw new IOException(e);
-    }
+  public void waitTableEnabled(TableName table, long timeoutMillis)
+  throws IOException {
+    waitFor(timeoutMillis, predicateTableEnabled(table));
   }
 
   /**
@@ -2994,12 +3006,12 @@ public class HBaseTestingUtility extends HBaseCommonTestingUtility {
    */
   public void waitTableDisabled(byte[] table)
           throws InterruptedException, IOException {
-    waitTableDisabled(getHBaseAdmin(), table, 30000);
+    waitTableDisabled(table, 30000);
   }
 
-  public void waitTableDisabled(Admin admin, byte[] table)
+  public void waitTableDisabled(TableName table, long millisTimeout)
           throws InterruptedException, IOException {
-    waitTableDisabled(admin, table, 30000);
+    waitFor(millisTimeout, predicateTableDisabled(table));
   }
 
   /**
@@ -3011,19 +3023,7 @@ public class HBaseTestingUtility extends HBaseCommonTestingUtility {
    */
   public void waitTableDisabled(byte[] table, long timeoutMillis)
           throws InterruptedException, IOException {
-    waitTableDisabled(getHBaseAdmin(), table, timeoutMillis);
-  }
-
-  public void waitTableDisabled(Admin admin, byte[] table, long timeoutMillis)
-          throws InterruptedException, IOException {
-    TableName tableName = TableName.valueOf(table);
-    long startWait = System.currentTimeMillis();
-    while (!admin.isTableDisabled(tableName)) {
-      assertTrue("Timed out waiting for table to become disabled " +
-                      Bytes.toStringBinary(table),
-              System.currentTimeMillis() - startWait < timeoutMillis);
-      Thread.sleep(200);
-    }
+    waitTableDisabled(TableName.valueOf(table), timeoutMillis);
   }
 
   /**
@@ -3189,7 +3189,12 @@ public class HBaseTestingUtility extends HBaseCommonTestingUtility {
       throws IOException {
     final Table meta = getConnection().getTable(TableName.META_TABLE_NAME);
     try {
-      waitFor(timeout, 200, true, new Predicate<IOException>() {
+      long l = waitFor(timeout, 200, true, new ExplainingPredicate<IOException>() {
+        @Override
+        public String explainFailure() throws IOException {
+          return explainTableAvailability(tableName);
+        }
+
         @Override
         public boolean evaluate() throws IOException {
           boolean allRegionsAssigned = true;
@@ -3199,7 +3204,7 @@ public class HBaseTestingUtility extends HBaseCommonTestingUtility {
           try {
             Result r;
             while ((r = s.next()) != null) {
-              byte [] b = r.getValue(HConstants.CATALOG_FAMILY, HConstants.REGIONINFO_QUALIFIER);
+              byte[] b = r.getValue(HConstants.CATALOG_FAMILY, HConstants.REGIONINFO_QUALIFIER);
               HRegionInfo info = HRegionInfo.parseFromOrNull(b);
               if (info != null && info.getTable().equals(tableName)) {
                 b = r.getValue(HConstants.CATALOG_FAMILY, HConstants.SERVER_QUALIFIER);
@@ -3222,7 +3227,12 @@ public class HBaseTestingUtility extends HBaseCommonTestingUtility {
       // returing -- sometimes this can lag.
       HMaster master = getHBaseCluster().getMaster();
       final RegionStates states = master.getAssignmentManager().getRegionStates();
-      waitFor(timeout, 200, new Predicate<IOException>() {
+      waitFor(timeout, 200, new ExplainingPredicate<IOException>() {
+        @Override
+        public String explainFailure() throws IOException {
+          return explainTableAvailability(tableName);
+        }
+
         @Override
         public boolean evaluate() throws IOException {
           List<HRegionInfo> hris = states.getRegionsOfTable(tableName);
@@ -3696,10 +3706,17 @@ public class HBaseTestingUtility extends HBaseCommonTestingUtility {
   /**
    * Returns a {@link Predicate} for checking that there are no regions in transition in master
    */
-  public Waiter.Predicate<Exception> predicateNoRegionsInTransition() {
-    return new Waiter.Predicate<Exception>() {
+  public ExplainingPredicate<IOException> predicateNoRegionsInTransition() {
+    return new ExplainingPredicate<IOException>() {
       @Override
-      public boolean evaluate() throws Exception {
+      public String explainFailure() throws IOException {
+        final RegionStates regionStates = getMiniHBaseCluster().getMaster()
+            .getAssignmentManager().getRegionStates();
+        return "found in transition: " + regionStates.getRegionsInTransition().toString();
+      }
+
+      @Override
+      public boolean evaluate() throws IOException {
         final RegionStates regionStates = getMiniHBaseCluster().getMaster()
             .getAssignmentManager().getRegionStates();
         return !regionStates.isRegionsInTransition();
@@ -3710,11 +3727,58 @@ public class HBaseTestingUtility extends HBaseCommonTestingUtility {
   /**
    * Returns a {@link Predicate} for checking that table is enabled
    */
-  public Waiter.Predicate<Exception> predicateTableEnabled(final TableName tableName) {
-    return new Waiter.Predicate<Exception>() {
-     @Override
-     public boolean evaluate() throws Exception {
-       return getHBaseAdmin().isTableEnabled(tableName);
+  public Waiter.Predicate<IOException> predicateTableEnabled(final TableName tableName) {
+    return new ExplainingPredicate<IOException>() {
+      @Override
+      public String explainFailure() throws IOException {
+        return explainTableState(tableName, TableState.State.ENABLED);
+      }
+
+      @Override
+      public boolean evaluate() throws IOException {
+        return getHBaseAdmin().tableExists(tableName) && getHBaseAdmin().isTableEnabled(tableName);
+      }
+    };
+  }
+
+  /**
+   * Returns a {@link Predicate} for checking that table is enabled
+   */
+  public Waiter.Predicate<IOException> predicateTableDisabled(final TableName tableName) {
+    return new ExplainingPredicate<IOException>() {
+      @Override
+      public String explainFailure() throws IOException {
+        return explainTableState(tableName, TableState.State.DISABLED);
+      }
+
+      @Override
+      public boolean evaluate() throws IOException {
+        return getHBaseAdmin().isTableDisabled(tableName);
+      }
+    };
+  }
+
+  /**
+   * Returns a {@link Predicate} for checking that table is enabled
+   */
+  public Waiter.Predicate<IOException> predicateTableAvailable(final TableName tableName) {
+    return new ExplainingPredicate<IOException>() {
+      @Override
+      public String explainFailure() throws IOException {
+        return explainTableAvailability(tableName);
+      }
+
+      @Override
+      public boolean evaluate() throws IOException {
+        boolean tableAvailable = getHBaseAdmin().isTableAvailable(tableName);
+        if (tableAvailable) {
+          try {
+            Canary.sniff(getHBaseAdmin(), tableName);
+          } catch (Exception e) {
+            throw new IOException("Canary sniff failed for table " + tableName, e);
+          }
+        }
+        return tableAvailable;
       }
     };
   }

http://git-wip-us.apache.org/repos/asf/hbase/blob/a3f495f2/hbase-server/src/test/java/org/apache/hadoop/hbase/security/access/TestAccessController.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/security/access/TestAccessController.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/security/access/TestAccessController.java
index 3c8fea5..8296e81 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/security/access/TestAccessController.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/security/access/TestAccessController.java
@@ -927,11 +927,9 @@ public class TestAccessController extends SecureTestUtil {
       setPermission(loadPath, FsPermission.valueOf("-rwxrwxrwx"));
 
       try (HTable table = (HTable)TEST_UTIL.getConnection().getTable(tableName)) {
-        try (Admin admin = TEST_UTIL.getHBaseAdmin()) {
-          TEST_UTIL.waitTableEnabled(admin, tableName.getName());
-          LoadIncrementalHFiles loader = new LoadIncrementalHFiles(conf);
-          loader.doBulkLoad(loadPath, table);
-        }
+        TEST_UTIL.waitTableEnabled(tableName);
+        LoadIncrementalHFiles loader = new LoadIncrementalHFiles(conf);
+        loader.doBulkLoad(loadPath, table);
       }
     }
 

http://git-wip-us.apache.org/repos/asf/hbase/blob/a3f495f2/hbase-server/src/test/java/org/apache/hadoop/hbase/snapshot/SnapshotTestingUtils.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/snapshot/SnapshotTestingUtils.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/snapshot/SnapshotTestingUtils.java
index a8588cc..b663a2a 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/snapshot/SnapshotTestingUtils.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/snapshot/SnapshotTestingUtils.java
@@ -635,13 +635,7 @@ public class SnapshotTestingUtils {
       region.waitForFlushesAndCompactions();
     }
     // Wait up to 60 seconds for a table to be available.
-    final HBaseAdmin hBaseAdmin = util.getHBaseAdmin();
-    util.waitFor(60000, new Waiter.Predicate<IOException>() {
-      @Override
-      public boolean evaluate() throws IOException {
-        return hBaseAdmin.isTableAvailable(tableName);
-      }
-    });
+    util.waitFor(60000, util.predicateTableAvailable(tableName));
   }
 
   public static void createTable(final HBaseTestingUtility util, final TableName tableName,