You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by ap...@apache.org on 2018/01/25 00:57:33 UTC

[5/5] hbase git commit: HBASE-19756 Master NPE during completed failed proc eviction

HBASE-19756 Master NPE during completed failed proc eviction

Signed-off-by: Andrew Purtell <ap...@apache.org>


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

Branch: refs/heads/branch-1.3
Commit: a02b3d9f7e340863fcfc10c7e1f3880fa05ce839
Parents: 40e78e5
Author: Thiruvel Thirumoolan <th...@gmail.com>
Authored: Wed Jan 10 15:01:57 2018 -0800
Committer: Andrew Purtell <ap...@apache.org>
Committed: Wed Jan 24 16:49:08 2018 -0800

----------------------------------------------------------------------
 .../hbase/procedure2/ProcedureExecutor.java     |   9 +-
 .../client/TestNonceProcCleanerOnFailure.java   | 101 +++++++++++++++++++
 2 files changed, 106 insertions(+), 4 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/hbase/blob/a02b3d9f/hbase-procedure/src/main/java/org/apache/hadoop/hbase/procedure2/ProcedureExecutor.java
----------------------------------------------------------------------
diff --git a/hbase-procedure/src/main/java/org/apache/hadoop/hbase/procedure2/ProcedureExecutor.java b/hbase-procedure/src/main/java/org/apache/hadoop/hbase/procedure2/ProcedureExecutor.java
index beba53b..51fd62f 100644
--- a/hbase-procedure/src/main/java/org/apache/hadoop/hbase/procedure2/ProcedureExecutor.java
+++ b/hbase-procedure/src/main/java/org/apache/hadoop/hbase/procedure2/ProcedureExecutor.java
@@ -181,13 +181,14 @@ public class ProcedureExecutor<TEnvironment> {
           if (isDebugEnabled) {
             LOG.debug("Evict completed procedure: " + procInfo);
           }
-          store.delete(entry.getKey());
-          it.remove();
-
           NonceKey nonceKey = procInfo.getNonceKey();
-          if (nonceKey != null) {
+          // Nonce procedures aren't persisted in WAL.
+          if (nonceKey == null) {
+            store.delete(entry.getKey());
+          } else {
             nonceKeysToProcIdsMap.remove(nonceKey);
           }
+          it.remove();
         }
       }
     }

http://git-wip-us.apache.org/repos/asf/hbase/blob/a02b3d9f/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestNonceProcCleanerOnFailure.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestNonceProcCleanerOnFailure.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestNonceProcCleanerOnFailure.java
new file mode 100644
index 0000000..bbf96b0
--- /dev/null
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestNonceProcCleanerOnFailure.java
@@ -0,0 +1,101 @@
+/*
+ * 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 static org.junit.Assert.fail;
+
+import java.io.IOException;
+import java.util.List;
+
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.hbase.HBaseTestingUtility;
+import org.apache.hadoop.hbase.HRegionInfo;
+import org.apache.hadoop.hbase.HTableDescriptor;
+import org.apache.hadoop.hbase.ProcedureInfo;
+import org.apache.hadoop.hbase.TableName;
+import org.apache.hadoop.hbase.coprocessor.BaseMasterObserver;
+import org.apache.hadoop.hbase.coprocessor.CoprocessorHost;
+import org.apache.hadoop.hbase.coprocessor.MasterCoprocessorEnvironment;
+import org.apache.hadoop.hbase.coprocessor.ObserverContext;
+import org.apache.hadoop.hbase.protobuf.generated.ProcedureProtos;
+import org.apache.hadoop.hbase.security.AccessDeniedException;
+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.Test;
+import org.junit.experimental.categories.Category;
+
+/**
+ * Check if CompletedProcedureCleaner cleans up failed nonce procedures.
+ */
+@Category(MediumTests.class)
+public class TestNonceProcCleanerOnFailure {
+  private static final Log LOG = LogFactory.getLog(TestNonceProcCleanerOnFailure.class);
+
+  protected static HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
+  private static final TableName TABLE = TableName.valueOf("test");
+  private static final byte[] FAMILY = Bytes.toBytesBinary("f");
+  private static final int evictionDelay = 10 * 1000;
+
+  @BeforeClass
+  public static void setUpBeforeClass() throws Exception {
+    Configuration conf = TEST_UTIL.getConfiguration();
+    conf.setInt("hbase.procedure.cleaner.evict.ttl", evictionDelay);
+    conf.set(CoprocessorHost.MASTER_COPROCESSOR_CONF_KEY, CreateFailObserver.class.getName());
+    TEST_UTIL.startMiniCluster(3);
+  }
+
+  @AfterClass
+  public static void tearDownAfterClass() throws Exception {
+    TEST_UTIL.cleanupTestDir();
+    TEST_UTIL.cleanupDataTestDirOnTestFS();
+    TEST_UTIL.shutdownMiniCluster();
+  }
+
+  @Test
+  public void testFailCreateTable() throws Exception {
+    try {
+      TEST_UTIL.createTable(TABLE, FAMILY);
+    } catch (AccessDeniedException e) {
+      LOG.debug("Ignoring exception: ", e);
+      Thread.sleep(evictionDelay * 3);
+    }
+    List<ProcedureInfo> procedureInfos =
+        TEST_UTIL.getMiniHBaseCluster().getMaster().getMasterProcedureExecutor().listProcedures();
+    for (ProcedureInfo procedureInfo : procedureInfos) {
+      if (procedureInfo.getProcName().equals("CreateTableProcedure")
+          && procedureInfo.getProcState() == ProcedureProtos.ProcedureState.ROLLEDBACK) {
+        fail("Found procedure " + procedureInfo + " that hasn't been cleaned up");
+      }
+    }
+  }
+
+  public static class CreateFailObserver extends BaseMasterObserver {
+
+    @Override
+    public void preCreateTable(ObserverContext<MasterCoprocessorEnvironment> ctx,
+        HTableDescriptor desc, HRegionInfo[] regions) throws IOException {
+
+      if (desc.getTableName().equals(TABLE)) {
+        throw new AccessDeniedException("Don't allow creation of table");
+      }
+    }
+  }
+}