You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by st...@apache.org on 2015/02/18 18:58:50 UTC

[5/5] hbase git commit: HBASE-13017 Backport HBASE-12035 Keep table state in Meta to 1.0 branch (Andrey Stepachev)

HBASE-13017 Backport HBASE-12035 Keep table state in Meta to 1.0 branch (Andrey Stepachev)


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

Branch: refs/heads/branch-1
Commit: f5e2a823284ebe610f8754f36135dd5290f93b06
Parents: 8afb80b
Author: stack <st...@apache.org>
Authored: Wed Feb 18 09:58:32 2015 -0800
Committer: stack <st...@apache.org>
Committed: Wed Feb 18 09:58:32 2015 -0800

----------------------------------------------------------------------
 .../apache/hadoop/hbase/HTableDescriptor.java   |  59 +-
 .../apache/hadoop/hbase/MetaTableAccessor.java  | 383 +++++++---
 .../hadoop/hbase/client/ConnectionCallable.java |  56 ++
 .../hadoop/hbase/client/ConnectionManager.java  | 137 ++--
 .../apache/hadoop/hbase/client/HBaseAdmin.java  |  95 ++-
 .../org/apache/hadoop/hbase/client/HTable.java  |   9 +-
 .../apache/hadoop/hbase/client/Registry.java    |   7 -
 .../apache/hadoop/hbase/client/TableState.java  | 219 ++++++
 .../hadoop/hbase/client/ZooKeeperRegistry.java  |  21 -
 .../zookeeper/ZKTableStateClientSideReader.java | 202 -----
 .../hbase/client/TestClientNoCluster.java       |   6 -
 .../org/apache/hadoop/hbase/HConstants.java     |  14 +-
 .../hbase/protobuf/generated/HBaseProtos.java   | 729 +++++++++++++++++--
 hbase-protocol/src/main/protobuf/HBase.proto    |  13 +
 .../hadoop/hbase/CoordinatedStateManager.java   |   8 -
 .../hbase/MetaMigrationConvertingToPB.java      |   4 +-
 .../apache/hadoop/hbase/TableStateManager.java  | 115 ---
 .../BaseCoordinatedStateManager.java            |   7 +-
 .../coordination/ZkCoordinatedStateManager.java |  16 +-
 .../coordination/ZkOpenRegionCoordination.java  |   3 +-
 .../hadoop/hbase/master/AssignmentManager.java  |  94 ++-
 .../org/apache/hadoop/hbase/master/HMaster.java |  55 +-
 .../hadoop/hbase/master/MasterServices.java     |   5 +
 .../hadoop/hbase/master/RegionStateStore.java   |   3 +-
 .../hadoop/hbase/master/RegionStates.java       |  11 +-
 .../SnapshotOfRegionAssignmentFromMeta.java     |   2 +-
 .../hbase/master/TableNamespaceManager.java     |   3 +-
 .../hadoop/hbase/master/TableStateManager.java  | 285 ++++++++
 .../master/handler/ClosedRegionHandler.java     |   4 +-
 .../master/handler/CreateTableHandler.java      |  98 +--
 .../master/handler/DeleteTableHandler.java      | 106 +--
 .../master/handler/DisableTableHandler.java     |  33 +-
 .../master/handler/EnableTableHandler.java      |  46 +-
 .../master/handler/ModifyTableHandler.java      |   4 +-
 .../master/handler/ServerShutdownHandler.java   |   8 +-
 .../hbase/master/handler/TableEventHandler.java |   4 +-
 .../master/handler/TruncateTableHandler.java    |  82 +--
 .../hbase/master/snapshot/SnapshotManager.java  |   8 +-
 .../hbase/regionserver/HRegionServer.java       |   5 +
 .../visibility/VisibilityController.java        |  25 +-
 .../hadoop/hbase/util/FSTableDescriptors.java   |   7 +-
 .../org/apache/hadoop/hbase/util/HBaseFsck.java | 207 +++---
 .../hadoop/hbase/util/ZKDataMigrator.java       |  79 ++
 .../apache/hadoop/hbase/wal/WALSplitter.java    |  58 +-
 .../hbase/zookeeper/ZKTableStateManager.java    | 346 ---------
 .../hadoop/hbase/HBaseTestingUtility.java       |  69 +-
 .../apache/hadoop/hbase/TestDrainingServer.java |  27 +-
 .../hbase/TestMetaMigrationConvertingToPB.java  |   2 +-
 .../hadoop/hbase/TestMetaTableAccessor.java     |   7 +-
 .../apache/hadoop/hbase/client/TestAdmin1.java  |  25 +-
 .../hbase/client/TestMetaWithReplicas.java      |  69 +-
 .../hbase/client/TestReplicaWithCluster.java    |   4 +-
 .../hbase/master/TestAssignmentManager.java     | 186 +++--
 .../master/TestAssignmentManagerOnCluster.java  |   7 +-
 .../hadoop/hbase/master/TestCatalogJanitor.java |   5 +
 .../master/TestDistributedLogSplitting.java     |   1 +
 .../apache/hadoop/hbase/master/TestMaster.java  |   4 +-
 .../hadoop/hbase/master/TestMasterFailover.java |  37 +-
 .../TestMasterOperationsForRegionReplicas.java  |   4 +-
 .../TestMasterRestartAfterDisablingTable.java   |   8 +-
 .../hbase/master/TestOpenedRegionHandler.java   |   5 +-
 .../hadoop/hbase/master/TestRegionStates.java   |   4 +-
 .../master/handler/TestEnableTableHandler.java  |  54 +-
 .../hadoop/hbase/migration/TestUpgradeTo96.java |   3 +-
 .../hbase/util/TestCoprocessorScanPolicy.java   |   1 +
 .../hbase/util/TestFSTableDescriptors.java      |   2 +-
 .../apache/hadoop/hbase/util/TestHBaseFsck.java |  85 +--
 .../util/hbck/OfflineMetaRebuildTestCore.java   |  17 +-
 .../util/hbck/TestOfflineMetaRebuildBase.java   |  15 +-
 .../util/hbck/TestOfflineMetaRebuildHole.java   |   2 -
 .../hbck/TestOfflineMetaRebuildOverlap.java     |   2 -
 .../zookeeper/TestZKTableStateManager.java      | 114 ---
 72 files changed, 2560 insertions(+), 1880 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/hbase/blob/f5e2a823/hbase-client/src/main/java/org/apache/hadoop/hbase/HTableDescriptor.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/HTableDescriptor.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/HTableDescriptor.java
index eb050fc..c37a63f 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/HTableDescriptor.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/HTableDescriptor.java
@@ -1414,27 +1414,11 @@ public class HTableDescriptor implements WritableComparable<HTableDescriptor> {
    * HBaseAdmin#getTableDescriptor(TableName.META_TABLE_NAME) instead.
    */
   @Deprecated
-  public static final HTableDescriptor META_TABLEDESC = new HTableDescriptor(
-      TableName.META_TABLE_NAME,
-      new HColumnDescriptor[] {
-          new HColumnDescriptor(HConstants.CATALOG_FAMILY)
-              // Ten is arbitrary number.  Keep versions to help debugging.
-              .setMaxVersions(HConstants.DEFAULT_HBASE_META_VERSIONS)
-              .setInMemory(true)
-              .setBlocksize(HConstants.DEFAULT_HBASE_META_BLOCK_SIZE)
-              .setScope(HConstants.REPLICATION_SCOPE_LOCAL)
-              // Disable blooms for meta.  Needs work.  Seems to mess w/ getClosestOrBefore.
-              .setBloomFilterType(BloomType.NONE)
-              // Enable cache of data blocks in L1 if more than one caching tier deployed:
-              // e.g. if using CombinedBlockCache (BucketCache).
-              .setCacheDataInL1(true)
-      });
+  public static final HTableDescriptor META_TABLEDESC;
 
   static {
     try {
-      META_TABLEDESC.addCoprocessor(
-          "org.apache.hadoop.hbase.coprocessor.MultiRowMutationEndpoint",
-          null, Coprocessor.PRIORITY_SYSTEM, null);
+      META_TABLEDESC = metaTableDescriptor(new Configuration());
     } catch (IOException ex) {
       //LOG.warn("exception in loading coprocessor for the hbase:meta table");
       throw new RuntimeException(ex);
@@ -1604,21 +1588,32 @@ public class HTableDescriptor implements WritableComparable<HTableDescriptor> {
   public static HTableDescriptor metaTableDescriptor(final Configuration conf)
       throws IOException {
     HTableDescriptor metaDescriptor = new HTableDescriptor(
-      TableName.META_TABLE_NAME,
-      new HColumnDescriptor[] {
-        new HColumnDescriptor(HConstants.CATALOG_FAMILY)
-          .setMaxVersions(conf.getInt(HConstants.HBASE_META_VERSIONS,
-            HConstants.DEFAULT_HBASE_META_VERSIONS))
-          .setInMemory(true)
-          .setBlocksize(conf.getInt(HConstants.HBASE_META_BLOCK_SIZE,
-            HConstants.DEFAULT_HBASE_META_BLOCK_SIZE))
-          .setScope(HConstants.REPLICATION_SCOPE_LOCAL)
-          // Disable blooms for meta.  Needs work.  Seems to mess w/ getClosestOrBefore.
-          .setBloomFilterType(BloomType.NONE)
-         });
+        TableName.META_TABLE_NAME,
+        new HColumnDescriptor[] {
+            new HColumnDescriptor(HConstants.CATALOG_FAMILY)
+                .setMaxVersions(conf.getInt(HConstants.HBASE_META_VERSIONS,
+                    HConstants.DEFAULT_HBASE_META_VERSIONS))
+                .setInMemory(true)
+                .setBlocksize(conf.getInt(HConstants.HBASE_META_BLOCK_SIZE,
+                    HConstants.DEFAULT_HBASE_META_BLOCK_SIZE))
+                .setScope(HConstants.REPLICATION_SCOPE_LOCAL)
+                    // Disable blooms for meta.  Needs work.  Seems to mess w/ getClosestOrBefore.
+                .setBloomFilterType(BloomType.NONE),
+            new HColumnDescriptor(HConstants.TABLE_FAMILY)
+                // Ten is arbitrary number.  Keep versions to help debugging.
+                .setMaxVersions(10)
+                .setInMemory(true)
+                .setBlocksize(8 * 1024)
+                .setScope(HConstants.REPLICATION_SCOPE_LOCAL)
+                    // Disable blooms for meta.  Needs work.  Seems to mess w/ getClosestOrBefore.
+                .setBloomFilterType(BloomType.NONE)
+                    // Enable cache of data blocks in L1 if more than one caching tier deployed:
+                    // e.g. if using CombinedBlockCache (BucketCache).
+                .setCacheDataInL1(true)
+        });
     metaDescriptor.addCoprocessor(
-      "org.apache.hadoop.hbase.coprocessor.MultiRowMutationEndpoint",
-      null, Coprocessor.PRIORITY_SYSTEM, null);
+        "org.apache.hadoop.hbase.coprocessor.MultiRowMutationEndpoint",
+        null, Coprocessor.PRIORITY_SYSTEM, null);
     return metaDescriptor;
   }
 

http://git-wip-us.apache.org/repos/asf/hbase/blob/f5e2a823/hbase-client/src/main/java/org/apache/hadoop/hbase/MetaTableAccessor.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/MetaTableAccessor.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/MetaTableAccessor.java
index 37ef788..8d486bf 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/MetaTableAccessor.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/MetaTableAccessor.java
@@ -17,9 +17,23 @@
  */
 package org.apache.hadoop.hbase;
 
+import javax.annotation.Nullable;
+import java.io.IOException;
+import java.io.InterruptedIOException;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.LinkedHashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.NavigableMap;
+import java.util.Set;
+import java.util.SortedMap;
+import java.util.TreeMap;
+import java.util.regex.Matcher;
+import java.util.regex.Pattern;
+
 import com.google.common.annotations.VisibleForTesting;
 import com.google.protobuf.ServiceException;
-
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.conf.Configuration;
@@ -38,6 +52,8 @@ 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.exceptions.DeserializationException;
 import org.apache.hadoop.hbase.ipc.CoprocessorRpcChannel;
 import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
 import org.apache.hadoop.hbase.protobuf.generated.ClientProtos;
@@ -50,17 +66,6 @@ import org.apache.hadoop.hbase.util.Threads;
 import org.apache.hadoop.hbase.zookeeper.MetaTableLocator;
 import org.apache.hadoop.hbase.zookeeper.ZooKeeperWatcher;
 
-import java.io.IOException;
-import java.io.InterruptedIOException;
-import java.util.ArrayList;
-import java.util.List;
-import java.util.Map;
-import java.util.NavigableMap;
-import java.util.Set;
-import java.util.SortedMap;
-import java.util.TreeMap;
-import java.util.regex.Matcher;
-import java.util.regex.Pattern;
 
 /**
  * Read/write operations on region and assignment information store in
@@ -80,6 +85,11 @@ public class MetaTableAccessor {
    * HRI defined which is called default replica.
    *
    * Meta layout (as of 0.98 + HBASE-10070) is like:
+   *
+   * For each table there is single row in column family 'table' formatted:
+   * <tableName> including namespace and columns are:
+   * table: state             => contains table state
+   *
    * For each table range, there is a single row, formatted like:
    * <tableName>,<startKey>,<regionId>,<encodedRegionName>. This row corresponds to the regionName
    * of the default region replica.
@@ -122,6 +132,24 @@ public class MetaTableAccessor {
       META_REGION_PREFIX, 0, len);
   }
 
+
+  @InterfaceAudience.Private
+  public enum QueryType {
+    ALL(HConstants.TABLE_FAMILY, HConstants.CATALOG_FAMILY),
+    REGION(HConstants.CATALOG_FAMILY),
+    TABLE(HConstants.TABLE_FAMILY);
+
+    private final byte[][] families;
+
+    QueryType(byte[]... families) {
+      this.families = families;
+    }
+
+    byte[][] getFamilies() {
+      return this.families;
+    }
+  }
+
   /** The delimiter for meta columns for replicaIds > 0 */
   protected static final char META_REPLICA_ID_DELIMITER = '_';
 
@@ -133,40 +161,64 @@ public class MetaTableAccessor {
   // Reading operations //
   ////////////////////////
 
- /**
-   * Performs a full scan of a <code>hbase:meta</code> table.
-   * @return List of {@link org.apache.hadoop.hbase.client.Result}
+  /**
+   * Performs a full scan of <code>hbase:meta</code> for regions.
+   * @param connection connection we're using
+   * @param visitor Visitor invoked against each row in regions family.
    * @throws IOException
    */
-  public static List<Result> fullScanOfMeta(Connection connection)
-  throws IOException {
-    CollectAllVisitor v = new CollectAllVisitor();
-    fullScan(connection, v, null);
-    return v.getResults();
+  public static void fullScanRegions(Connection connection,
+      final Visitor visitor)
+      throws IOException {
+    fullScan(connection, visitor, null, QueryType.REGION);
+  }
+
+  /**
+   * Performs a full scan of <code>hbase:meta</code> for regions.
+   * @param connection connection we're using
+   * @throws IOException
+   */
+  public static List<Result> fullScanRegions(Connection connection)
+      throws IOException {
+    return fullScan(connection, QueryType.REGION);
+  }
+
+  /**
+   * Performs a full scan of <code>hbase:meta</code> for tables.
+   * @param connection connection we're using
+   * @param visitor Visitor invoked against each row in tables family.
+   * @throws IOException
+   */
+  public static void fullScanTables(Connection connection,
+      final Visitor visitor)
+      throws IOException {
+    fullScan(connection, visitor, null, QueryType.TABLE);
   }
 
   /**
    * Performs a full scan of <code>hbase:meta</code>.
    * @param connection connection we're using
    * @param visitor Visitor invoked against each row.
+   * @param type scanned part of meta
    * @throws IOException
    */
   public static void fullScan(Connection connection,
-      final Visitor visitor)
+      final Visitor visitor, QueryType type)
   throws IOException {
-    fullScan(connection, visitor, null);
+    fullScan(connection, visitor, null, type);
   }
 
   /**
    * Performs a full scan of <code>hbase:meta</code>.
    * @param connection connection we're using
+   * @param type scanned part of meta
    * @return List of {@link Result}
    * @throws IOException
    */
-  public static List<Result> fullScan(Connection connection)
+  public static List<Result> fullScan(Connection connection, QueryType type)
     throws IOException {
     CollectAllVisitor v = new CollectAllVisitor();
-    fullScan(connection, v, null);
+    fullScan(connection, v, null, type);
     return v.getResults();
   }
 
@@ -307,6 +359,7 @@ public class MetaTableAccessor {
    * @return null if it doesn't contain merge qualifier, else two merge regions
    * @throws IOException
    */
+  @Nullable
   public static Pair<HRegionInfo, HRegionInfo> getRegionsFromMergeQualifier(
       Connection connection, byte[] regionName) throws IOException {
     Result result = getRegionResult(connection, regionName);
@@ -329,42 +382,9 @@ public class MetaTableAccessor {
   public static boolean tableExists(Connection connection,
       final TableName tableName)
   throws IOException {
-    if (tableName.equals(TableName.META_TABLE_NAME)) {
-      // Catalog tables always exist.
-      return true;
-    }
-    // Make a version of ResultCollectingVisitor that only collects the first
-    CollectingVisitor<HRegionInfo> visitor = new CollectingVisitor<HRegionInfo>() {
-      private HRegionInfo current = null;
-
-      @Override
-      public boolean visit(Result r) throws IOException {
-        RegionLocations locations = getRegionLocations(r);
-        if (locations == null || locations.getRegionLocation().getRegionInfo() == null) {
-          LOG.warn("No serialized HRegionInfo in " + r);
-          return true;
-        }
-        this.current = locations.getRegionLocation().getRegionInfo();
-        if (this.current == null) {
-          LOG.warn("No serialized HRegionInfo in " + r);
-          return true;
-        }
-        if (!isInsideTable(this.current, tableName)) return false;
-        // Else call super and add this Result to the collection.
-        super.visit(r);
-        // Stop collecting regions from table after we get one.
-        return false;
-      }
-
-      @Override
-      void add(Result r) {
-        // Add the current HRI.
-        this.results.add(this.current);
-      }
-    };
-    fullScan(connection, visitor, getTableStartRowForMeta(tableName));
-    // If visitor has results >= 1 then table exists.
-    return visitor.getResults().size() >= 1;
+    // Catalog tables always exist.
+    return tableName.equals(TableName.META_TABLE_NAME)
+        || getTableState(connection, tableName) != null;
   }
 
   /**
@@ -400,6 +420,7 @@ public class MetaTableAccessor {
     return getListOfHRegionInfos(result);
   }
 
+  @Nullable
   static List<HRegionInfo> getListOfHRegionInfos(final List<Pair<HRegionInfo, ServerName>> pairs) {
     if (pairs == null || pairs.isEmpty()) return null;
     List<HRegionInfo> result = new ArrayList<HRegionInfo>(pairs.size());
@@ -519,7 +540,7 @@ public class MetaTableAccessor {
           }
         }
       };
-    fullScan(connection, visitor, getTableStartRowForMeta(tableName));
+    fullScan(connection, visitor, getTableStartRowForMeta(tableName), QueryType.REGION);
     return visitor.getResults();
   }
 
@@ -551,7 +572,7 @@ public class MetaTableAccessor {
         }
       }
     };
-    fullScan(connection, v);
+    fullScan(connection, v, QueryType.REGION);
     return hris;
   }
 
@@ -562,17 +583,22 @@ public class MetaTableAccessor {
       public boolean visit(Result r) throws IOException {
         if (r ==  null || r.isEmpty()) return true;
         LOG.info("fullScanMetaAndPrint.Current Meta Row: " + r);
-        RegionLocations locations = getRegionLocations(r);
-        if (locations == null) return true;
-        for (HRegionLocation loc : locations.getRegionLocations()) {
-          if (loc != null) {
-            LOG.info("fullScanMetaAndPrint.HRI Print= " + loc.getRegionInfo());
+        TableState state = getTableState(r);
+        if (state != null) {
+          LOG.info("Table State: " + state);
+        } else {
+          RegionLocations locations = getRegionLocations(r);
+          if (locations == null) return true;
+          for (HRegionLocation loc : locations.getRegionLocations()) {
+            if (loc != null) {
+              LOG.info("fullScanMetaAndPrint.HRI Print= " + loc.getRegionInfo());
+            }
           }
         }
         return true;
       }
     };
-    fullScan(connection, v);
+    fullScan(connection, v, QueryType.ALL);
   }
 
   /**
@@ -581,20 +607,40 @@ public class MetaTableAccessor {
    * @param visitor Visitor invoked against each row.
    * @param startrow Where to start the scan. Pass null if want to begin scan
    * at first row.
+   * @param type scanned part of meta
    * <code>hbase:meta</code>, the default (pass false to scan hbase:meta)
    * @throws IOException
    */
   public static void fullScan(Connection connection,
-    final Visitor visitor, final byte [] startrow)
+      final Visitor visitor, @Nullable final byte[] startrow, QueryType type) throws IOException {
+    fullScan(connection, visitor, startrow, type, false);
+  }
+
+  /**
+   * Performs a full scan of a catalog table.
+   * @param connection connection we're using
+   * @param visitor Visitor invoked against each row.
+   * @param startrow Where to start the scan. Pass null if want to begin scan
+   * at first row.
+   * @param type scanned part of meta
+   * @param raw read raw data including Delete tumbstones
+   * <code>hbase:meta</code>, the default (pass false to scan hbase:meta)
+   * @throws IOException
+   */
+  public static void fullScan(Connection connection,
+      final Visitor visitor, @Nullable final byte[] startrow, QueryType type, boolean raw)
   throws IOException {
     Scan scan = new Scan();
+    scan.setRaw(raw);
     if (startrow != null) scan.setStartRow(startrow);
     if (startrow == null) {
       int caching = connection.getConfiguration()
           .getInt(HConstants.HBASE_META_SCANNER_CACHING, 100);
       scan.setCaching(caching);
     }
-    scan.addFamily(HConstants.CATALOG_FAMILY);
+    for (byte[] family : type.getFamilies()) {
+      scan.addFamily(family);
+    }
     Table metaTable = getMetaHTable(connection);
     ResultScanner scanner = null;
     try {
@@ -615,11 +661,19 @@ public class MetaTableAccessor {
    * Returns the column family used for meta columns.
    * @return HConstants.CATALOG_FAMILY.
    */
-  protected static byte[] getFamily() {
+  protected static byte[] getCatalogFamily() {
     return HConstants.CATALOG_FAMILY;
   }
 
   /**
+   * Returns the column family used for table columns.
+   * @return HConstants.TABLE_FAMILY.
+   */
+  protected static byte[] getTableFamily() {
+    return HConstants.TABLE_FAMILY;
+  }
+
+  /**
    * Returns the column qualifier for serialized region info
    * @return HConstants.REGIONINFO_QUALIFIER
    */
@@ -628,6 +682,15 @@ public class MetaTableAccessor {
   }
 
   /**
+   * Returns the column qualifier for serialized table state
+   *
+   * @return HConstants.TABLE_STATE_QUALIFIER
+   */
+  protected static byte[] getStateColumn() {
+    return HConstants.TABLE_STATE_QUALIFIER;
+  }
+
+  /**
    * Returns the column qualifier for server column for replicaId
    * @param replicaId the replicaId of the region
    * @return a byte[] for server column qualifier
@@ -693,14 +756,15 @@ public class MetaTableAccessor {
    * @param r Result to pull from
    * @return A ServerName instance or null if necessary fields not found or empty.
    */
+  @Nullable
   private static ServerName getServerName(final Result r, final int replicaId) {
     byte[] serverColumn = getServerColumn(replicaId);
-    Cell cell = r.getColumnLatestCell(getFamily(), serverColumn);
+    Cell cell = r.getColumnLatestCell(getCatalogFamily(), serverColumn);
     if (cell == null || cell.getValueLength() == 0) return null;
     String hostAndPort = Bytes.toString(
       cell.getValueArray(), cell.getValueOffset(), cell.getValueLength());
     byte[] startcodeColumn = getStartCodeColumn(replicaId);
-    cell = r.getColumnLatestCell(getFamily(), startcodeColumn);
+    cell = r.getColumnLatestCell(getCatalogFamily(), startcodeColumn);
     if (cell == null || cell.getValueLength() == 0) return null;
     return ServerName.valueOf(hostAndPort,
       Bytes.toLong(cell.getValueArray(), cell.getValueOffset(), cell.getValueLength()));
@@ -713,7 +777,7 @@ public class MetaTableAccessor {
    * @return SeqNum, or HConstants.NO_SEQNUM if there's no value written.
    */
   private static long getSeqNumDuringOpen(final Result r, final int replicaId) {
-    Cell cell = r.getColumnLatestCell(getFamily(), getSeqNumColumn(replicaId));
+    Cell cell = r.getColumnLatestCell(getCatalogFamily(), getSeqNumColumn(replicaId));
     if (cell == null || cell.getValueLength() == 0) return HConstants.NO_SEQNUM;
     return Bytes.toLong(cell.getValueArray(), cell.getValueOffset(), cell.getValueLength());
   }
@@ -723,6 +787,7 @@ public class MetaTableAccessor {
    * @return an HRegionLocationList containing all locations for the region range or null if
    *  we can't deserialize the result.
    */
+  @Nullable
   public static RegionLocations getRegionLocations(final Result r) {
     if (r == null) return null;
     HRegionInfo regionInfo = getHRegionInfo(r, getRegionInfoColumn());
@@ -733,7 +798,7 @@ public class MetaTableAccessor {
 
     locations.add(getRegionLocation(r, regionInfo, 0));
 
-    NavigableMap<byte[], byte[]> infoMap = familyMap.get(getFamily());
+    NavigableMap<byte[], byte[]> infoMap = familyMap.get(getCatalogFamily());
     if (infoMap == null) return new RegionLocations(locations);
 
     // iterate until all serverName columns are seen
@@ -795,8 +860,9 @@ public class MetaTableAccessor {
    * @param qualifier Column family qualifier
    * @return An HRegionInfo instance or null.
    */
+  @Nullable
   private static HRegionInfo getHRegionInfo(final Result r, byte [] qualifier) {
-    Cell cell = r.getColumnLatestCell(getFamily(), qualifier);
+    Cell cell = r.getColumnLatestCell(getCatalogFamily(), qualifier);
     if (cell == null) return null;
     return HRegionInfo.parseFromOrNull(cell.getValueArray(),
       cell.getValueOffset(), cell.getValueLength());
@@ -831,6 +897,80 @@ public class MetaTableAccessor {
   }
 
   /**
+   * Fetch table state for given table from META table
+   * @param conn connection to use
+   * @param tableName table to fetch state for
+   * @return state
+   * @throws IOException
+   */
+  @Nullable
+  public static TableState getTableState(Connection conn, TableName tableName)
+      throws IOException {
+    Table metaHTable = getMetaHTable(conn);
+    Get get = new Get(tableName.getName()).addColumn(getTableFamily(), getStateColumn());
+    long time = EnvironmentEdgeManager.currentTime();
+    get.setTimeRange(0, time);
+    Result result =
+        metaHTable.get(get);
+    return getTableState(result);
+  }
+
+  /**
+   * Fetch table states from META table
+   * @param conn connection to use
+   * @return map {tableName -> state}
+   * @throws IOException
+   */
+  public static Map<TableName, TableState> getTableStates(Connection conn)
+      throws IOException {
+    final Map<TableName, TableState> states = new LinkedHashMap<>();
+    Visitor collector = new Visitor() {
+      @Override
+      public boolean visit(Result r) throws IOException {
+        TableState state = getTableState(r);
+        if (state != null)
+          states.put(state.getTableName(), state);
+        return true;
+      }
+    };
+    fullScanTables(conn, collector);
+    return states;
+  }
+
+  /**
+   * Updates state in META
+   * @param conn connection to use
+   * @param tableName table to look for
+   * @throws IOException
+   */
+  public static void updateTableState(Connection conn, TableName tableName,
+      TableState.State actual) throws IOException {
+    updateTableState(conn, new TableState(tableName, actual));
+  }
+
+  /**
+   * Decode table state from META Result.
+   * Should contain cell from HConstants.TABLE_FAMILY
+   * @param r result
+   * @return null if not found
+   * @throws IOException
+   */
+  @Nullable
+  public static TableState getTableState(Result r)
+      throws IOException {
+    Cell cell = r.getColumnLatestCell(getTableFamily(), getStateColumn());
+    if (cell == null) return null;
+    try {
+      return TableState.parseFrom(TableName.valueOf(r.getRow()),
+          Arrays.copyOfRange(cell.getValueArray(),
+          cell.getValueOffset(), cell.getValueOffset() + cell.getValueLength()));
+    } catch (DeserializationException e) {
+      throw new IOException(e);
+    }
+
+  }
+
+  /**
    * Implementations 'visit' a catalog table row.
    */
   public interface Visitor {
@@ -927,7 +1067,8 @@ public class MetaTableAccessor {
    */
   public static Put makePutFromRegionInfo(HRegionInfo regionInfo)
     throws IOException {
-    Put put = new Put(regionInfo.getRegionName());
+    long now = EnvironmentEdgeManager.currentTime();
+    Put put = new Put(regionInfo.getRegionName(), now);
     addRegionInfo(put, regionInfo);
     return put;
   }
@@ -940,7 +1081,9 @@ public class MetaTableAccessor {
     if (regionInfo == null) {
       throw new IllegalArgumentException("Can't make a delete for null region");
     }
+    long now = EnvironmentEdgeManager.currentTime();
     Delete delete = new Delete(regionInfo.getRegionName());
+    delete.addFamily(getCatalogFamily(), now);
     return delete;
   }
 
@@ -1041,14 +1184,15 @@ public class MetaTableAccessor {
       throws IOException {
     int absoluteIndex = replicaIndexToDeleteFrom + numReplicasToRemove;
     for (byte[] row : metaRows) {
+      long now = EnvironmentEdgeManager.currentTime();
       Delete deleteReplicaLocations = new Delete(row);
       for (int i = replicaIndexToDeleteFrom; i < absoluteIndex; i++) {
-        deleteReplicaLocations.deleteColumns(HConstants.CATALOG_FAMILY,
-          getServerColumn(i));
-        deleteReplicaLocations.deleteColumns(HConstants.CATALOG_FAMILY,
-          getSeqNumColumn(i));
-        deleteReplicaLocations.deleteColumns(HConstants.CATALOG_FAMILY,
-          getStartCodeColumn(i));
+        deleteReplicaLocations.addColumns(getCatalogFamily(),
+          getServerColumn(i), now);
+        deleteReplicaLocations.addColumns(getCatalogFamily(),
+          getSeqNumColumn(i), now);
+        deleteReplicaLocations.addColumns(getCatalogFamily(),
+          getStartCodeColumn(i), now);
       }
       deleteFromMetaTable(connection, deleteReplicaLocations);
     }
@@ -1178,7 +1322,8 @@ public class MetaTableAccessor {
   public static void addDaughter(final Connection connection,
       final HRegionInfo regionInfo, final ServerName sn, final long openSeqNum)
       throws NotAllMetaRegionsOnlineException, IOException {
-    Put put = new Put(regionInfo.getRegionName());
+    long now = EnvironmentEdgeManager.currentTime();
+    Put put = new Put(regionInfo.getRegionName(), now);
     addRegionInfo(put, regionInfo);
     if (sn != null) {
       addLocation(put, sn, openSeqNum, regionInfo.getReplicaId());
@@ -1280,6 +1425,45 @@ public class MetaTableAccessor {
   }
 
   /**
+   * Update state of the table in meta.
+   * @param connection what we use for update
+   * @param state new state
+   * @throws IOException
+   */
+  public static void updateTableState(Connection connection, TableState state)
+      throws IOException {
+    Put put = makePutFromTableState(state);
+    putToMetaTable(connection, put);
+    LOG.info(
+        "Updated table " + state.getTableName() + " state to " + state.getState() + " in META");
+  }
+
+  /**
+   * Construct PUT for given state
+   * @param state new state
+   */
+  public static Put makePutFromTableState(TableState state) {
+    long time = EnvironmentEdgeManager.currentTime();
+    Put put = new Put(state.getTableName().getName(), time);
+    put.add(getTableFamily(), getStateColumn(), state.convert().toByteArray());
+    return put;
+  }
+
+  /**
+   * Remove state for table from meta
+   * @param connection to use for deletion
+   * @param table to delete state for
+   */
+  public static void deleteTableState(Connection connection, TableName table)
+      throws IOException {
+    long time = EnvironmentEdgeManager.currentTime();
+    Delete delete = new Delete(table.getName());
+    delete.addColumns(getTableFamily(), getStateColumn(), time);
+    deleteFromMetaTable(connection, delete);
+    LOG.info("Deleted table " + table + " state from META");
+  }
+
+  /**
    * Performs an atomic multi-Mutate operation against the given table.
    */
   private static void multiMutate(Table table, byte[] row, Mutation... mutations)
@@ -1344,7 +1528,8 @@ public class MetaTableAccessor {
                                      HRegionInfo regionInfo, ServerName sn, long openSeqNum)
     throws IOException {
     // region replicas are kept in the primary region's row
-    Put put = new Put(getMetaKeyForRegion(regionInfo));
+    long time = EnvironmentEdgeManager.currentTime();
+    Put put = new Put(getMetaKeyForRegion(regionInfo), time);
     addLocation(put, sn, openSeqNum, regionInfo.getReplicaId());
     putToMetaTable(connection, put);
     LOG.info("Updated row " + regionInfo.getRegionNameAsString() +
@@ -1360,7 +1545,9 @@ public class MetaTableAccessor {
   public static void deleteRegion(Connection connection,
                                   HRegionInfo regionInfo)
     throws IOException {
+    long time = EnvironmentEdgeManager.currentTime();
     Delete delete = new Delete(regionInfo.getRegionName());
+    delete.addFamily(getCatalogFamily(), time);
     deleteFromMetaTable(connection, delete);
     LOG.info("Deleted " + regionInfo.getRegionNameAsString());
   }
@@ -1374,8 +1561,11 @@ public class MetaTableAccessor {
   public static void deleteRegions(Connection connection,
                                    List<HRegionInfo> regionsInfo) throws IOException {
     List<Delete> deletes = new ArrayList<Delete>(regionsInfo.size());
+    long time = EnvironmentEdgeManager.currentTime();
     for (HRegionInfo hri: regionsInfo) {
-      deletes.add(new Delete(hri.getRegionName()));
+      Delete e = new Delete(hri.getRegionName());
+      e.addFamily(getCatalogFamily(), time);
+      deletes.add(e);
     }
     deleteFromMetaTable(connection, deletes);
     LOG.info("Deleted " + regionsInfo);
@@ -1395,7 +1585,7 @@ public class MetaTableAccessor {
     List<Mutation> mutation = new ArrayList<Mutation>();
     if (regionsToRemove != null) {
       for (HRegionInfo hri: regionsToRemove) {
-        mutation.add(new Delete(hri.getRegionName()));
+        mutation.add(makeDeleteFromRegionInfo(hri));
       }
     }
     if (regionsToAdd != null) {
@@ -1438,9 +1628,10 @@ public class MetaTableAccessor {
    */
   public static void deleteMergeQualifiers(Connection connection,
                                            final HRegionInfo mergedRegion) throws IOException {
+    long time = EnvironmentEdgeManager.currentTime();
     Delete delete = new Delete(mergedRegion.getRegionName());
-    delete.deleteColumns(HConstants.CATALOG_FAMILY, HConstants.MERGEA_QUALIFIER);
-    delete.deleteColumns(HConstants.CATALOG_FAMILY, HConstants.MERGEB_QUALIFIER);
+    delete.addColumns(getCatalogFamily(), HConstants.MERGEA_QUALIFIER, time);
+    delete.addColumns(getCatalogFamily(), HConstants.MERGEB_QUALIFIER, time);
     deleteFromMetaTable(connection, delete);
     LOG.info("Deleted references in merged region "
       + mergedRegion.getRegionNameAsString() + ", qualifier="
@@ -1450,7 +1641,7 @@ public class MetaTableAccessor {
 
   private static Put addRegionInfo(final Put p, final HRegionInfo hri)
     throws IOException {
-    p.addImmutable(HConstants.CATALOG_FAMILY, HConstants.REGIONINFO_QUALIFIER,
+    p.addImmutable(getCatalogFamily(), HConstants.REGIONINFO_QUALIFIER,
       hri.toByteArray());
     return p;
   }
@@ -1459,20 +1650,20 @@ public class MetaTableAccessor {
     // using regionserver's local time as the timestamp of Put.
     // See: HBASE-11536
     long now = EnvironmentEdgeManager.currentTime();
-    p.addImmutable(HConstants.CATALOG_FAMILY, getServerColumn(replicaId), now,
+    p.addImmutable(getCatalogFamily(), getServerColumn(replicaId), now,
       Bytes.toBytes(sn.getHostAndPort()));
-    p.addImmutable(HConstants.CATALOG_FAMILY, getStartCodeColumn(replicaId), now,
+    p.addImmutable(getCatalogFamily(), getStartCodeColumn(replicaId), now,
       Bytes.toBytes(sn.getStartcode()));
-    p.addImmutable(HConstants.CATALOG_FAMILY, getSeqNumColumn(replicaId), now,
+    p.addImmutable(getCatalogFamily(), getSeqNumColumn(replicaId), now,
       Bytes.toBytes(openSeqNum));
     return p;
   }
 
   public static Put addEmptyLocation(final Put p, int replicaId) {
     long now = EnvironmentEdgeManager.currentTime();
-    p.addImmutable(HConstants.CATALOG_FAMILY, getServerColumn(replicaId), now, null);
-    p.addImmutable(HConstants.CATALOG_FAMILY, getStartCodeColumn(replicaId), now, null);
-    p.addImmutable(HConstants.CATALOG_FAMILY, getSeqNumColumn(replicaId), now, null);
+    p.addImmutable(getCatalogFamily(), getServerColumn(replicaId), now, null);
+    p.addImmutable(getCatalogFamily(), getStartCodeColumn(replicaId), now, null);
+    p.addImmutable(getCatalogFamily(), getSeqNumColumn(replicaId), now, null);
     return p;
   }
 }

http://git-wip-us.apache.org/repos/asf/hbase/blob/f5e2a823/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ConnectionCallable.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ConnectionCallable.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ConnectionCallable.java
new file mode 100644
index 0000000..3f44927
--- /dev/null
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ConnectionCallable.java
@@ -0,0 +1,56 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.hbase.client;
+
+import java.io.Closeable;
+import java.io.IOException;
+
+/**
+ * A RetryingCallable for generic connection operations.
+ * @param <V> return type
+ */
+abstract class ConnectionCallable<V> implements RetryingCallable<V>, Closeable {
+  protected Connection connection;
+
+  public ConnectionCallable(final Connection connection) {
+    this.connection = connection;
+  }
+
+  @Override
+  public void prepare(boolean reload) throws IOException {
+  }
+
+  @Override
+  public void close() throws IOException {
+  }
+
+  @Override
+  public void throwable(Throwable t, boolean retrying) {
+  }
+
+  @Override
+  public String getExceptionMessageAdditionalDetail() {
+    return "";
+  }
+
+  @Override
+  public long sleep(long pause, int tries) {
+    return ConnectionUtils.getPauseTime(pause, tries);
+  }
+}
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/hbase/blob/f5e2a823/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ConnectionManager.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ConnectionManager.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ConnectionManager.java
index 9b8f324..4b267c0 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ConnectionManager.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ConnectionManager.java
@@ -18,6 +18,8 @@
  */
 package org.apache.hadoop.hbase.client;
 
+import javax.annotation.Nonnull;
+import javax.annotation.Nullable;
 import java.io.Closeable;
 import java.io.IOException;
 import java.io.InterruptedIOException;
@@ -37,9 +39,12 @@ import java.util.concurrent.ExecutorService;
 import java.util.concurrent.LinkedBlockingQueue;
 import java.util.concurrent.ThreadPoolExecutor;
 import java.util.concurrent.TimeUnit;
-import java.util.concurrent.atomic.AtomicBoolean;
 import java.util.concurrent.atomic.AtomicInteger;
 
+import com.google.common.annotations.VisibleForTesting;
+import com.google.protobuf.BlockingRpcChannel;
+import com.google.protobuf.RpcController;
+import com.google.protobuf.ServiceException;
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.conf.Configuration;
@@ -60,8 +65,6 @@ import org.apache.hadoop.hbase.TableNotFoundException;
 import org.apache.hadoop.hbase.ZooKeeperConnectionException;
 import org.apache.hadoop.hbase.classification.InterfaceAudience;
 import org.apache.hadoop.hbase.client.AsyncProcess.AsyncRequestFuture;
-import org.apache.hadoop.hbase.client.MetaScanner.MetaScannerVisitor;
-import org.apache.hadoop.hbase.client.MetaScanner.MetaScannerVisitorBase;
 import org.apache.hadoop.hbase.client.backoff.ClientBackoffPolicy;
 import org.apache.hadoop.hbase.client.backoff.ClientBackoffPolicyFactory;
 import org.apache.hadoop.hbase.client.coprocessor.Batch;
@@ -168,6 +171,7 @@ import org.apache.hadoop.hbase.security.UserProvider;
 import org.apache.hadoop.hbase.util.Bytes;
 import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
 import org.apache.hadoop.hbase.util.ExceptionUtil;
+import org.apache.hadoop.hbase.util.Pair;
 import org.apache.hadoop.hbase.util.Threads;
 import org.apache.hadoop.hbase.zookeeper.MasterAddressTracker;
 import org.apache.hadoop.hbase.zookeeper.ZKUtil;
@@ -175,11 +179,6 @@ import org.apache.hadoop.hbase.zookeeper.ZooKeeperWatcher;
 import org.apache.hadoop.ipc.RemoteException;
 import org.apache.zookeeper.KeeperException;
 
-import com.google.common.annotations.VisibleForTesting;
-import com.google.protobuf.BlockingRpcChannel;
-import com.google.protobuf.RpcController;
-import com.google.protobuf.ServiceException;
-
 /**
  * An internal, non-instantiable class that manages creation of {@link HConnection}s.
  */
@@ -892,7 +891,7 @@ class ConnectionManager {
 
     @Override
     public boolean isTableEnabled(TableName tableName) throws IOException {
-      return this.registry.isTableOnlineState(tableName, true);
+      return getTableState(tableName).inStates(TableState.State.ENABLED);
     }
 
     @Override
@@ -902,7 +901,7 @@ class ConnectionManager {
 
     @Override
     public boolean isTableDisabled(TableName tableName) throws IOException {
-      return this.registry.isTableOnlineState(tableName, false);
+      return getTableState(tableName).inStates(TableState.State.DISABLED);
     }
 
     @Override
@@ -912,30 +911,7 @@ class ConnectionManager {
 
     @Override
     public boolean isTableAvailable(final TableName tableName) throws IOException {
-      final AtomicBoolean available = new AtomicBoolean(true);
-      final AtomicInteger regionCount = new AtomicInteger(0);
-      MetaScannerVisitor visitor = new MetaScannerVisitorBase() {
-        @Override
-        public boolean processRow(Result row) throws IOException {
-          HRegionInfo info = MetaScanner.getHRegionInfo(row);
-          if (info != null && !info.isSplitParent()) {
-            if (tableName.equals(info.getTable())) {
-              ServerName server = HRegionInfo.getServerName(row);
-              if (server == null) {
-                available.set(false);
-                return false;
-              }
-              regionCount.incrementAndGet();
-            } else if (tableName.compareTo(info.getTable()) < 0) {
-              // Return if we are done with the current table
-              return false;
-            }
-          }
-          return true;
-        }
-      };
-      MetaScanner.metaScan(this, visitor, tableName);
-      return available.get() && (regionCount.get() > 0);
+      return isTableAvailable(tableName, null);
     }
 
     @Override
@@ -944,44 +920,62 @@ class ConnectionManager {
     }
 
     @Override
-    public boolean isTableAvailable(final TableName tableName, final byte[][] splitKeys)
+    public boolean isTableAvailable(final TableName tableName, @Nullable final byte[][] splitKeys)
         throws IOException {
-      final AtomicBoolean available = new AtomicBoolean(true);
-      final AtomicInteger regionCount = new AtomicInteger(0);
-      MetaScannerVisitor visitor = new MetaScannerVisitorBase() {
-        @Override
-        public boolean processRow(Result row) throws IOException {
-          HRegionInfo info = MetaScanner.getHRegionInfo(row);
-          if (info != null && !info.isSplitParent()) {
-            if (tableName.equals(info.getTable())) {
-              ServerName server = HRegionInfo.getServerName(row);
-              if (server == null) {
-                available.set(false);
-                return false;
-              }
-              if (!Bytes.equals(info.getStartKey(), HConstants.EMPTY_BYTE_ARRAY)) {
-                for (byte[] splitKey : splitKeys) {
-                  // Just check if the splitkey is available
-                  if (Bytes.equals(info.getStartKey(), splitKey)) {
-                    regionCount.incrementAndGet();
-                    break;
-                  }
-                }
-              } else {
-                // Always empty start row should be counted
-                regionCount.incrementAndGet();
+      try {
+        if (!isTableEnabled(tableName)) {
+          LOG.debug("Table " + tableName + " not enabled");
+          return false;
+        }
+        ClusterConnection connection = getConnectionInternal(getConfiguration());
+        List<Pair<HRegionInfo, ServerName>> locations = MetaTableAccessor
+            .getTableRegionsAndLocations(getKeepAliveZooKeeperWatcher(),
+                connection, tableName, true);
+        int notDeployed = 0;
+        int regionCount = 0;
+        for (Pair<HRegionInfo, ServerName> pair : locations) {
+          HRegionInfo info = pair.getFirst();
+          if (pair.getSecond() == null) {
+            if (LOG.isDebugEnabled()) {
+              LOG.debug("Table " + tableName + " has not deployed region " + pair.getFirst()
+                  .getEncodedName());
+            }
+            notDeployed++;
+          } else if (splitKeys != null
+              && !Bytes.equals(info.getStartKey(), HConstants.EMPTY_BYTE_ARRAY)) {
+            for (byte[] splitKey : splitKeys) {
+              // Just check if the splitkey is available
+              if (Bytes.equals(info.getStartKey(), splitKey)) {
+                regionCount++;
+                break;
               }
-            } else if (tableName.compareTo(info.getTable()) < 0) {
-              // Return if we are done with the current table
-              return false;
             }
+          } else {
+            // Always empty start row should be counted
+            regionCount++;
+          }
+        }
+        if (notDeployed > 0) {
+          if (LOG.isDebugEnabled()) {
+            LOG.debug("Table " + tableName + " has " + notDeployed + " regions");
+          }
+          return false;
+        } else if (splitKeys != null && regionCount != splitKeys.length + 1) {
+          if (LOG.isDebugEnabled()) {
+            LOG.debug("Table " + tableName + " expected to have " + (splitKeys.length + 1)
+                + " regions, but only " + regionCount + " available");
+          }
+          return false;
+        } else {
+          if (LOG.isDebugEnabled()) {
+            LOG.debug("Table " + tableName + " should be available");
           }
           return true;
         }
-      };
-      MetaScanner.metaScan(this, visitor, tableName);
-      // +1 needs to be added so that the empty start row is also taken into account
-      return available.get() && (regionCount.get() == splitKeys.length + 1);
+      } catch (TableNotFoundException tnfe) {
+        LOG.warn("Table " + tableName + " not enabled, it is not exists");
+        return false;
+      }
     }
 
     @Override
@@ -2412,6 +2406,15 @@ class ConnectionManager {
       return getHTableDescriptorsByTableName(tableNames);
     }
 
+    @Nonnull
+    public TableState getTableState(TableName tableName) throws IOException {
+      ClusterConnection conn = getConnectionInternal(getConfiguration());
+      TableState tableState = MetaTableAccessor.getTableState(conn, tableName);
+      if (tableState == null)
+        throw new TableNotFoundException(tableName);
+      return tableState;
+    }
+
     @Override
     public NonceGenerator getNonceGenerator() {
       return this.nonceGenerator;
@@ -2433,7 +2436,7 @@ class ConnectionManager {
       GetTableDescriptorsResponse htds;
       try {
         GetTableDescriptorsRequest req =
-          RequestConverter.buildGetTableDescriptorsRequest(tableName);
+            RequestConverter.buildGetTableDescriptorsRequest(tableName);
         htds = master.getTableDescriptors(null, req);
       } catch (ServiceException se) {
         throw ProtobufUtil.getRemoteException(se);

http://git-wip-us.apache.org/repos/asf/hbase/blob/f5e2a823/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HBaseAdmin.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HBaseAdmin.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HBaseAdmin.java
index 520b953..cae6132 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HBaseAdmin.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HBaseAdmin.java
@@ -18,6 +18,7 @@
  */
 package org.apache.hadoop.hbase.client;
 
+import java.io.Closeable;
 import java.io.IOException;
 import java.io.InterruptedIOException;
 import java.net.SocketTimeoutException;
@@ -32,6 +33,9 @@ import java.util.concurrent.atomic.AtomicInteger;
 import java.util.concurrent.atomic.AtomicReference;
 import java.util.regex.Pattern;
 
+import com.google.common.annotations.VisibleForTesting;
+import com.google.protobuf.ByteString;
+import com.google.protobuf.ServiceException;
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.conf.Configuration;
@@ -147,10 +151,6 @@ import org.apache.hadoop.ipc.RemoteException;
 import org.apache.hadoop.util.StringUtils;
 import org.apache.zookeeper.KeeperException;
 
-import com.google.common.annotations.VisibleForTesting;
-import com.google.protobuf.ByteString;
-import com.google.protobuf.ServiceException;
-
 /**
  * HBaseAdmin is no longer a client API. It is marked InterfaceAudience.Private indicating that
  * this is an HBase-internal class as defined in
@@ -282,7 +282,12 @@ public class HBaseAdmin implements Admin {
    */
   @Override
   public boolean tableExists(final TableName tableName) throws IOException {
-    return MetaTableAccessor.tableExists(connection, tableName);
+    return executeCallable(new ConnectionCallable<Boolean>(getConnection()) {
+      @Override
+      public Boolean call(int callTimeout) throws ServiceException, IOException {
+        return MetaTableAccessor.tableExists(connection, tableName);
+      }
+    });
   }
 
   public boolean tableExists(final byte[] tableName)
@@ -543,11 +548,11 @@ public class HBaseAdmin implements Admin {
     }
     int numRegs = (splitKeys == null ? 1 : splitKeys.length + 1) * desc.getRegionReplication();
     int prevRegCount = 0;
-    boolean doneWithMetaScan = false;
+    boolean tableWasEnabled = false;
     for (int tries = 0; tries < this.numRetries * this.retryLongerMultiplier;
       ++tries) {
-      if (!doneWithMetaScan) {
-        // Wait for new table to come on-line
+      if (tableWasEnabled) {
+        // Wait all table regions comes online
         final AtomicInteger actualRegCount = new AtomicInteger(0);
         MetaScannerVisitor visitor = new MetaScannerVisitorBase() {
           @Override
@@ -595,17 +600,26 @@ public class HBaseAdmin implements Admin {
             tries = -1;
           }
         } else {
-          doneWithMetaScan = true;
-          tries = -1;
+          return;
         }
-      } else if (isTableEnabled(desc.getTableName())) {
-        return;
       } else {
-        try { // Sleep
-          Thread.sleep(getPauseTime(tries));
-        } catch (InterruptedException e) {
-          throw new InterruptedIOException("Interrupted when waiting" +
-            " for table to be enabled; meta scan was done");
+        try {
+          tableWasEnabled = isTableAvailable(desc.getTableName());
+        } catch (TableNotFoundException tnfe) {
+          LOG.debug(
+              "Table " + desc.getTableName() + " was not enabled, sleeping, still " + numRetries
+                  + " retries left");
+        }
+        if (tableWasEnabled) {
+          // no we will scan meta to ensure all regions are online
+          tries = -1;
+        } else {
+          try { // Sleep
+            Thread.sleep(getPauseTime(tries));
+          } catch (InterruptedException e) {
+            throw new InterruptedIOException("Interrupted when waiting" +
+                " for table to be enabled; meta scan was done");
+          }
         }
       }
     }
@@ -694,24 +708,11 @@ public class HBaseAdmin implements Admin {
     });
 
     int failures = 0;
-    // Wait until all regions deleted
     for (int tries = 0; tries < (this.numRetries * this.retryLongerMultiplier); tries++) {
       try {
-        // Find whether all regions are deleted.
-        List<RegionLocations> regionLations =
-            MetaScanner.listTableRegionLocations(conf, connection, tableName);
-
-        // let us wait until hbase:meta table is updated and
-        // HMaster removes the table from its HTableDescriptors
-        if (regionLations == null || regionLations.size() == 0) {
-          HTableDescriptor htd = getTableDescriptorByTableName(tableName);
-
-          if (htd == null) {
-            // table could not be found in master - we are done.
-            tableExists = false;
-            break;
-          }
-        }
+        tableExists = tableExists(tableName);
+        if (!tableExists)
+          break;
       } catch (IOException ex) {
         failures++;
         if(failures >= numRetries - 1) {           // no more tries left
@@ -1105,9 +1106,17 @@ public class HBaseAdmin implements Admin {
    * @throws IOException if a remote or network exception occurs
    */
   @Override
-  public boolean isTableEnabled(TableName tableName) throws IOException {
+  public boolean isTableEnabled(final TableName tableName) throws IOException {
     checkTableExistence(tableName);
-    return connection.isTableEnabled(tableName);
+    return executeCallable(new ConnectionCallable<Boolean>(getConnection()) {
+      @Override
+      public Boolean call(int callTimeout) throws ServiceException, IOException {
+        TableState tableState = MetaTableAccessor.getTableState(connection, tableName);
+        if (tableState == null)
+          throw new TableNotFoundException(tableName);
+        return tableState.inStates(TableState.State.ENABLED);
+      }
+    });
   }
 
   public boolean isTableEnabled(byte[] tableName) throws IOException {
@@ -2284,10 +2293,15 @@ public class HBaseAdmin implements Admin {
    */
   private TableName checkTableExists(final TableName tableName)
       throws IOException {
-    if (!MetaTableAccessor.tableExists(connection, tableName)) {
-      throw new TableNotFoundException(tableName);
-    }
-    return tableName;
+    return executeCallable(new ConnectionCallable<TableName>(getConnection()) {
+      @Override
+      public TableName call(int callTimeout) throws ServiceException, IOException {
+        if (!MetaTableAccessor.tableExists(connection, tableName)) {
+          throw new TableNotFoundException(tableName);
+        }
+        return tableName;
+      }
+    });
   }
 
   /**
@@ -3617,7 +3631,8 @@ public class HBaseAdmin implements Admin {
     });
   }
 
-  private <V> V executeCallable(MasterCallable<V> callable) throws IOException {
+  private <C extends RetryingCallable<V> & Closeable, V> V executeCallable(C callable)
+      throws IOException {
     RpcRetryingCaller<V> caller = rpcCallerFactory.newCaller();
     try {
       return caller.callWithRetries(callable, operationTimeout);

http://git-wip-us.apache.org/repos/asf/hbase/blob/f5e2a823/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HTable.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HTable.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HTable.java
index d6f066d..4afaf22 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HTable.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/HTable.java
@@ -128,6 +128,7 @@ public class HTable implements HTableInterface {
   protected AsyncProcess multiAp;
   private RpcRetryingCallerFactory rpcCallerFactory;
   private RpcControllerFactory rpcControllerFactory;
+  private final HTableDescriptor metaTableDescriptor;
 
   /**
    * Creates an object to access a HBase table.
@@ -172,12 +173,14 @@ public class HTable implements HTableInterface {
     this.cleanupPoolOnClose = this.cleanupConnectionOnClose = true;
     if (conf == null) {
       this.connection = null;
+      this.metaTableDescriptor = HTableDescriptor.metaTableDescriptor(new Configuration());
       return;
     }
     this.connection = ConnectionManager.getConnectionInternal(conf);
     this.configuration = conf;
 
     this.pool = getDefaultExecutor(conf);
+    this.metaTableDescriptor = HTableDescriptor.metaTableDescriptor(conf);
     this.finishSetup();
   }
 
@@ -197,6 +200,7 @@ public class HTable implements HTableInterface {
     this.configuration = connection.getConfiguration();
 
     this.pool = getDefaultExecutor(this.configuration);
+    this.metaTableDescriptor = HTableDescriptor.metaTableDescriptor(configuration);
     this.finishSetup();
   }
 
@@ -257,6 +261,7 @@ public class HTable implements HTableInterface {
     }
     this.tableName = tableName;
     this.cleanupConnectionOnClose = true;
+    this.metaTableDescriptor = HTableDescriptor.metaTableDescriptor(conf);
     this.finishSetup();
   }
 
@@ -303,6 +308,7 @@ public class HTable implements HTableInterface {
     this.cleanupConnectionOnClose = false;
     this.connection = connection;
     this.configuration = connection.getConfiguration();
+    this.metaTableDescriptor = HTableDescriptor.metaTableDescriptor(configuration);
     this.tableConfiguration = tableConfig;
     this.pool = pool;
     if (pool == null) {
@@ -329,6 +335,7 @@ public class HTable implements HTableInterface {
     tableConfiguration = new TableConfiguration(connection.getConfiguration());
     cleanupPoolOnClose = false;
     cleanupConnectionOnClose = false;
+    this.metaTableDescriptor = HTableDescriptor.metaTableDescriptor(new Configuration());
     // used from tests, don't trust the connection is real
     this.mutator = new BufferedMutatorImpl(conn, null, null, params);
   }
@@ -565,7 +572,7 @@ public class HTable implements HTableInterface {
     // TODO: This is the same as HBaseAdmin.getTableDescriptor(). Only keep one.
     if (tableName == null) return null;
     if (tableName.equals(TableName.META_TABLE_NAME)) {
-      return HTableDescriptor.META_TABLEDESC;
+      return metaTableDescriptor;
     }
     HTableDescriptor htd = executeMasterCallable(
       new MasterCallable<HTableDescriptor>(getConnection()) {

http://git-wip-us.apache.org/repos/asf/hbase/blob/f5e2a823/hbase-client/src/main/java/org/apache/hadoop/hbase/client/Registry.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/Registry.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/Registry.java
index 58ec3c4..9debd63 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/Registry.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/Registry.java
@@ -21,7 +21,6 @@ import java.io.IOException;
 
 import org.apache.hadoop.hbase.classification.InterfaceAudience;
 import org.apache.hadoop.hbase.RegionLocations;
-import org.apache.hadoop.hbase.TableName;
 
 /**
  * Cluster registry.
@@ -47,12 +46,6 @@ interface Registry {
   String getClusterId();
 
   /**
-   * @param enabled Return true if table is enabled
-   * @throws IOException
-   */
-  boolean isTableOnlineState(TableName tableName, boolean enabled) throws IOException;
-
-  /**
    * @return Count of 'running' regionservers
    * @throws IOException
    */

http://git-wip-us.apache.org/repos/asf/hbase/blob/f5e2a823/hbase-client/src/main/java/org/apache/hadoop/hbase/client/TableState.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/TableState.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/TableState.java
new file mode 100644
index 0000000..964d401
--- /dev/null
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/TableState.java
@@ -0,0 +1,219 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.hbase.client;
+
+import com.google.protobuf.InvalidProtocolBufferException;
+import org.apache.hadoop.hbase.TableName;
+import org.apache.hadoop.hbase.classification.InterfaceAudience;
+import org.apache.hadoop.hbase.classification.InterfaceStability;
+import org.apache.hadoop.hbase.exceptions.DeserializationException;
+import org.apache.hadoop.hbase.protobuf.generated.HBaseProtos;
+
+/**
+ * Represents table state.
+ */
+@InterfaceAudience.Private
+public class TableState {
+
+  @InterfaceAudience.Public
+  @InterfaceStability.Evolving
+  public static enum State {
+    ENABLED,
+    DISABLED,
+    DISABLING,
+    ENABLING;
+
+    /**
+     * Covert from PB version of State
+     *
+     * @param state convert from
+     * @return POJO
+     */
+    public static State convert(HBaseProtos.TableState.State state) {
+      State ret;
+      switch (state) {
+      case ENABLED:
+        ret = State.ENABLED;
+        break;
+      case DISABLED:
+        ret = State.DISABLED;
+        break;
+      case DISABLING:
+        ret = State.DISABLING;
+        break;
+      case ENABLING:
+        ret = State.ENABLING;
+        break;
+      default:
+        throw new IllegalStateException(state.toString());
+      }
+      return ret;
+    }
+
+    /**
+     * Covert to PB version of State
+     *
+     * @return PB
+     */
+    public HBaseProtos.TableState.State convert() {
+      HBaseProtos.TableState.State state;
+      switch (this) {
+      case ENABLED:
+        state = HBaseProtos.TableState.State.ENABLED;
+        break;
+      case DISABLED:
+        state = HBaseProtos.TableState.State.DISABLED;
+        break;
+      case DISABLING:
+        state = HBaseProtos.TableState.State.DISABLING;
+        break;
+      case ENABLING:
+        state = HBaseProtos.TableState.State.ENABLING;
+        break;
+      default:
+        throw new IllegalStateException(this.toString());
+      }
+      return state;
+    }
+
+  }
+
+  private final TableName tableName;
+  private final State state;
+
+  /**
+   * Create instance of TableState.
+   * @param state table state
+   */
+  public TableState(TableName tableName, State state) {
+    this.tableName = tableName;
+    this.state = state;
+  }
+
+  /**
+   * @return table state
+   */
+  public State getState() {
+    return state;
+  }
+
+  /**
+   * Table name for state
+   *
+   * @return milliseconds
+   */
+  public TableName getTableName() {
+    return tableName;
+  }
+
+  /**
+   * Check that table in given states
+   * @param state state
+   * @return true if satisfies
+   */
+  public boolean inStates(State state) {
+    return this.state.equals(state);
+  }
+
+  /**
+   * Check that table in given states
+   * @param states state list
+   * @return true if satisfies
+   */
+  public boolean inStates(State... states) {
+    for (State s : states) {
+      if (s.equals(this.state))
+        return true;
+    }
+    return false;
+  }
+
+
+  /**
+   * Covert to PB version of TableState
+   * @return PB
+   */
+  public HBaseProtos.TableState convert() {
+    return HBaseProtos.TableState.newBuilder()
+        .setState(this.state.convert()).build();
+  }
+
+  /**
+   * Covert from PB version of TableState
+   *
+   * @param tableName table this state of
+   * @param tableState convert from
+   * @return POJO
+   */
+  public static TableState convert(TableName tableName, HBaseProtos.TableState tableState) {
+    TableState.State state = State.convert(tableState.getState());
+    return new TableState(tableName, state);
+  }
+
+  public static TableState parseFrom(TableName tableName, byte[] bytes)
+      throws DeserializationException {
+    try {
+      return convert(tableName, HBaseProtos.TableState.parseFrom(bytes));
+    } catch (InvalidProtocolBufferException e) {
+      throw new DeserializationException(e);
+    }
+  }
+
+  /**
+   * Static version of state checker
+   * @param state desired
+   * @param target equals to any of
+   * @return true if satisfies
+   */
+  public static boolean isInStates(State state, State... target) {
+    for (State tableState : target) {
+      if (state.equals(tableState))
+        return true;
+    }
+    return false;
+  }
+
+  @Override
+  public boolean equals(Object o) {
+    if (this == o) return true;
+    if (o == null || getClass() != o.getClass()) return false;
+
+    TableState that = (TableState) o;
+
+    if (state != that.state) return false;
+    if (tableName != null ? !tableName.equals(that.tableName) : that.tableName != null)
+      return false;
+
+    return true;
+  }
+
+  @Override
+  public int hashCode() {
+    int result = (tableName != null ? tableName.hashCode() : 0);
+    result = 31 * result + (state != null ? state.hashCode() : 0);
+    return result;
+  }
+
+  @Override
+  public String toString() {
+    return "TableState{" +
+        ", tableName=" + tableName +
+        ", state=" + state +
+        '}';
+  }
+}

http://git-wip-us.apache.org/repos/asf/hbase/blob/f5e2a823/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ZooKeeperRegistry.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ZooKeeperRegistry.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ZooKeeperRegistry.java
index 3fb3a05..26aca18 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ZooKeeperRegistry.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ZooKeeperRegistry.java
@@ -18,7 +18,6 @@
 package org.apache.hadoop.hbase.client;
 
 import java.io.IOException;
-import java.io.InterruptedIOException;
 import java.util.List;
 
 import org.apache.commons.logging.Log;
@@ -27,10 +26,8 @@ import org.apache.hadoop.hbase.HRegionInfo;
 import org.apache.hadoop.hbase.HRegionLocation;
 import org.apache.hadoop.hbase.RegionLocations;
 import org.apache.hadoop.hbase.ServerName;
-import org.apache.hadoop.hbase.TableName;
 import org.apache.hadoop.hbase.zookeeper.MetaTableLocator;
 import org.apache.hadoop.hbase.zookeeper.ZKClusterId;
-import org.apache.hadoop.hbase.zookeeper.ZKTableStateClientSideReader;
 import org.apache.hadoop.hbase.zookeeper.ZKUtil;
 import org.apache.zookeeper.KeeperException;
 
@@ -117,24 +114,6 @@ class ZooKeeperRegistry implements Registry {
   }
 
   @Override
-  public boolean isTableOnlineState(TableName tableName, boolean enabled)
-  throws IOException {
-    ZooKeeperKeepAliveConnection zkw = hci.getKeepAliveZooKeeperWatcher();
-    try {
-      if (enabled) {
-        return ZKTableStateClientSideReader.isEnabledTable(zkw, tableName);
-      }
-      return ZKTableStateClientSideReader.isDisabledTable(zkw, tableName);
-    } catch (KeeperException e) {
-      throw new IOException("Enable/Disable failed", e);
-    } catch (InterruptedException e) {
-      throw new InterruptedIOException();
-    } finally {
-       zkw.close();
-    }
-  }
-
-  @Override
   public int getCurrentNrHRS() throws IOException {
     ZooKeeperKeepAliveConnection zkw = hci.getKeepAliveZooKeeperWatcher();
     try {

http://git-wip-us.apache.org/repos/asf/hbase/blob/f5e2a823/hbase-client/src/main/java/org/apache/hadoop/hbase/zookeeper/ZKTableStateClientSideReader.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/zookeeper/ZKTableStateClientSideReader.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/zookeeper/ZKTableStateClientSideReader.java
deleted file mode 100644
index d12231b..0000000
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/zookeeper/ZKTableStateClientSideReader.java
+++ /dev/null
@@ -1,202 +0,0 @@
-/**
- * Copyright The Apache Software Foundation
- *
- * Licensed to the Apache Software Foundation (ASF) under one
- * or more contributor license agreements.  See the NOTICE file
- * distributed with this work for additional information
- * regarding copyright ownership.  The ASF licenses this file
- * to you under the Apache License, Version 2.0 (the
- * "License"); you may not use this file except in compliance
- * with the License.  You may obtain a copy of the License at
- *
- *     http://www.apache.org/licenses/LICENSE-2.0
- *
- * Unless required by applicable law or agreed to in writing, software
- * distributed under the License is distributed on an "AS IS" BASIS,
- * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
- * See the License for the specific language governing permissions and
- * limitations under the License.
- */
-package org.apache.hadoop.hbase.zookeeper;
-
-import com.google.protobuf.InvalidProtocolBufferException;
-
-import org.apache.hadoop.hbase.classification.InterfaceAudience;
-import org.apache.hadoop.hbase.TableName;
-import org.apache.hadoop.hbase.exceptions.DeserializationException;
-import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
-import org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos;
-import org.apache.zookeeper.KeeperException;
-
-import java.util.HashSet;
-import java.util.List;
-import java.util.Set;
-
-/**
- * Non-instantiable class that provides helper functions to learn
- * about HBase table state for code running on client side (hence, not having
- * access to consensus context).
- *
- * Doesn't cache any table state, just goes directly to ZooKeeper.
- * TODO: decouple this class from ZooKeeper.
- */
-@InterfaceAudience.Private
-public class ZKTableStateClientSideReader {
-
-  private ZKTableStateClientSideReader() {}
-  
-  /**
-   * Go to zookeeper and see if state of table is {@code ZooKeeperProtos.Table.State#DISABLED}.
-   * This method does not use cache.
-   * This method is for clients other than AssignmentManager
-   * @param zkw ZooKeeperWatcher instance to use
-   * @param tableName table we're checking
-   * @return True if table is enabled.
-   * @throws KeeperException
-   */
-  public static boolean isDisabledTable(final ZooKeeperWatcher zkw,
-      final TableName tableName)
-      throws KeeperException, InterruptedException {
-    ZooKeeperProtos.Table.State state = getTableState(zkw, tableName);
-    return isTableState(ZooKeeperProtos.Table.State.DISABLED, state);
-  }
-
-  /**
-   * Go to zookeeper and see if state of table is {@code ZooKeeperProtos.Table.State#ENABLED}.
-   * This method does not use cache.
-   * This method is for clients other than AssignmentManager
-   * @param zkw ZooKeeperWatcher instance to use
-   * @param tableName table we're checking
-   * @return True if table is enabled.
-   * @throws KeeperException
-   */
-  public static boolean isEnabledTable(final ZooKeeperWatcher zkw,
-      final TableName tableName)
-      throws KeeperException, InterruptedException {
-    return getTableState(zkw, tableName) == ZooKeeperProtos.Table.State.ENABLED;
-  }
-
-  /**
-   * Go to zookeeper and see if state of table is {@code ZooKeeperProtos.Table.State#DISABLING}
-   * of {@code ZooKeeperProtos.Table.State#DISABLED}.
-   * This method does not use cache.
-   * This method is for clients other than AssignmentManager.
-   * @param zkw ZooKeeperWatcher instance to use
-   * @param tableName table we're checking
-   * @return True if table is enabled.
-   * @throws KeeperException
-   */
-  public static boolean isDisablingOrDisabledTable(final ZooKeeperWatcher zkw,
-      final TableName tableName)
-      throws KeeperException, InterruptedException {
-    ZooKeeperProtos.Table.State state = getTableState(zkw, tableName);
-    return isTableState(ZooKeeperProtos.Table.State.DISABLING, state) ||
-      isTableState(ZooKeeperProtos.Table.State.DISABLED, state);
-  }
-
-  /**
-   * Gets a list of all the tables set as disabled in zookeeper.
-   * @return Set of disabled tables, empty Set if none
-   * @throws KeeperException
-   */
-  public static Set<TableName> getDisabledTables(ZooKeeperWatcher zkw)
-      throws KeeperException, InterruptedException {
-    Set<TableName> disabledTables = new HashSet<TableName>();
-    List<String> children =
-      ZKUtil.listChildrenNoWatch(zkw, zkw.tableZNode);
-    for (String child: children) {
-      TableName tableName =
-          TableName.valueOf(child);
-      ZooKeeperProtos.Table.State state = getTableState(zkw, tableName);
-      if (state == ZooKeeperProtos.Table.State.DISABLED) disabledTables.add(tableName);
-    }
-    return disabledTables;
-  }
-
-  /**
-   * Gets a list of all the tables set as disabled in zookeeper.
-   * @return Set of disabled tables, empty Set if none
-   * @throws KeeperException
-   */
-  public static Set<TableName> getDisabledOrDisablingTables(ZooKeeperWatcher zkw)
-      throws KeeperException, InterruptedException {
-    return
-        getTablesInStates(
-          zkw,
-          ZooKeeperProtos.Table.State.DISABLED,
-          ZooKeeperProtos.Table.State.DISABLING);
-  }
-
-  /**
-   * Gets a list of all the tables set as enabling in zookeeper.
-   * @param zkw ZooKeeperWatcher instance to use
-   * @return Set of enabling tables, empty Set if none
-   * @throws KeeperException
-   * @throws InterruptedException
-   */
-  public static Set<TableName> getEnablingTables(ZooKeeperWatcher zkw)
-      throws KeeperException, InterruptedException {
-    return getTablesInStates(zkw, ZooKeeperProtos.Table.State.ENABLING);
-  }
-
-  /**
-   * Gets a list of tables that are set as one of the passing in states in zookeeper.
-   * @param zkw ZooKeeperWatcher instance to use
-   * @param states the list of states that a table could be in
-   * @return Set of tables in one of the states, empty Set if none
-   * @throws KeeperException
-   * @throws InterruptedException
-   */
-  private static Set<TableName> getTablesInStates(
-    ZooKeeperWatcher zkw,
-    ZooKeeperProtos.Table.State... states)
-      throws KeeperException, InterruptedException {
-    Set<TableName> tableNameSet = new HashSet<TableName>();
-    List<String> children = ZKUtil.listChildrenNoWatch(zkw, zkw.tableZNode);
-    TableName tableName;
-    ZooKeeperProtos.Table.State tableState;
-    for (String child: children) {
-      tableName = TableName.valueOf(child);
-      tableState = getTableState(zkw, tableName);
-      for (ZooKeeperProtos.Table.State state : states) {
-         if (tableState == state) {
-           tableNameSet.add(tableName);
-           break;
-         }
-      }
-    }
-    return tableNameSet;
-  }
-
-  static boolean isTableState(final ZooKeeperProtos.Table.State expectedState,
-      final ZooKeeperProtos.Table.State currentState) {
-    return currentState != null && currentState.equals(expectedState);
-  }
-
-  /**
-   * @param zkw ZooKeeperWatcher instance to use
-   * @param tableName table we're checking
-   * @return Null or {@link ZooKeeperProtos.Table.State} found in znode.
-   * @throws KeeperException
-   */
-  static ZooKeeperProtos.Table.State getTableState(final ZooKeeperWatcher zkw,
-      final TableName tableName)
-      throws KeeperException, InterruptedException {
-    String znode = ZKUtil.joinZNode(zkw.tableZNode, tableName.getNameAsString());
-    byte [] data = ZKUtil.getData(zkw, znode);
-    if (data == null || data.length <= 0) return null;
-    try {
-      ProtobufUtil.expectPBMagicPrefix(data);
-      ZooKeeperProtos.Table.Builder builder = ZooKeeperProtos.Table.newBuilder();
-      int magicLen = ProtobufUtil.lengthOfPBMagic();
-      ZooKeeperProtos.Table t = builder.mergeFrom(data, magicLen, data.length - magicLen).build();
-      return t.getState();
-    } catch (InvalidProtocolBufferException e) {
-      KeeperException ke = new KeeperException.DataInconsistencyException();
-      ke.initCause(e);
-      throw ke;
-    } catch (DeserializationException e) {
-      throw ZKUtil.convert(e);
-    }
-  }
-}

http://git-wip-us.apache.org/repos/asf/hbase/blob/f5e2a823/hbase-client/src/test/java/org/apache/hadoop/hbase/client/TestClientNoCluster.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/test/java/org/apache/hadoop/hbase/client/TestClientNoCluster.java b/hbase-client/src/test/java/org/apache/hadoop/hbase/client/TestClientNoCluster.java
index 2d50c1b..dbe6204 100644
--- a/hbase-client/src/test/java/org/apache/hadoop/hbase/client/TestClientNoCluster.java
+++ b/hbase-client/src/test/java/org/apache/hadoop/hbase/client/TestClientNoCluster.java
@@ -129,12 +129,6 @@ public class TestClientNoCluster extends Configured implements Tool {
     }
 
     @Override
-    public boolean isTableOnlineState(TableName tableName, boolean enabled)
-    throws IOException {
-      return enabled;
-    }
-
-    @Override
     public int getCurrentNrHRS() throws IOException {
       return 1;
     }

http://git-wip-us.apache.org/repos/asf/hbase/blob/f5e2a823/hbase-common/src/main/java/org/apache/hadoop/hbase/HConstants.java
----------------------------------------------------------------------
diff --git a/hbase-common/src/main/java/org/apache/hadoop/hbase/HConstants.java b/hbase-common/src/main/java/org/apache/hadoop/hbase/HConstants.java
index 2704516..42254c6 100644
--- a/hbase-common/src/main/java/org/apache/hadoop/hbase/HConstants.java
+++ b/hbase-common/src/main/java/org/apache/hadoop/hbase/HConstants.java
@@ -19,7 +19,6 @@ package org.apache.hadoop.hbase;
 
 import static org.apache.hadoop.hbase.io.hfile.BlockType.MAGIC_LENGTH;
 
-import java.nio.ByteBuffer;
 import java.nio.charset.Charset;
 import java.util.Arrays;
 import java.util.Collections;
@@ -443,6 +442,16 @@ public final class HConstants {
   /** The upper-half merge region column qualifier */
   public static final byte[] MERGEB_QUALIFIER = Bytes.toBytes("mergeB");
 
+  /** The catalog family as a string*/
+  public static final String TABLE_FAMILY_STR = "table";
+
+  /** The catalog family */
+  public static final byte [] TABLE_FAMILY = Bytes.toBytes(TABLE_FAMILY_STR);
+
+  /** The serialized table state qualifier */
+  public static final byte[] TABLE_STATE_QUALIFIER = Bytes.toBytes("state");
+
+
   /**
    * The meta table version column qualifier.
    * We keep current version of the meta table in this column in <code>-ROOT-</code>
@@ -730,7 +739,8 @@ public final class HConstants {
   /**
    * The client scanner timeout period in milliseconds.
    */
-  public static final String HBASE_CLIENT_SCANNER_TIMEOUT_PERIOD = "hbase.client.scanner.timeout.period";
+  public static final String HBASE_CLIENT_SCANNER_TIMEOUT_PERIOD =
+      "hbase.client.scanner.timeout.period";
 
   /**
    * Use {@link #HBASE_CLIENT_SCANNER_TIMEOUT_PERIOD} instead.