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

[2/2] hbase git commit: HBASE-19953 Ensure post DDL hooks are only called after successful operations

HBASE-19953 Ensure post DDL hooks are only called after successful operations

The 1.x functionality of Master DDL operations is that "post" observer hooks
are only invoked when the DDL action was successful. With the async-ness of
ProcV2, we find ourselves in a case where the post-hook may be invoked before
the Procedure runs and fails. We need to introduce some blocking to wait and
see if the Procedure is going to fail on a precondition before invoking the hook.

Signed-off-by: Michael Stack <st...@apache.org>


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

Branch: refs/heads/master
Commit: d9b8dcc1d300ae114febc22dbc71866088387111
Parents: a27ef55
Author: Josh Elser <el...@apache.org>
Authored: Thu Feb 15 18:00:09 2018 -0500
Committer: Josh Elser <el...@apache.org>
Committed: Wed Feb 21 14:02:49 2018 -0500

----------------------------------------------------------------------
 .../hadoop/hbase/master/ClusterSchema.java      |  10 +-
 .../hbase/master/ClusterSchemaServiceImpl.java  |  16 +-
 .../org/apache/hadoop/hbase/master/HMaster.java |  43 ++-
 .../hbase/master/TableNamespaceManager.java     |   3 +-
 .../AbstractStateMachineNamespaceProcedure.java |  13 +
 .../procedure/CreateNamespaceProcedure.java     |  22 +-
 .../procedure/DeleteNamespaceProcedure.java     |  34 +-
 .../procedure/ModifyNamespaceProcedure.java     |  30 +-
 .../master/procedure/ProcedurePrepareLatch.java |  14 +
 .../procedure/TestMasterObserverPostCalls.java  | 368 +++++++++++++++++++
 10 files changed, 518 insertions(+), 35 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/hbase/blob/d9b8dcc1/hbase-server/src/main/java/org/apache/hadoop/hbase/master/ClusterSchema.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/ClusterSchema.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/ClusterSchema.java
index 746f9ad..56a1f33 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/ClusterSchema.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/ClusterSchema.java
@@ -22,6 +22,7 @@ import java.io.InterruptedIOException;
 import java.util.List;
 
 import org.apache.hadoop.hbase.NamespaceDescriptor;
+import org.apache.hadoop.hbase.master.procedure.ProcedurePrepareLatch;
 import org.apache.yetus.audience.InterfaceAudience;
 import org.apache.hadoop.hbase.util.NonceKey;
 
@@ -79,32 +80,35 @@ public interface ClusterSchema {
    * Create a new Namespace.
    * @param namespaceDescriptor descriptor for new Namespace
    * @param nonceKey A unique identifier for this operation from the client or process.
+   * @param latch A latch to block on for precondition validation
    * @return procedure id
    * @throws IOException Throws {@link ClusterSchemaException} and {@link InterruptedIOException}
    *    as well as {@link IOException}
    */
-  long createNamespace(NamespaceDescriptor namespaceDescriptor, NonceKey nonceKey)
+  long createNamespace(NamespaceDescriptor namespaceDescriptor, NonceKey nonceKey, ProcedurePrepareLatch latch)
   throws IOException;
 
   /**
    * Modify an existing Namespace.
    * @param nonceKey A unique identifier for this operation from the client or process.
+   * @param latch A latch to block on for precondition validation
    * @return procedure id
    * @throws IOException Throws {@link ClusterSchemaException} and {@link InterruptedIOException}
    *    as well as {@link IOException}
    */
-  long modifyNamespace(NamespaceDescriptor descriptor, NonceKey nonceKey)
+  long modifyNamespace(NamespaceDescriptor descriptor, NonceKey nonceKey, ProcedurePrepareLatch latch)
   throws IOException;
 
   /**
    * Delete an existing Namespace.
    * Only empty Namespaces (no tables) can be removed.
    * @param nonceKey A unique identifier for this operation from the client or process.
+   * @param latch A latch to block on for precondition validation
    * @return procedure id
    * @throws IOException Throws {@link ClusterSchemaException} and {@link InterruptedIOException}
    *    as well as {@link IOException}
    */
-  long deleteNamespace(String name, NonceKey nonceKey)
+  long deleteNamespace(String name, NonceKey nonceKey, ProcedurePrepareLatch latch)
   throws IOException;
 
   /**

http://git-wip-us.apache.org/repos/asf/hbase/blob/d9b8dcc1/hbase-server/src/main/java/org/apache/hadoop/hbase/master/ClusterSchemaServiceImpl.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/ClusterSchemaServiceImpl.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/ClusterSchemaServiceImpl.java
index 4dd8de0..7ad5b56 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/ClusterSchemaServiceImpl.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/ClusterSchemaServiceImpl.java
@@ -31,6 +31,7 @@ import org.apache.hadoop.hbase.master.procedure.CreateNamespaceProcedure;
 import org.apache.hadoop.hbase.master.procedure.DeleteNamespaceProcedure;
 import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv;
 import org.apache.hadoop.hbase.master.procedure.ModifyNamespaceProcedure;
+import org.apache.hadoop.hbase.master.procedure.ProcedurePrepareLatch;
 import org.apache.hadoop.hbase.procedure2.Procedure;
 import org.apache.hadoop.hbase.procedure2.ProcedureExecutor;
 import org.apache.hbase.thirdparty.com.google.common.util.concurrent.AbstractService;
@@ -88,26 +89,27 @@ class ClusterSchemaServiceImpl extends AbstractService implements ClusterSchemaS
   }
 
   @Override
-  public long createNamespace(NamespaceDescriptor namespaceDescriptor, final NonceKey nonceKey)
+  public long createNamespace(NamespaceDescriptor namespaceDescriptor, final NonceKey nonceKey,
+      final ProcedurePrepareLatch latch)
       throws IOException {
     return submitProcedure(new CreateNamespaceProcedure(
-      this.masterServices.getMasterProcedureExecutor().getEnvironment(), namespaceDescriptor),
+      this.masterServices.getMasterProcedureExecutor().getEnvironment(), namespaceDescriptor, latch),
         nonceKey);
   }
 
   @Override
-  public long modifyNamespace(NamespaceDescriptor namespaceDescriptor, final NonceKey nonceKey)
-      throws IOException {
+  public long modifyNamespace(NamespaceDescriptor namespaceDescriptor, final NonceKey nonceKey,
+      final ProcedurePrepareLatch latch) throws IOException {
     return submitProcedure(new ModifyNamespaceProcedure(
-      this.masterServices.getMasterProcedureExecutor().getEnvironment(), namespaceDescriptor),
+      this.masterServices.getMasterProcedureExecutor().getEnvironment(), namespaceDescriptor, latch),
         nonceKey);
   }
 
   @Override
-  public long deleteNamespace(String name, final NonceKey nonceKey)
+  public long deleteNamespace(String name, final NonceKey nonceKey, final ProcedurePrepareLatch latch)
       throws IOException {
     return submitProcedure(new DeleteNamespaceProcedure(
-      this.masterServices.getMasterProcedureExecutor().getEnvironment(), name),
+      this.masterServices.getMasterProcedureExecutor().getEnvironment(), name, latch),
       nonceKey);
   }
 

http://git-wip-us.apache.org/repos/asf/hbase/blob/d9b8dcc1/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java
index b639503..818a0ed 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/HMaster.java
@@ -117,6 +117,7 @@ import org.apache.hadoop.hbase.master.normalizer.RegionNormalizer;
 import org.apache.hadoop.hbase.master.normalizer.RegionNormalizerChore;
 import org.apache.hadoop.hbase.master.normalizer.RegionNormalizerFactory;
 import org.apache.hadoop.hbase.master.procedure.CreateTableProcedure;
+import org.apache.hadoop.hbase.master.procedure.DeleteNamespaceProcedure;
 import org.apache.hadoop.hbase.master.procedure.DeleteTableProcedure;
 import org.apache.hadoop.hbase.master.procedure.DisableTableProcedure;
 import org.apache.hadoop.hbase.master.procedure.EnableTableProcedure;
@@ -124,6 +125,7 @@ import org.apache.hadoop.hbase.master.procedure.MasterProcedureConstants;
 import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv;
 import org.apache.hadoop.hbase.master.procedure.MasterProcedureScheduler;
 import org.apache.hadoop.hbase.master.procedure.MasterProcedureUtil;
+import org.apache.hadoop.hbase.master.procedure.ModifyNamespaceProcedure;
 import org.apache.hadoop.hbase.master.procedure.ModifyTableProcedure;
 import org.apache.hadoop.hbase.master.procedure.ProcedurePrepareLatch;
 import org.apache.hadoop.hbase.master.procedure.RecoverMetaProcedure;
@@ -1743,7 +1745,10 @@ public class HMaster extends HRegionServer implements MasterServices {
 
         // TODO: We can handle/merge duplicate requests, and differentiate the case of
         //       TableExistsException by saying if the schema is the same or not.
-        ProcedurePrepareLatch latch = ProcedurePrepareLatch.createLatch();
+        //
+        // We need to wait for the procedure to potentially fail due to "prepare" sanity
+        // checks. This will block only the beginning of the procedure. See HBASE-19953.
+        ProcedurePrepareLatch latch = ProcedurePrepareLatch.createBlockingLatch();
         submitProcedure(new CreateTableProcedure(
             procedureExecutor.getEnvironment(), tableDescriptor, newRegions, latch));
         latch.await();
@@ -2089,7 +2094,10 @@ public class HMaster extends HRegionServer implements MasterServices {
         LOG.info(getClientIdAuditPrefix() + " delete " + tableName);
 
         // TODO: We can handle/merge duplicate request
-        ProcedurePrepareLatch latch = ProcedurePrepareLatch.createLatch();
+        //
+        // We need to wait for the procedure to potentially fail due to "prepare" sanity
+        // checks. This will block only the beginning of the procedure. See HBASE-19953.
+        ProcedurePrepareLatch latch = ProcedurePrepareLatch.createBlockingLatch();
         submitProcedure(new DeleteTableProcedure(procedureExecutor.getEnvironment(),
             tableName, latch));
         latch.await();
@@ -2276,7 +2284,10 @@ public class HMaster extends HRegionServer implements MasterServices {
         // we want to make sure that the table is prepared to be
         // enabled (the table is locked and the table state is set).
         // Note: if the procedure throws exception, we will catch it and rethrow.
-        final ProcedurePrepareLatch prepareLatch = ProcedurePrepareLatch.createLatch();
+        //
+        // We need to wait for the procedure to potentially fail due to "prepare" sanity
+        // checks. This will block only the beginning of the procedure. See HBASE-19953.
+        final ProcedurePrepareLatch prepareLatch = ProcedurePrepareLatch.createBlockingLatch();
         submitProcedure(new DisableTableProcedure(procedureExecutor.getEnvironment(),
             tableName, false, prepareLatch));
         prepareLatch.await();
@@ -2339,7 +2350,10 @@ public class HMaster extends HRegionServer implements MasterServices {
         LOG.info(getClientIdAuditPrefix() + " modify " + tableName);
 
         // Execute the operation synchronously - wait for the operation completes before continuing.
-        ProcedurePrepareLatch latch = ProcedurePrepareLatch.createLatch(2, 0);
+        //
+        // We need to wait for the procedure to potentially fail due to "prepare" sanity
+        // checks. This will block only the beginning of the procedure. See HBASE-19953.
+        ProcedurePrepareLatch latch = ProcedurePrepareLatch.createBlockingLatch();
         submitProcedure(new ModifyTableProcedure(procedureExecutor.getEnvironment(),
             descriptor, latch));
         latch.await();
@@ -2931,10 +2945,14 @@ public class HMaster extends HRegionServer implements MasterServices {
       @Override
       protected void run() throws IOException {
         getMaster().getMasterCoprocessorHost().preCreateNamespace(namespaceDescriptor);
+        // We need to wait for the procedure to potentially fail due to "prepare" sanity
+        // checks. This will block only the beginning of the procedure. See HBASE-19953.
+        ProcedurePrepareLatch latch = ProcedurePrepareLatch.createBlockingLatch();
         LOG.info(getClientIdAuditPrefix() + " creating " + namespaceDescriptor);
         // Execute the operation synchronously - wait for the operation to complete before
         // continuing.
-        setProcId(getClusterSchema().createNamespace(namespaceDescriptor, getNonceKey()));
+        setProcId(getClusterSchema().createNamespace(namespaceDescriptor, getNonceKey(), latch));
+        latch.await();
         getMaster().getMasterCoprocessorHost().postCreateNamespace(namespaceDescriptor);
       }
 
@@ -2963,10 +2981,14 @@ public class HMaster extends HRegionServer implements MasterServices {
       @Override
       protected void run() throws IOException {
         getMaster().getMasterCoprocessorHost().preModifyNamespace(namespaceDescriptor);
+        // We need to wait for the procedure to potentially fail due to "prepare" sanity
+        // checks. This will block only the beginning of the procedure. See HBASE-19953.
+        ProcedurePrepareLatch latch = ProcedurePrepareLatch.createBlockingLatch();
         LOG.info(getClientIdAuditPrefix() + " modify " + namespaceDescriptor);
         // Execute the operation synchronously - wait for the operation to complete before
         // continuing.
-        setProcId(getClusterSchema().modifyNamespace(namespaceDescriptor, getNonceKey()));
+        setProcId(getClusterSchema().modifyNamespace(namespaceDescriptor, getNonceKey(), latch));
+        latch.await();
         getMaster().getMasterCoprocessorHost().postModifyNamespace(namespaceDescriptor);
       }
 
@@ -2996,7 +3018,14 @@ public class HMaster extends HRegionServer implements MasterServices {
         LOG.info(getClientIdAuditPrefix() + " delete " + name);
         // Execute the operation synchronously - wait for the operation to complete before
         // continuing.
-        setProcId(getClusterSchema().deleteNamespace(name, getNonceKey()));
+        //
+        // We need to wait for the procedure to potentially fail due to "prepare" sanity
+        // checks. This will block only the beginning of the procedure. See HBASE-19953.
+        ProcedurePrepareLatch latch = ProcedurePrepareLatch.createBlockingLatch();
+        setProcId(submitProcedure(
+              new DeleteNamespaceProcedure(procedureExecutor.getEnvironment(), name, latch)));
+        latch.await();
+        // Will not be invoked in the face of Exception thrown by the Procedure's execution
         getMaster().getMasterCoprocessorHost().postDeleteNamespace(name);
       }
 

http://git-wip-us.apache.org/repos/asf/hbase/blob/d9b8dcc1/hbase-server/src/main/java/org/apache/hadoop/hbase/master/TableNamespaceManager.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/TableNamespaceManager.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/TableNamespaceManager.java
index 47b27f4..f334d32 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/TableNamespaceManager.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/TableNamespaceManager.java
@@ -46,6 +46,7 @@ import org.apache.hadoop.hbase.client.TableState;
 import org.apache.hadoop.hbase.constraint.ConstraintException;
 import org.apache.hadoop.hbase.exceptions.TimeoutIOException;
 import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv;
+import org.apache.hadoop.hbase.master.procedure.ProcedurePrepareLatch;
 import org.apache.hadoop.hbase.procedure2.ProcedureExecutor;
 import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
 import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
@@ -228,7 +229,7 @@ public class TableNamespaceManager implements Stoppable {
   private void blockingCreateNamespace(final NamespaceDescriptor namespaceDescriptor)
       throws IOException {
     ClusterSchema clusterSchema = this.masterServices.getClusterSchema();
-    long procId = clusterSchema.createNamespace(namespaceDescriptor, null);
+    long procId = clusterSchema.createNamespace(namespaceDescriptor, null, ProcedurePrepareLatch.getNoopLatch());
     block(this.masterServices, procId);
   }
 

http://git-wip-us.apache.org/repos/asf/hbase/blob/d9b8dcc1/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/AbstractStateMachineNamespaceProcedure.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/AbstractStateMachineNamespaceProcedure.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/AbstractStateMachineNamespaceProcedure.java
index 2c6ae34..574706a 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/AbstractStateMachineNamespaceProcedure.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/AbstractStateMachineNamespaceProcedure.java
@@ -31,12 +31,21 @@ public abstract class AbstractStateMachineNamespaceProcedure<TState>
     extends StateMachineProcedure<MasterProcedureEnv, TState>
     implements TableProcedureInterface {
 
+  private final ProcedurePrepareLatch syncLatch;
+
   protected AbstractStateMachineNamespaceProcedure() {
     // Required by the Procedure framework to create the procedure on replay
+    syncLatch = null;
   }
 
   protected AbstractStateMachineNamespaceProcedure(final MasterProcedureEnv env) {
+    this(env, null);
+  }
+
+  protected AbstractStateMachineNamespaceProcedure(final MasterProcedureEnv env,
+      final ProcedurePrepareLatch latch) {
     this.setOwner(env.getRequestUser());
+    this.syncLatch = latch;
   }
 
   protected abstract String getNamespaceName();
@@ -69,4 +78,8 @@ public abstract class AbstractStateMachineNamespaceProcedure<TState>
   protected void releaseLock(final MasterProcedureEnv env) {
     env.getProcedureScheduler().wakeNamespaceExclusiveLock(this, getNamespaceName());
   }
+
+  protected void releaseSyncLatch() {
+    ProcedurePrepareLatch.releaseLatch(syncLatch, this);
+  }
 }

http://git-wip-us.apache.org/repos/asf/hbase/blob/d9b8dcc1/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/CreateNamespaceProcedure.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/CreateNamespaceProcedure.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/CreateNamespaceProcedure.java
index 5eb56f5..c63f420 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/CreateNamespaceProcedure.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/CreateNamespaceProcedure.java
@@ -50,7 +50,12 @@ public class CreateNamespaceProcedure
 
   public CreateNamespaceProcedure(final MasterProcedureEnv env,
       final NamespaceDescriptor nsDescriptor) {
-    super(env);
+    this(env, nsDescriptor, null);
+  }
+
+  public CreateNamespaceProcedure(final MasterProcedureEnv env,
+      final NamespaceDescriptor nsDescriptor, ProcedurePrepareLatch latch) {
+    super(env, latch);
     this.nsDescriptor = nsDescriptor;
     this.traceEnabled = null;
   }
@@ -64,7 +69,12 @@ public class CreateNamespaceProcedure
     try {
       switch (state) {
       case CREATE_NAMESPACE_PREPARE:
-        prepareCreate(env);
+        boolean success = prepareCreate(env);
+        releaseSyncLatch();
+        if (!success) {
+          assert isFailed() : "createNamespace should have an exception here";
+          return Flow.NO_MORE_STATE;
+        }
         setNextState(CreateNamespaceState.CREATE_NAMESPACE_CREATE_DIRECTORY);
         break;
       case CREATE_NAMESPACE_CREATE_DIRECTORY:
@@ -102,6 +112,7 @@ public class CreateNamespaceProcedure
     if (state == CreateNamespaceState.CREATE_NAMESPACE_PREPARE) {
       // nothing to rollback, pre-create is just state checks.
       // TODO: coprocessor rollback semantic is still undefined.
+      releaseSyncLatch();
       return;
     }
     // The procedure doesn't have a rollback. The execution will succeed, at some point.
@@ -190,11 +201,14 @@ public class CreateNamespaceProcedure
    * @param env MasterProcedureEnv
    * @throws IOException
    */
-  private void prepareCreate(final MasterProcedureEnv env) throws IOException {
+  private boolean prepareCreate(final MasterProcedureEnv env) throws IOException {
     if (getTableNamespaceManager(env).doesNamespaceExist(nsDescriptor.getName())) {
-      throw new NamespaceExistException("Namespace " + nsDescriptor.getName() + " already exists");
+      setFailure("master-create-namespace",
+          new NamespaceExistException("Namespace " + nsDescriptor.getName() + " already exists"));
+      return false;
     }
     getTableNamespaceManager(env).validateTableAndRegionCount(nsDescriptor);
+    return true;
   }
 
   /**

http://git-wip-us.apache.org/repos/asf/hbase/blob/d9b8dcc1/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/DeleteNamespaceProcedure.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/DeleteNamespaceProcedure.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/DeleteNamespaceProcedure.java
index 1c587eb..5f7959e 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/DeleteNamespaceProcedure.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/DeleteNamespaceProcedure.java
@@ -57,7 +57,12 @@ public class DeleteNamespaceProcedure
   }
 
   public DeleteNamespaceProcedure(final MasterProcedureEnv env, final String namespaceName) {
-    super(env);
+    this(env, namespaceName, null);
+  }
+
+  public DeleteNamespaceProcedure(final MasterProcedureEnv env, final String namespaceName,
+      final ProcedurePrepareLatch latch) {
+    super(env, latch);
     this.namespaceName = namespaceName;
     this.nsDescriptor = null;
     this.traceEnabled = null;
@@ -74,7 +79,12 @@ public class DeleteNamespaceProcedure
     try {
       switch (state) {
       case DELETE_NAMESPACE_PREPARE:
-        prepareDelete(env);
+        boolean present = prepareDelete(env);
+        releaseSyncLatch();
+        if (!present) {
+          assert isFailed() : "Delete namespace should have an exception here";
+          return Flow.NO_MORE_STATE;
+        }
         setNextState(DeleteNamespaceState.DELETE_NAMESPACE_DELETE_FROM_NS_TABLE);
         break;
       case DELETE_NAMESPACE_DELETE_FROM_NS_TABLE:
@@ -113,6 +123,7 @@ public class DeleteNamespaceProcedure
       // nothing to rollback, pre is just table-state checks.
       // We can fail if the table does not exist or is not disabled.
       // TODO: coprocessor rollback semantic is still undefined.
+      releaseSyncLatch();
       return;
     }
 
@@ -188,27 +199,34 @@ public class DeleteNamespaceProcedure
    * @param env MasterProcedureEnv
    * @throws IOException
    */
-  private void prepareDelete(final MasterProcedureEnv env) throws IOException {
+  private boolean prepareDelete(final MasterProcedureEnv env) throws IOException {
     if (getTableNamespaceManager(env).doesNamespaceExist(namespaceName) == false) {
-      throw new NamespaceNotFoundException(namespaceName);
+      setFailure("master-delete-namespace", new NamespaceNotFoundException(namespaceName));
+      return false;
     }
     if (NamespaceDescriptor.RESERVED_NAMESPACES.contains(namespaceName)) {
-      throw new ConstraintException("Reserved namespace "+ namespaceName +" cannot be removed.");
+      setFailure("master-delete-namespace", new ConstraintException(
+          "Reserved namespace "+ namespaceName +" cannot be removed."));
+      return false;
     }
 
     int tableCount = 0;
     try {
       tableCount = env.getMasterServices().listTableDescriptorsByNamespace(namespaceName).size();
     } catch (FileNotFoundException fnfe) {
-      throw new NamespaceNotFoundException(namespaceName);
+      setFailure("master-delete-namespace", new NamespaceNotFoundException(namespaceName));
+      return false;
     }
     if (tableCount > 0) {
-      throw new ConstraintException("Only empty namespaces can be removed. " +
-          "Namespace "+ namespaceName + " has "+ tableCount +" tables");
+      setFailure("master-delete-namespace", new ConstraintException(
+          "Only empty namespaces can be removed. Namespace "+ namespaceName + " has "
+          + tableCount +" tables"));
+      return false;
     }
 
     // This is used for rollback
     nsDescriptor = getTableNamespaceManager(env).get(namespaceName);
+    return true;
   }
 
   /**

http://git-wip-us.apache.org/repos/asf/hbase/blob/d9b8dcc1/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/ModifyNamespaceProcedure.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/ModifyNamespaceProcedure.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/ModifyNamespaceProcedure.java
index 2a7dc5b..d3f982f 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/ModifyNamespaceProcedure.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/ModifyNamespaceProcedure.java
@@ -22,6 +22,7 @@ import java.io.IOException;
 
 import org.apache.hadoop.hbase.NamespaceDescriptor;
 import org.apache.hadoop.hbase.NamespaceNotFoundException;
+import org.apache.hadoop.hbase.constraint.ConstraintException;
 import org.apache.yetus.audience.InterfaceAudience;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
@@ -50,7 +51,12 @@ public class ModifyNamespaceProcedure
 
   public ModifyNamespaceProcedure(final MasterProcedureEnv env,
       final NamespaceDescriptor newNsDescriptor) {
-    super(env);
+    this(env, newNsDescriptor, null);
+  }
+
+  public ModifyNamespaceProcedure(final MasterProcedureEnv env,
+      final NamespaceDescriptor newNsDescriptor, final ProcedurePrepareLatch latch) {
+    super(env, latch);
     this.oldNsDescriptor = null;
     this.newNsDescriptor = newNsDescriptor;
     this.traceEnabled = null;
@@ -66,7 +72,12 @@ public class ModifyNamespaceProcedure
     try {
       switch (state) {
       case MODIFY_NAMESPACE_PREPARE:
-        prepareModify(env);
+        boolean success = prepareModify(env);
+        releaseSyncLatch();
+        if (!success) {
+          assert isFailed() : "Modify namespace should have an exception here";
+          return Flow.NO_MORE_STATE;
+        }
         setNextState(ModifyNamespaceState.MODIFY_NAMESPACE_UPDATE_NS_TABLE);
         break;
       case MODIFY_NAMESPACE_UPDATE_NS_TABLE:
@@ -96,6 +107,7 @@ public class ModifyNamespaceProcedure
     if (state == ModifyNamespaceState.MODIFY_NAMESPACE_PREPARE) {
       // nothing to rollback, pre-modify is just checks.
       // TODO: coprocessor rollback semantic is still undefined.
+      releaseSyncLatch();
       return;
     }
 
@@ -173,14 +185,22 @@ public class ModifyNamespaceProcedure
    * @param env MasterProcedureEnv
    * @throws IOException
    */
-  private void prepareModify(final MasterProcedureEnv env) throws IOException {
+  private boolean prepareModify(final MasterProcedureEnv env) throws IOException {
     if (getTableNamespaceManager(env).doesNamespaceExist(newNsDescriptor.getName()) == false) {
-      throw new NamespaceNotFoundException(newNsDescriptor.getName());
+      setFailure("master-modify-namespace", new NamespaceNotFoundException(
+            newNsDescriptor.getName()));
+      return false;
+    }
+    try {
+      getTableNamespaceManager(env).validateTableAndRegionCount(newNsDescriptor);
+    } catch (ConstraintException e) {
+      setFailure("master-modify-namespace", e);
+      return false;
     }
-    getTableNamespaceManager(env).validateTableAndRegionCount(newNsDescriptor);
 
     // This is used for rollback
     oldNsDescriptor = getTableNamespaceManager(env).get(newNsDescriptor.getName());
+    return true;
   }
 
   /**

http://git-wip-us.apache.org/repos/asf/hbase/blob/d9b8dcc1/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/ProcedurePrepareLatch.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/ProcedurePrepareLatch.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/ProcedurePrepareLatch.java
index f572cef..2011c0b 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/ProcedurePrepareLatch.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/ProcedurePrepareLatch.java
@@ -57,6 +57,20 @@ public abstract class ProcedurePrepareLatch {
     return hasProcedureSupport(major, minor) ? noopLatch : new CompatibilityLatch();
   }
 
+  /**
+   * Creates a latch which blocks.
+   */
+  public static ProcedurePrepareLatch createBlockingLatch() {
+    return new CompatibilityLatch();
+  }
+
+  /**
+   * Returns the singleton latch which does nothing.
+   */
+  public static ProcedurePrepareLatch getNoopLatch() {
+    return noopLatch;
+  }
+
   private static boolean hasProcedureSupport(int major, int minor) {
     return VersionInfoUtil.currentClientHasMinimumVersion(major, minor);
   }

http://git-wip-us.apache.org/repos/asf/hbase/blob/d9b8dcc1/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestMasterObserverPostCalls.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestMasterObserverPostCalls.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestMasterObserverPostCalls.java
new file mode 100644
index 0000000..65033a3
--- /dev/null
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestMasterObserverPostCalls.java
@@ -0,0 +1,368 @@
+/**
+ * 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.master.procedure;
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.fail;
+
+import java.io.IOException;
+import java.util.Optional;
+import java.util.concurrent.atomic.AtomicInteger;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.hbase.CoprocessorEnvironment;
+import org.apache.hadoop.hbase.HBaseClassTestRule;
+import org.apache.hadoop.hbase.HBaseTestingUtility;
+import org.apache.hadoop.hbase.NamespaceDescriptor;
+import org.apache.hadoop.hbase.TableName;
+import org.apache.hadoop.hbase.client.Admin;
+import org.apache.hadoop.hbase.client.ColumnFamilyDescriptorBuilder;
+import org.apache.hadoop.hbase.client.RegionInfo;
+import org.apache.hadoop.hbase.client.TableDescriptor;
+import org.apache.hadoop.hbase.client.TableDescriptorBuilder;
+import org.apache.hadoop.hbase.coprocessor.MasterCoprocessor;
+import org.apache.hadoop.hbase.coprocessor.MasterCoprocessorEnvironment;
+import org.apache.hadoop.hbase.coprocessor.MasterObserver;
+import org.apache.hadoop.hbase.coprocessor.ObserverContext;
+import org.apache.hadoop.hbase.master.HMaster;
+import org.apache.hadoop.hbase.master.MasterCoprocessorHost;
+import org.apache.hadoop.hbase.testclassification.MasterTests;
+import org.apache.hadoop.hbase.testclassification.MediumTests;
+import org.apache.hadoop.hbase.util.Bytes;
+import org.junit.AfterClass;
+import org.junit.BeforeClass;
+import org.junit.ClassRule;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Tests class that validates that "post" observer hook methods are only invoked when the operation was successful.
+ */
+@Category({MasterTests.class, MediumTests.class})
+public class TestMasterObserverPostCalls {
+
+  @ClassRule
+  public static final HBaseClassTestRule CLASS_RULE =
+      HBaseClassTestRule.forClass(TestMasterObserverPostCalls.class);
+
+  private static final Logger LOG = LoggerFactory.getLogger(TestMasterObserverPostCalls.class);
+  protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility();
+
+  @BeforeClass
+  public static void setupCluster() throws Exception {
+    setupConf(UTIL.getConfiguration());
+    UTIL.startMiniCluster(1);
+  }
+
+  private static void setupConf(Configuration conf) {
+    conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1);
+    conf.set(MasterCoprocessorHost.MASTER_COPROCESSOR_CONF_KEY,
+        MasterObserverForTest.class.getName());
+  }
+
+  @AfterClass
+  public static void cleanupTest() throws Exception {
+    try {
+      UTIL.shutdownMiniCluster();
+    } catch (Exception e) {
+      LOG.warn("failure shutting down cluster", e);
+    }
+  }
+
+  public static class MasterObserverForTest implements MasterCoprocessor, MasterObserver {
+    private AtomicInteger postHookCalls = null;
+
+    @Override
+    public Optional<MasterObserver> getMasterObserver() {
+      return Optional.of(this);
+    }
+
+    @Override
+    public void start(@SuppressWarnings("rawtypes") CoprocessorEnvironment ctx) throws IOException {
+      this.postHookCalls = new AtomicInteger(0);
+    }
+
+    @Override
+    public void postDeleteNamespace(ObserverContext<MasterCoprocessorEnvironment> ctx,
+        String namespace) {
+      postHookCalls.incrementAndGet();
+    }
+
+    @Override
+    public void postModifyNamespace(
+        ObserverContext<MasterCoprocessorEnvironment> ctx, NamespaceDescriptor desc) {
+      postHookCalls.incrementAndGet();
+    }
+
+    @Override
+    public void postCreateNamespace(
+        ObserverContext<MasterCoprocessorEnvironment> ctx, NamespaceDescriptor desc) {
+      postHookCalls.incrementAndGet();
+    }
+
+    @Override
+    public void postCreateTable(
+        ObserverContext<MasterCoprocessorEnvironment> ctx, TableDescriptor td,
+        RegionInfo[] regions) {
+      postHookCalls.incrementAndGet();
+    }
+
+    @Override
+    public void postModifyTable(
+        ObserverContext<MasterCoprocessorEnvironment> ctx, TableName tn, TableDescriptor td) {
+      postHookCalls.incrementAndGet();
+    }
+
+    @Override
+    public void postDisableTable(ObserverContext<MasterCoprocessorEnvironment> ctx, TableName tn) {
+      postHookCalls.incrementAndGet();
+    }
+
+    @Override
+    public void postDeleteTable(ObserverContext<MasterCoprocessorEnvironment> ctx, TableName tn) {
+      postHookCalls.incrementAndGet();
+    }
+  }
+
+  @Test
+  public void testPostDeleteNamespace() throws IOException {
+    final Admin admin = UTIL.getAdmin();
+    final String ns = "postdeletens";
+    final TableName tn1 = TableName.valueOf(ns, "table1");
+
+    admin.createNamespace(NamespaceDescriptor.create(ns).build());
+    admin.createTable(TableDescriptorBuilder.newBuilder(tn1)
+        .addColumnFamily(ColumnFamilyDescriptorBuilder.newBuilder(Bytes.toBytes("f1")).build())
+        .build());
+
+    HMaster master = UTIL.getMiniHBaseCluster().getMaster();
+    MasterObserverForTest observer = master.getMasterCoprocessorHost().findCoprocessor(
+        MasterObserverForTest.class);
+    int preCount = observer.postHookCalls.get();
+    try {
+      admin.deleteNamespace(ns);
+      fail("Deleting a non-empty namespace should be disallowed");
+    } catch (IOException e) {
+      // Pass
+    }
+    int postCount = observer.postHookCalls.get();
+    assertEquals("Expected no invocations of postDeleteNamespace when the operation fails",
+        preCount, postCount);
+
+    // Disable and delete the table so that we can delete the NS.
+    admin.disableTable(tn1);
+    admin.deleteTable(tn1);
+
+    // Validate that the postDeletNS hook is invoked
+    preCount = observer.postHookCalls.get();
+    admin.deleteNamespace(ns);
+    postCount = observer.postHookCalls.get();
+    assertEquals("Expected 1 invocation of postDeleteNamespace", preCount + 1, postCount);
+  }
+
+  @Test
+  public void testPostModifyNamespace() throws IOException {
+    final Admin admin = UTIL.getAdmin();
+    final String ns = "postmodifyns";
+
+    NamespaceDescriptor nsDesc = NamespaceDescriptor.create(ns).build();
+    admin.createNamespace(nsDesc);
+
+    HMaster master = UTIL.getMiniHBaseCluster().getMaster();
+    MasterObserverForTest observer = master.getMasterCoprocessorHost().findCoprocessor(
+        MasterObserverForTest.class);
+    int preCount = observer.postHookCalls.get();
+    try {
+      admin.modifyNamespace(NamespaceDescriptor.create("nonexistent").build());
+      fail("Modifying a missing namespace should fail");
+    } catch (IOException e) {
+      // Pass
+    }
+    int postCount = observer.postHookCalls.get();
+    assertEquals("Expected no invocations of postModifyNamespace when the operation fails",
+        preCount, postCount);
+
+    // Validate that the postDeletNS hook is invoked
+    preCount = observer.postHookCalls.get();
+    admin.modifyNamespace(
+        NamespaceDescriptor.create(nsDesc).addConfiguration("foo", "bar").build());
+    postCount = observer.postHookCalls.get();
+    assertEquals("Expected 1 invocation of postModifyNamespace", preCount + 1, postCount);
+  }
+
+  @Test
+  public void testPostCreateNamespace() throws IOException {
+    final Admin admin = UTIL.getAdmin();
+    final String ns = "postcreatens";
+
+    HMaster master = UTIL.getMiniHBaseCluster().getMaster();
+    MasterObserverForTest observer = master.getMasterCoprocessorHost().findCoprocessor(
+        MasterObserverForTest.class);
+
+    // Validate that the post hook is called
+    int preCount = observer.postHookCalls.get();
+    NamespaceDescriptor nsDesc = NamespaceDescriptor.create(ns).build();
+    admin.createNamespace(nsDesc);
+    int postCount = observer.postHookCalls.get();
+    assertEquals("Expected 1 invocation of postModifyNamespace", preCount + 1, postCount);
+
+    // Then, validate that it's not called when the call fails
+    preCount = observer.postHookCalls.get();
+    try {
+      admin.createNamespace(nsDesc);
+      fail("Creating an already present namespace should fail");
+    } catch (IOException e) {
+      // Pass
+    }
+    postCount = observer.postHookCalls.get();
+    assertEquals("Expected no invocations of postModifyNamespace when the operation fails",
+        preCount, postCount);
+  }
+
+  @Test
+  public void testPostCreateTable() throws IOException {
+    final Admin admin = UTIL.getAdmin();
+    final TableName tn = TableName.valueOf("postcreatetable");
+    final TableDescriptor td = TableDescriptorBuilder.newBuilder(tn).addColumnFamily(
+        ColumnFamilyDescriptorBuilder.newBuilder(Bytes.toBytes("f1")).build()).build();
+
+    HMaster master = UTIL.getMiniHBaseCluster().getMaster();
+    MasterObserverForTest observer = master.getMasterCoprocessorHost().findCoprocessor(
+        MasterObserverForTest.class);
+
+    // Validate that the post hook is called
+    int preCount = observer.postHookCalls.get();
+    admin.createTable(td);
+    int postCount = observer.postHookCalls.get();
+    assertEquals("Expected 1 invocation of postCreateTable", preCount + 1, postCount);
+
+    // Then, validate that it's not called when the call fails
+    preCount = observer.postHookCalls.get();
+    try {
+      admin.createTable(td);
+      fail("Creating an already present table should fail");
+    } catch (IOException e) {
+      // Pass
+    }
+    postCount = observer.postHookCalls.get();
+    assertEquals("Expected no invocations of postCreateTable when the operation fails",
+        preCount, postCount);
+  }
+
+  @Test
+  public void testPostModifyTable() throws IOException {
+    final Admin admin = UTIL.getAdmin();
+    final TableName tn = TableName.valueOf("postmodifytable");
+    final TableDescriptor td = TableDescriptorBuilder.newBuilder(tn).addColumnFamily(
+        ColumnFamilyDescriptorBuilder.newBuilder(Bytes.toBytes("f1")).build()).build();
+
+    HMaster master = UTIL.getMiniHBaseCluster().getMaster();
+    MasterObserverForTest observer = master.getMasterCoprocessorHost().findCoprocessor(
+        MasterObserverForTest.class);
+
+    // Create the table
+    admin.createTable(td);
+
+    // Validate that the post hook is called
+    int preCount = observer.postHookCalls.get();
+    admin.modifyTable(td);
+    int postCount = observer.postHookCalls.get();
+    assertEquals("Expected 1 invocation of postModifyTable", preCount + 1, postCount);
+
+    // Then, validate that it's not called when the call fails
+    preCount = observer.postHookCalls.get();
+    try {
+      admin.modifyTable(TableDescriptorBuilder.newBuilder(TableName.valueOf("missing"))
+          .addColumnFamily(td.getColumnFamily(Bytes.toBytes("f1"))).build());
+      fail("Modifying a missing table should fail");
+    } catch (IOException e) {
+      // Pass
+    }
+    postCount = observer.postHookCalls.get();
+    assertEquals("Expected no invocations of postModifyTable when the operation fails",
+        preCount, postCount);
+  }
+
+  @Test
+  public void testPostDisableTable() throws IOException {
+    final Admin admin = UTIL.getAdmin();
+    final TableName tn = TableName.valueOf("postdisabletable");
+    final TableDescriptor td = TableDescriptorBuilder.newBuilder(tn).addColumnFamily(
+        ColumnFamilyDescriptorBuilder.newBuilder(Bytes.toBytes("f1")).build()).build();
+
+    HMaster master = UTIL.getMiniHBaseCluster().getMaster();
+    MasterObserverForTest observer = master.getMasterCoprocessorHost().findCoprocessor(
+        MasterObserverForTest.class);
+
+    // Create the table and disable it
+    admin.createTable(td);
+
+    // Validate that the post hook is called
+    int preCount = observer.postHookCalls.get();
+    admin.disableTable(td.getTableName());
+    int postCount = observer.postHookCalls.get();
+    assertEquals("Expected 1 invocation of postDisableTable", preCount + 1, postCount);
+
+    // Then, validate that it's not called when the call fails
+    preCount = observer.postHookCalls.get();
+    try {
+      admin.disableTable(TableName.valueOf("Missing"));
+      fail("Disabling a missing table should fail");
+    } catch (IOException e) {
+      // Pass
+    }
+    postCount = observer.postHookCalls.get();
+    assertEquals("Expected no invocations of postDisableTable when the operation fails",
+        preCount, postCount);
+  }
+
+  @Test
+  public void testPostDeleteTable() throws IOException {
+    final Admin admin = UTIL.getAdmin();
+    final TableName tn = TableName.valueOf("postdeletetable");
+    final TableDescriptor td = TableDescriptorBuilder.newBuilder(tn).addColumnFamily(
+        ColumnFamilyDescriptorBuilder.newBuilder(Bytes.toBytes("f1")).build()).build();
+
+    HMaster master = UTIL.getMiniHBaseCluster().getMaster();
+    MasterObserverForTest observer = master.getMasterCoprocessorHost().findCoprocessor(
+        MasterObserverForTest.class);
+
+    // Create the table and disable it
+    admin.createTable(td);
+    admin.disableTable(td.getTableName());
+
+    // Validate that the post hook is called
+    int preCount = observer.postHookCalls.get();
+    admin.deleteTable(td.getTableName());
+    int postCount = observer.postHookCalls.get();
+    assertEquals("Expected 1 invocation of postDeleteTable", preCount + 1, postCount);
+
+    // Then, validate that it's not called when the call fails
+    preCount = observer.postHookCalls.get();
+    try {
+      admin.deleteTable(TableName.valueOf("missing"));
+      fail("Deleting a missing table should fail");
+    } catch (IOException e) {
+      // Pass
+    }
+    postCount = observer.postHookCalls.get();
+    assertEquals("Expected no invocations of postDeleteTable when the operation fails",
+        preCount, postCount);
+  }
+}