From ce33cf2d3d3044bc4a24180325830fc767745d6f Mon Sep 17 00:00:00 2001 From: Stephen Yuan Jiang Date: Wed, 28 Dec 2016 15:43:50 -0800 Subject: [PATCH] HBASE-17149 Procedure V2 - Fix nonce submission to avoid unnecessary calling coprocessor multiple times (Matteo Bertozzi) --- .../hbase/procedure2/ProcedureExecutor.java | 155 ++++-- .../procedure2/ProcedureTestingUtility.java | 20 +- .../hbase/procedure2/TestProcedureNonce.java | 285 +++++++++++ .../procedure2/TestProcedureRecovery.java | 43 -- .../apache/hadoop/hbase/master/HMaster.java | 459 ++++++++++-------- .../hadoop/hbase/master/MasterServices.java | 3 +- .../hbase/master/TableNamespaceManager.java | 6 +- .../master/procedure/MasterProcedureUtil.java | 88 ++++ .../hbase/master/MockNoopMasterServices.java | 6 +- .../hbase/master/TestCatalogJanitor.java | 3 +- .../TestAddColumnFamilyProcedure.java | 70 +-- .../TestCreateNamespaceProcedure.java | 65 +-- .../procedure/TestCreateTableProcedure.java | 41 +- .../TestDeleteColumnFamilyProcedure.java | 78 +-- .../TestDeleteNamespaceProcedure.java | 63 +-- .../procedure/TestDeleteTableProcedure.java | 44 +- .../procedure/TestDisableTableProcedure.java | 40 +- .../procedure/TestEnableTableProcedure.java | 46 +- .../procedure/TestMasterProcedureEvents.java | 10 +- .../TestModifyColumnFamilyProcedure.java | 31 +- .../TestModifyNamespaceProcedure.java | 36 +- .../procedure/TestModifyTableProcedure.java | 17 +- .../master/procedure/TestProcedureAdmin.java | 16 +- .../procedure/TestTruncateTableProcedure.java | 12 +- 24 files changed, 866 insertions(+), 771 deletions(-) create mode 100644 hbase-procedure/src/test/java/org/apache/hadoop/hbase/procedure2/TestProcedureNonce.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 d6ed2075357..5c1f30c7b95 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 @@ -53,8 +53,10 @@ import org.apache.hadoop.hbase.procedure2.util.TimeoutBlockingQueue.TimeoutRetri import org.apache.hadoop.hbase.protobuf.generated.ProcedureProtos.ProcedureState; import org.apache.hadoop.hbase.security.User; import org.apache.hadoop.hbase.util.EnvironmentEdgeManager; +import org.apache.hadoop.hbase.util.ForeignExceptionUtil; import org.apache.hadoop.hbase.util.NonceKey; import org.apache.hadoop.hbase.util.Pair; +import org.apache.hadoop.hbase.util.Threads; /** * Thread Pool that executes the submitted procedures. @@ -609,59 +611,142 @@ public class ProcedureExecutor { return waitingTimeout.remove(chore); } + // ========================================================================== + // Nonce Procedure helpers + // ========================================================================== + /** + * Create a NoneKey from the specified nonceGroup and nonce. + * @param nonceGroup + * @param nonce + * @return the generated NonceKey + */ + public NonceKey createNonceKey(final long nonceGroup, final long nonce) { + return (nonce == HConstants.NO_NONCE) ? null : new NonceKey(nonceGroup, nonce); + } + + /** + * Register a nonce for a procedure that is going to be submitted. + * A procId will be reserved and on submitProcedure(), + * the procedure with the specified nonce will take the reserved ProcId. + * If someone already reserved the nonce, this method will return the procId reserved, + * otherwise an invalid procId will be returned. and the caller should procede + * and submit the procedure. + * + * @param nonceKey A unique identifier for this operation from the client or process. + * @return the procId associated with the nonce, if any otherwise an invalid procId. + */ + public long registerNonce(final NonceKey nonceKey) { + if (nonceKey == null) return -1; + + // check if we have already a Reserved ID for the nonce + Long oldProcId = nonceKeysToProcIdsMap.get(nonceKey); + if (oldProcId == null) { + // reserve a new Procedure ID, this will be associated with the nonce + // and the procedure submitted with the specified nonce will use this ID. + final long newProcId = nextProcId(); + oldProcId = nonceKeysToProcIdsMap.putIfAbsent(nonceKey, newProcId); + if (oldProcId == null) return -1; + } + + // we found a registered nonce, but the procedure may not have been submitted yet. + // since the client expect the procedure to be submitted, spin here until it is. + final boolean isTraceEnabled = LOG.isTraceEnabled(); + while (isRunning() && + !(procedures.containsKey(oldProcId) || completed.containsKey(oldProcId)) && + nonceKeysToProcIdsMap.containsKey(nonceKey)) { + if (isTraceEnabled) { + LOG.trace("waiting for procId=" + oldProcId.longValue() + " to be submitted"); + } + Threads.sleep(100); + } + return oldProcId.longValue(); + } + + /** + * Remove the NonceKey if the procedure was not submitted to the executor. + * @param nonceKey A unique identifier for this operation from the client or process. + */ + public void unregisterNonceIfProcedureWasNotSubmitted(final NonceKey nonceKey) { + if (nonceKey == null) return; + + final Long procId = nonceKeysToProcIdsMap.get(nonceKey); + if (procId == null) return; + + // if the procedure was not submitted, remove the nonce + if (!(procedures.containsKey(procId) || completed.containsKey(procId))) { + nonceKeysToProcIdsMap.remove(nonceKey); + } + } + + /** + * If the failure failed before submitting it, we may want to give back the + * same error to the requests with the same nonceKey. + * + * @param nonceKey A unique identifier for this operation from the client or process + * @param procName name of the procedure, used to inform the user + * @param procOwner name of the owner of the procedure, used to inform the user + * @param exception the failure to report to the user + */ + public void setFailureResultForNonce(final NonceKey nonceKey, final String procName, + final User procOwner, final IOException exception) { + if (nonceKey == null) return; + + final Long procId = nonceKeysToProcIdsMap.get(nonceKey); + if (procId == null || completed.containsKey(procId)) return; + + final long currentTime = EnvironmentEdgeManager.currentTime(); + final ProcedureInfo result = new ProcedureInfo( + procId.longValue(), + procName, + procOwner != null ? procOwner.getShortName() : null, + ProcedureState.ROLLEDBACK, + -1, + nonceKey, + ForeignExceptionUtil.toProtoForeignException("ProcedureExecutor", exception), + currentTime, + currentTime, + null); + completed.putIfAbsent(procId, result); + } + + // ========================================================================== + // Submit/Abort Procedure + // ========================================================================== /** * Add a new root-procedure to the executor. * @param proc the new procedure to execute. * @return the procedure id, that can be used to monitor the operation */ public long submitProcedure(final Procedure proc) { - return submitProcedure(proc, HConstants.NO_NONCE, HConstants.NO_NONCE); + return submitProcedure(proc, null); } /** * Add a new root-procedure to the executor. * @param proc the new procedure to execute. - * @param nonceGroup - * @param nonce + * @param nonceKey the registered unique identifier for this operation from the client or process. * @return the procedure id, that can be used to monitor the operation */ - public long submitProcedure( - final Procedure proc, - final long nonceGroup, - final long nonce) { + @edu.umd.cs.findbugs.annotations.SuppressWarnings(value="NP_NULL_ON_SOME_PATH", + justification = "FindBugs is blind to the check-for-null") + public long submitProcedure(final Procedure proc, final NonceKey nonceKey) { Preconditions.checkArgument(proc.getState() == ProcedureState.INITIALIZING); - Preconditions.checkArgument(isRunning()); + Preconditions.checkArgument(isRunning(), "executor not running"); Preconditions.checkArgument(lastProcId.get() >= 0); - Preconditions.checkArgument(!proc.hasParent()); + Preconditions.checkArgument(!proc.hasParent(), "unexpected parent", proc); - Long currentProcId; - - // The following part of the code has to be synchronized to prevent multiple request - // with the same nonce to execute at the same time. - synchronized (this) { - // Check whether the proc exists. If exist, just return the proc id. - // This is to prevent the same proc to submit multiple times (it could happen - // when client could not talk to server and resubmit the same request). - NonceKey noncekey = null; - if (nonce != HConstants.NO_NONCE) { - noncekey = new NonceKey(nonceGroup, nonce); - currentProcId = nonceKeysToProcIdsMap.get(noncekey); - if (currentProcId != null) { - // Found the proc - return currentProcId; - } - } - - // Initialize the Procedure ID + final Long currentProcId; + if (nonceKey != null) { + currentProcId = nonceKeysToProcIdsMap.get(nonceKey); + Preconditions.checkArgument(currentProcId != null, + "expected nonceKey=" + nonceKey + " to be reserved, use registerNonce()"); + } else { currentProcId = nextProcId(); - proc.setProcId(currentProcId); + } - // This is new procedure. Set the noncekey and insert into the map. - if (noncekey != null) { - proc.setNonceKey(noncekey); - nonceKeysToProcIdsMap.put(noncekey, currentProcId); - } - } // end of synchronized (this) + // Initialize the procedure + proc.setNonceKey(nonceKey); + proc.setProcId(currentProcId.longValue()); // Commit the transaction store.insert(proc, null); diff --git a/hbase-procedure/src/test/java/org/apache/hadoop/hbase/procedure2/ProcedureTestingUtility.java b/hbase-procedure/src/test/java/org/apache/hadoop/hbase/procedure2/ProcedureTestingUtility.java index 8b49e53dc30..f5b0464fd76 100644 --- a/hbase-procedure/src/test/java/org/apache/hadoop/hbase/procedure2/ProcedureTestingUtility.java +++ b/hbase-procedure/src/test/java/org/apache/hadoop/hbase/procedure2/ProcedureTestingUtility.java @@ -40,6 +40,7 @@ import org.apache.hadoop.hbase.procedure2.store.NoopProcedureStore; import org.apache.hadoop.hbase.procedure2.store.wal.WALProcedureStore; import org.apache.hadoop.hbase.protobuf.generated.ErrorHandlingProtos.ForeignExceptionMessage; import org.apache.hadoop.hbase.protobuf.generated.ProcedureProtos.ProcedureState; +import org.apache.hadoop.hbase.util.NonceKey; import org.apache.hadoop.hbase.util.Threads; import static org.junit.Assert.assertEquals; @@ -158,13 +159,20 @@ public class ProcedureTestingUtility { } public static long submitAndWait(ProcedureExecutor procExecutor, Procedure proc, - final long nonceGroup, - final long nonce) { - long procId = procExecutor.submitProcedure(proc, nonceGroup, nonce); + final long nonceGroup, final long nonce) { + long procId = submitProcedure(procExecutor, proc, nonceGroup, nonce); waitProcedure(procExecutor, procId); return procId; } + public static long submitProcedure(ProcedureExecutor procExecutor, Procedure proc, + final long nonceGroup, final long nonce) { + final NonceKey nonceKey = procExecutor.createNonceKey(nonceGroup, nonce); + long procId = procExecutor.registerNonce(nonceKey); + assertFalse(procId >= 0); + return procExecutor.submitProcedure(proc, nonceKey); + } + public static void waitProcedure(ProcedureExecutor procExecutor, Procedure proc) { while (proc.getState() == ProcedureState.INITIALIZING) { Threads.sleepWithoutInterrupt(250); @@ -210,6 +218,12 @@ public class ProcedureTestingUtility { assertFalse(msg, result.isFailed()); } + public static Throwable assertProcFailed(final ProcedureInfo result) { + assertEquals(true, result.isFailed()); + LOG.info("procId=" + result.getProcId() + " exception: " + result.getException().getMessage()); + return getExceptionCause(result); + } + public static void assertIsAbortException(final ProcedureInfo result) { assertEquals(true, result.isFailed()); LOG.info(result.getExceptionFullMessage()); diff --git a/hbase-procedure/src/test/java/org/apache/hadoop/hbase/procedure2/TestProcedureNonce.java b/hbase-procedure/src/test/java/org/apache/hadoop/hbase/procedure2/TestProcedureNonce.java new file mode 100644 index 00000000000..f2754266280 --- /dev/null +++ b/hbase-procedure/src/test/java/org/apache/hadoop/hbase/procedure2/TestProcedureNonce.java @@ -0,0 +1,285 @@ +/** + * 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.procedure2; + +import java.io.IOException; +import java.util.concurrent.atomic.AtomicReference; +import java.util.concurrent.CountDownLatch; + +import org.apache.commons.logging.Log; +import org.apache.commons.logging.LogFactory; +import org.apache.hadoop.fs.FileSystem; +import org.apache.hadoop.fs.Path; +import org.apache.hadoop.hbase.HBaseCommonTestingUtility; +import org.apache.hadoop.hbase.ProcedureInfo; +import org.apache.hadoop.hbase.procedure2.store.ProcedureStore; +import org.apache.hadoop.hbase.testclassification.SmallTests; +import org.apache.hadoop.hbase.testclassification.MasterTests; +import org.apache.hadoop.hbase.security.User; +import org.apache.hadoop.hbase.util.Bytes; +import org.apache.hadoop.hbase.util.NonceKey; +import org.apache.hadoop.hbase.util.Threads; + +import org.junit.After; +import org.junit.Before; +import org.junit.Test; +import org.junit.experimental.categories.Category; + +import static org.junit.Assert.assertEquals; +import static org.junit.Assert.assertFalse; +import static org.junit.Assert.assertTrue; +import static org.junit.Assert.fail; + +@Category({MasterTests.class, SmallTests.class}) +public class TestProcedureNonce { + private static final Log LOG = LogFactory.getLog(TestProcedureNonce.class); + + private static final int PROCEDURE_EXECUTOR_SLOTS = 2; + + private static TestProcEnv procEnv; + private static ProcedureExecutor procExecutor; + private static ProcedureStore procStore; + + private HBaseCommonTestingUtility htu; + private FileSystem fs; + private Path logDir; + + @Before + public void setUp() throws IOException { + htu = new HBaseCommonTestingUtility(); + Path testDir = htu.getDataTestDir(); + fs = testDir.getFileSystem(htu.getConfiguration()); + assertTrue(testDir.depth() > 1); + + logDir = new Path(testDir, "proc-logs"); + procEnv = new TestProcEnv(); + procStore = ProcedureTestingUtility.createStore(htu.getConfiguration(), fs, logDir); + procExecutor = new ProcedureExecutor(htu.getConfiguration(), procEnv, procStore); + procExecutor.testing = new ProcedureExecutor.Testing(); + procStore.start(PROCEDURE_EXECUTOR_SLOTS); + procExecutor.start(PROCEDURE_EXECUTOR_SLOTS, true); + } + + @After + public void tearDown() throws IOException { + procExecutor.stop(); + procStore.stop(false); + fs.delete(logDir, true); + } + + @Test(timeout=30000) + public void testCompletedProcWithSameNonce() throws Exception { + final long nonceGroup = 123; + final long nonce = 2222; + + // register the nonce + final NonceKey nonceKey = procExecutor.createNonceKey(nonceGroup, nonce); + assertFalse(procExecutor.registerNonce(nonceKey) >= 0); + + // Submit a proc and wait for its completion + Procedure proc = new TestSingleStepProcedure(); + long procId = procExecutor.submitProcedure(proc, nonceKey); + ProcedureTestingUtility.waitProcedure(procExecutor, procId); + + // Restart + ProcedureTestingUtility.restart(procExecutor); + ProcedureTestingUtility.waitProcedure(procExecutor, procId); + + // try to register a procedure with the same nonce + // we should get back the old procId + assertEquals(procId, procExecutor.registerNonce(nonceKey)); + + ProcedureInfo result = procExecutor.getResult(procId); + ProcedureTestingUtility.assertProcNotFailed(result); + } + + @Test(timeout=30000) + public void testRunningProcWithSameNonce() throws Exception { + final long nonceGroup = 456; + final long nonce = 33333; + + // register the nonce + final NonceKey nonceKey = procExecutor.createNonceKey(nonceGroup, nonce); + assertFalse(procExecutor.registerNonce(nonceKey) >= 0); + + // Submit a proc and use a latch to prevent the step execution until we submitted proc2 + CountDownLatch latch = new CountDownLatch(1); + TestSingleStepProcedure proc = new TestSingleStepProcedure(); + procEnv.setWaitLatch(latch); + long procId = procExecutor.submitProcedure(proc, nonceKey); + while (proc.step != 1) Threads.sleep(25); + + // try to register a procedure with the same nonce + // we should get back the old procId + assertEquals(procId, procExecutor.registerNonce(nonceKey)); + + // complete the procedure + latch.countDown(); + + // Restart, the procedure is not completed yet + ProcedureTestingUtility.restart(procExecutor); + ProcedureTestingUtility.waitProcedure(procExecutor, procId); + + // try to register a procedure with the same nonce + // we should get back the old procId + assertEquals(procId, procExecutor.registerNonce(nonceKey)); + + ProcedureInfo result = procExecutor.getResult(procId); + ProcedureTestingUtility.assertProcNotFailed(result); + } + + @Test + public void testSetFailureResultForNonce() throws IOException { + final long nonceGroup = 234; + final long nonce = 55555; + + // check and register the request nonce + final NonceKey nonceKey = procExecutor.createNonceKey(nonceGroup, nonce); + assertFalse(procExecutor.registerNonce(nonceKey) >= 0); + + procExecutor.setFailureResultForNonce(nonceKey, "testProc", User.getCurrent(), + new IOException("test failure")); + + final long procId = procExecutor.registerNonce(nonceKey); + ProcedureInfo result = procExecutor.getResult(procId); + ProcedureTestingUtility.assertProcFailed(result); + } + + @Test(timeout=30000) + public void testConcurrentNonceRegistration() throws IOException { + testConcurrentNonceRegistration(true, 567, 44444); + } + + @Test(timeout=30000) + public void testConcurrentNonceRegistrationWithRollback() throws IOException { + testConcurrentNonceRegistration(false, 890, 55555); + } + + private void testConcurrentNonceRegistration(final boolean submitProcedure, + final long nonceGroup, final long nonce) throws IOException { + // register the nonce + final NonceKey nonceKey = procExecutor.createNonceKey(nonceGroup, nonce); + + final AtomicReference t1Exception = new AtomicReference(); + final AtomicReference t2Exception = new AtomicReference(); + + final CountDownLatch t1NonceRegisteredLatch = new CountDownLatch(1); + final CountDownLatch t2BeforeNonceRegisteredLatch = new CountDownLatch(1); + final Thread[] threads = new Thread[2]; + threads[0] = new Thread() { + @Override + public void run() { + try { + // release the nonce and wake t2 + assertFalse("unexpected already registered nonce", + procExecutor.registerNonce(nonceKey) >= 0); + t1NonceRegisteredLatch.countDown(); + + // hold the submission until t2 is registering the nonce + t2BeforeNonceRegisteredLatch.await(); + Threads.sleep(1000); + + if (submitProcedure) { + CountDownLatch latch = new CountDownLatch(1); + TestSingleStepProcedure proc = new TestSingleStepProcedure(); + procEnv.setWaitLatch(latch); + + procExecutor.submitProcedure(proc, nonceKey); + Threads.sleep(100); + + // complete the procedure + latch.countDown(); + } else { + procExecutor.unregisterNonceIfProcedureWasNotSubmitted(nonceKey); + } + } catch (Throwable e) { + t1Exception.set(e); + } finally { + t1NonceRegisteredLatch.countDown(); + t2BeforeNonceRegisteredLatch.countDown(); + } + } + }; + + threads[1] = new Thread() { + @Override + public void run() { + try { + // wait until t1 has registered the nonce + t1NonceRegisteredLatch.await(); + + // register the nonce + t2BeforeNonceRegisteredLatch.countDown(); + assertFalse("unexpected non registered nonce", + procExecutor.registerNonce(nonceKey) < 0); + } catch (Throwable e) { + t2Exception.set(e); + } finally { + t1NonceRegisteredLatch.countDown(); + t2BeforeNonceRegisteredLatch.countDown(); + } + } + }; + + for (int i = 0; i < threads.length; ++i) threads[i].start(); + for (int i = 0; i < threads.length; ++i) Threads.shutdown(threads[i]); + ProcedureTestingUtility.waitNoProcedureRunning(procExecutor); + assertEquals(null, t1Exception.get()); + assertEquals(null, t2Exception.get()); + } + + public static class TestSingleStepProcedure extends SequentialProcedure { + private int step = 0; + + public TestSingleStepProcedure() { } + + @Override + protected Procedure[] execute(TestProcEnv env) throws InterruptedException { + step++; + env.waitOnLatch(); + LOG.debug("execute procedure " + this + " step=" + step); + step++; + setResult(Bytes.toBytes(step)); + return null; + } + + @Override + protected void rollback(TestProcEnv env) { } + + @Override + protected boolean abort(TestProcEnv env) { return true; } + } + + private static class TestProcEnv { + private CountDownLatch latch = null; + + /** + * set/unset a latch. every procedure execute() step will wait on the latch if any. + */ + public void setWaitLatch(CountDownLatch latch) { + this.latch = latch; + } + + public void waitOnLatch() throws InterruptedException { + if (latch != null) { + latch.await(); + } + } + } +} \ No newline at end of file diff --git a/hbase-procedure/src/test/java/org/apache/hadoop/hbase/procedure2/TestProcedureRecovery.java b/hbase-procedure/src/test/java/org/apache/hadoop/hbase/procedure2/TestProcedureRecovery.java index 98aedcef637..fa9ddfa6782 100644 --- a/hbase-procedure/src/test/java/org/apache/hadoop/hbase/procedure2/TestProcedureRecovery.java +++ b/hbase-procedure/src/test/java/org/apache/hadoop/hbase/procedure2/TestProcedureRecovery.java @@ -287,49 +287,6 @@ public class TestProcedureRecovery { ProcedureTestingUtility.assertIsAbortException(result); } - @Test(timeout=30000) - public void testCompletedProcWithSameNonce() throws Exception { - final long nonceGroup = 123; - final long nonce = 2222; - Procedure proc = new TestSingleStepProcedure(); - // Submit a proc and wait for its completion - long procId = ProcedureTestingUtility.submitAndWait(procExecutor, proc, nonceGroup, nonce); - - // Restart - restart(); - waitProcedure(procId); - - Procedure proc2 = new TestSingleStepProcedure(); - // Submit a procedure with the same nonce and expect the same procedure would return. - long procId2 = ProcedureTestingUtility.submitAndWait(procExecutor, proc2, nonceGroup, nonce); - assertTrue(procId == procId2); - - ProcedureInfo result = procExecutor.getResult(procId2); - ProcedureTestingUtility.assertProcNotFailed(result); - } - - @Test(timeout=30000) - public void testRunningProcWithSameNonce() throws Exception { - final long nonceGroup = 456; - final long nonce = 33333; - Procedure proc = new TestSingleStepProcedure(); - long procId = ProcedureTestingUtility.submitAndWait(procExecutor, proc, nonceGroup, nonce); - - // Restart (use a latch to prevent the step execution until we submitted proc2) - CountDownLatch latch = new CountDownLatch(1); - procEnv.setWaitLatch(latch); - restart(); - // Submit a procedure with the same nonce and expect the same procedure would return. - Procedure proc2 = new TestSingleStepProcedure(); - long procId2 = procExecutor.submitProcedure(proc2, nonceGroup, nonce); - latch.countDown(); - procEnv.setWaitLatch(null); - - // The original proc is not completed and the new submission should have the same proc Id. - assertTrue(procId == procId2); - } - - public static class TestStateMachineProcedure extends StateMachineProcedure { enum State { STATE_1, STATE_2, STATE_3, DONE } 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 f50764253b0..3a6ae593e75 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 @@ -118,6 +118,7 @@ import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv; import org.apache.hadoop.hbase.master.procedure.MasterProcedureScheduler.ProcedureEvent; import org.apache.hadoop.hbase.master.procedure.ModifyColumnFamilyProcedure; import org.apache.hadoop.hbase.master.procedure.ModifyNamespaceProcedure; +import org.apache.hadoop.hbase.master.procedure.MasterProcedureUtil; import org.apache.hadoop.hbase.master.procedure.ModifyTableProcedure; import org.apache.hadoop.hbase.master.procedure.ProcedurePrepareLatch; import org.apache.hadoop.hbase.master.procedure.ProcedureSyncWait; @@ -1658,30 +1659,34 @@ public class HMaster extends HRegionServer implements MasterServices, Server { String namespace = hTableDescriptor.getTableName().getNamespaceAsString(); ensureNamespaceExists(namespace); - HRegionInfo[] newRegions = ModifyRegionUtils.createHRegionInfos(hTableDescriptor, splitKeys); + final HRegionInfo[] newRegions = + ModifyRegionUtils.createHRegionInfos(hTableDescriptor, splitKeys); checkInitialized(); sanityCheckTableDescriptor(hTableDescriptor); - if (cpHost != null) { - cpHost.preCreateTable(hTableDescriptor, newRegions); - } - LOG.info(getClientIdAuditPrefix() + " create " + hTableDescriptor); + return MasterProcedureUtil.submitProcedure( + new MasterProcedureUtil.NonceProcedureRunnable(this, nonceGroup, nonce) { + @Override + protected void run() throws IOException { + getMaster().getMasterCoprocessorHost().preCreateTable(hTableDescriptor, newRegions); - // 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(); - long procId = this.procedureExecutor.submitProcedure( - new CreateTableProcedure( - procedureExecutor.getEnvironment(), hTableDescriptor, newRegions, latch), - nonceGroup, - nonce); - latch.await(); + LOG.info(getClientIdAuditPrefix() + " create " + hTableDescriptor); - if (cpHost != null) { - cpHost.postCreateTable(hTableDescriptor, newRegions); - } + // 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(); + submitProcedure(new CreateTableProcedure( + procedureExecutor.getEnvironment(), hTableDescriptor, newRegions, latch)); + latch.await(); - return procId; + getMaster().getMasterCoprocessorHost().postCreateTable(hTableDescriptor, newRegions); + } + + @Override + protected String getDescription() { + return "CreateTableProcedure"; + } + }); } @Override @@ -2020,24 +2025,29 @@ public class HMaster extends HRegionServer implements MasterServices, Server { final long nonceGroup, final long nonce) throws IOException { checkInitialized(); - if (cpHost != null) { - cpHost.preDeleteTable(tableName); - } - LOG.info(getClientIdAuditPrefix() + " delete " + tableName); - // TODO: We can handle/merge duplicate request - ProcedurePrepareLatch latch = ProcedurePrepareLatch.createLatch(); - long procId = this.procedureExecutor.submitProcedure( - new DeleteTableProcedure(procedureExecutor.getEnvironment(), tableName, latch), - nonceGroup, - nonce); - latch.await(); + return MasterProcedureUtil.submitProcedure( + new MasterProcedureUtil.NonceProcedureRunnable(this, nonceGroup, nonce) { + @Override + protected void run() throws IOException { + getMaster().getMasterCoprocessorHost().preDeleteTable(tableName); - if (cpHost != null) { - cpHost.postDeleteTable(tableName); - } + LOG.info(getClientIdAuditPrefix() + " delete " + tableName); - return procId; + // TODO: We can handle/merge duplicate request + ProcedurePrepareLatch latch = ProcedurePrepareLatch.createLatch(); + submitProcedure(new DeleteTableProcedure(procedureExecutor.getEnvironment(), + tableName, latch)); + latch.await(); + + getMaster().getMasterCoprocessorHost().postDeleteTable(tableName); + } + + @Override + protected String getDescription() { + return "DeleteTableProcedure"; + } + }); } @Override @@ -2047,20 +2057,27 @@ public class HMaster extends HRegionServer implements MasterServices, Server { final long nonceGroup, final long nonce) throws IOException { checkInitialized(); - if (cpHost != null) { - cpHost.preTruncateTable(tableName); - } - LOG.info(getClientIdAuditPrefix() + " truncate " + tableName); - long procId = this.procedureExecutor.submitProcedure( - new TruncateTableProcedure(procedureExecutor.getEnvironment(), tableName, preserveSplits), - nonceGroup, - nonce); - ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); + MasterProcedureUtil.submitProcedure( + new MasterProcedureUtil.NonceProcedureRunnable(this, nonceGroup, nonce) { + @Override + protected void run() throws IOException { + getMaster().getMasterCoprocessorHost().preTruncateTable(tableName); - if (cpHost != null) { - cpHost.postTruncateTable(tableName); - } + LOG.info(getClientIdAuditPrefix() + " truncate " + tableName); + + long procId = submitProcedure(new TruncateTableProcedure(procedureExecutor.getEnvironment(), + tableName, preserveSplits)); + ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); + + getMaster().getMasterCoprocessorHost().postTruncateTable(tableName); + } + + @Override + protected String getDescription() { + return "TruncateTableProcedure"; + } + }); } @Override @@ -2074,20 +2091,28 @@ public class HMaster extends HRegionServer implements MasterServices, Server { checkCompression(columnDescriptor); checkEncryption(conf, columnDescriptor); checkReplicationScope(columnDescriptor); - if (cpHost != null) { - if (cpHost.preAddColumn(tableName, columnDescriptor)) { - return; + + MasterProcedureUtil.submitProcedure( + new MasterProcedureUtil.NonceProcedureRunnable(this, nonceGroup, nonce) { + @Override + protected void run() throws IOException { + if (getMaster().getMasterCoprocessorHost().preAddColumn(tableName, columnDescriptor)) { + return; + } + + // Execute the operation synchronously, wait for the operation to complete before continuing + long procId = submitProcedure(new AddColumnFamilyProcedure( + procedureExecutor.getEnvironment(), tableName, columnDescriptor)); + ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); + + getMaster().getMasterCoprocessorHost().postAddColumn(tableName, columnDescriptor); } - } - // Execute the operation synchronously - wait for the operation to complete before continuing. - long procId = this.procedureExecutor.submitProcedure( - new AddColumnFamilyProcedure(procedureExecutor.getEnvironment(), tableName, columnDescriptor), - nonceGroup, - nonce); - ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); - if (cpHost != null) { - cpHost.postAddColumn(tableName, columnDescriptor); - } + + @Override + protected String getDescription() { + return "AddColumnFamilyProcedure"; + } + }); } @Override @@ -2101,23 +2126,30 @@ public class HMaster extends HRegionServer implements MasterServices, Server { checkCompression(descriptor); checkEncryption(conf, descriptor); checkReplicationScope(descriptor); - if (cpHost != null) { - if (cpHost.preModifyColumn(tableName, descriptor)) { - return; + + MasterProcedureUtil.submitProcedure( + new MasterProcedureUtil.NonceProcedureRunnable(this, nonceGroup, nonce) { + @Override + protected void run() throws IOException { + if (getMaster().getMasterCoprocessorHost().preModifyColumn(tableName, descriptor)) { + return; + } + + LOG.info(getClientIdAuditPrefix() + " modify " + descriptor); + + // Execute the operation synchronously - wait for the operation to complete before continuing. + long procId = submitProcedure(new ModifyColumnFamilyProcedure( + procedureExecutor.getEnvironment(), tableName, descriptor)); + ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); + + getMaster().getMasterCoprocessorHost().postModifyColumn(tableName, descriptor); } - } - LOG.info(getClientIdAuditPrefix() + " modify " + descriptor); - // Execute the operation synchronously - wait for the operation to complete before continuing. - long procId = this.procedureExecutor.submitProcedure( - new ModifyColumnFamilyProcedure(procedureExecutor.getEnvironment(), tableName, descriptor), - nonceGroup, - nonce); - ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); - - if (cpHost != null) { - cpHost.postModifyColumn(tableName, descriptor); - } + @Override + protected String getDescription() { + return "ModifyColumnFamilyProcedure"; + } + }); } @Override @@ -2128,84 +2160,97 @@ public class HMaster extends HRegionServer implements MasterServices, Server { final long nonce) throws IOException { checkInitialized(); - if (cpHost != null) { - if (cpHost.preDeleteColumn(tableName, columnName)) { - return; + + MasterProcedureUtil.submitProcedure( + new MasterProcedureUtil.NonceProcedureRunnable(this, nonceGroup, nonce) { + @Override + protected void run() throws IOException { + if (getMaster().getMasterCoprocessorHost().preDeleteColumn(tableName, columnName)) { + return; + } + + LOG.info(getClientIdAuditPrefix() + " delete " + Bytes.toString(columnName)); + + // Execute the operation synchronously - wait for the operation to complete before + // continuing. + long procId = submitProcedure(new DeleteColumnFamilyProcedure( + procedureExecutor.getEnvironment(), tableName, columnName)); + ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); + + getMaster().getMasterCoprocessorHost().postDeleteColumn(tableName, columnName); } - } - LOG.info(getClientIdAuditPrefix() + " delete " + Bytes.toString(columnName)); - // Execute the operation synchronously - wait for the operation to complete before continuing. - long procId = this.procedureExecutor.submitProcedure( - new DeleteColumnFamilyProcedure(procedureExecutor.getEnvironment(), tableName, columnName), - nonceGroup, - nonce); - ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); - - if (cpHost != null) { - cpHost.postDeleteColumn(tableName, columnName); - } + @Override + protected String getDescription() { + return "DeleteColumnFamilyProcedure"; + } + }); } @Override - public long enableTable( - final TableName tableName, - final long nonceGroup, - final long nonce) throws IOException { + public long enableTable(final TableName tableName, final long nonceGroup, final long nonce) + throws IOException { checkInitialized(); - if (cpHost != null) { - cpHost.preEnableTable(tableName); - } - LOG.info(getClientIdAuditPrefix() + " enable " + tableName); - // Execute the operation asynchronously - client will check the progress of the operation - final ProcedurePrepareLatch prepareLatch = ProcedurePrepareLatch.createLatch(); - long procId = this.procedureExecutor.submitProcedure( - new EnableTableProcedure(procedureExecutor.getEnvironment(), tableName, false, prepareLatch), - nonceGroup, - nonce); - // Before returning to client, 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. - prepareLatch.await(); + return MasterProcedureUtil.submitProcedure( + new MasterProcedureUtil.NonceProcedureRunnable(this, nonceGroup, nonce) { + @Override + protected void run() throws IOException { + getMaster().getMasterCoprocessorHost().preEnableTable(tableName); - if (cpHost != null) { - cpHost.postEnableTable(tableName); - } + LOG.info(getClientIdAuditPrefix() + " enable " + tableName); - return procId; + // Execute the operation asynchronously - client will check the progress of the operation + // In case the request is from a <1.1 client before returning, + // 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(); + submitProcedure(new EnableTableProcedure(procedureExecutor.getEnvironment(), + tableName, false, prepareLatch)); + prepareLatch.await(); + + getMaster().getMasterCoprocessorHost().postEnableTable(tableName); + } + + @Override + protected String getDescription() { + return "EnableTableProcedure"; + } + }); } @Override - public long disableTable( - final TableName tableName, - final long nonceGroup, - final long nonce) throws IOException { + public long disableTable(final TableName tableName, final long nonceGroup, final long nonce) + throws IOException { checkInitialized(); - if (cpHost != null) { - cpHost.preDisableTable(tableName); - } - LOG.info(getClientIdAuditPrefix() + " disable " + tableName); - // Execute the operation asynchronously - client will check the progress of the operation - final ProcedurePrepareLatch prepareLatch = ProcedurePrepareLatch.createLatch(); - // Execute the operation asynchronously - client will check the progress of the operation - long procId = this.procedureExecutor.submitProcedure( - new DisableTableProcedure(procedureExecutor.getEnvironment(), tableName, false, prepareLatch), - nonceGroup, - nonce); - // Before returning to client, 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. - prepareLatch.await(); + return MasterProcedureUtil.submitProcedure( + new MasterProcedureUtil.NonceProcedureRunnable(this, nonceGroup, nonce) { + @Override + protected void run() throws IOException { + getMaster().getMasterCoprocessorHost().preDisableTable(tableName); - if (cpHost != null) { - cpHost.postDisableTable(tableName); - } + LOG.info(getClientIdAuditPrefix() + " disable " + tableName); - return procId; + // Execute the operation asynchronously - client will check the progress of the operation + // In case the request is from a <1.1 client before returning, + // 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(); + submitProcedure(new DisableTableProcedure(procedureExecutor.getEnvironment(), + tableName, false, prepareLatch)); + prepareLatch.await(); + + getMaster().getMasterCoprocessorHost().postDisableTable(tableName); + } + + @Override + protected String getDescription() { + return "DisableTableProcedure"; + } + }); } /** @@ -2253,23 +2298,28 @@ public class HMaster extends HRegionServer implements MasterServices, Server { throws IOException { checkInitialized(); sanityCheckTableDescriptor(descriptor); - if (cpHost != null) { - cpHost.preModifyTable(tableName, descriptor); - } - LOG.info(getClientIdAuditPrefix() + " modify " + tableName); + MasterProcedureUtil.submitProcedure( + new MasterProcedureUtil.NonceProcedureRunnable(this, nonceGroup, nonce) { + @Override + protected void run() throws IOException { + getMaster().getMasterCoprocessorHost().preModifyTable(tableName, descriptor); - // Execute the operation synchronously - wait for the operation completes before continuing. - long procId = this.procedureExecutor.submitProcedure( - new ModifyTableProcedure(procedureExecutor.getEnvironment(), descriptor), - nonceGroup, - nonce); + LOG.info(getClientIdAuditPrefix() + " modify " + tableName); - ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); + // Execute the operation synchronously - wait for the operation completes before continuing. + long procId = submitProcedure(new ModifyTableProcedure( + procedureExecutor.getEnvironment(), descriptor)); + ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); - if (cpHost != null) { - cpHost.postModifyTable(tableName, descriptor); - } + getMaster().getMasterCoprocessorHost().postModifyTable(tableName, descriptor); + } + + @Override + protected String getDescription() { + return "ModifyTableProcedure"; + } + }); } @Override @@ -2684,29 +2734,40 @@ public class HMaster extends HRegionServer implements MasterServices, Server { final long nonce) throws IOException { TableName.isLegalNamespaceName(Bytes.toBytes(descriptor.getName())); checkNamespaceManagerReady(); - if (cpHost != null) { - if (cpHost.preCreateNamespace(descriptor)) { - return; - } - } - createNamespaceSync(descriptor, nonceGroup, nonce); - if (cpHost != null) { - cpHost.postCreateNamespace(descriptor); - } + + createNamespaceSync(descriptor, nonceGroup, nonce, true); } @Override public void createNamespaceSync( final NamespaceDescriptor descriptor, final long nonceGroup, - final long nonce) throws IOException { - LOG.info(getClientIdAuditPrefix() + " creating " + descriptor); - // Execute the operation synchronously - wait for the operation to complete before continuing. - long procId = this.procedureExecutor.submitProcedure( - new CreateNamespaceProcedure(procedureExecutor.getEnvironment(), descriptor), - nonceGroup, - nonce); - ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); + final long nonce, + final boolean executeCoprocessor) throws IOException { + MasterProcedureUtil.submitProcedure( + new MasterProcedureUtil.NonceProcedureRunnable(this, nonceGroup, nonce) { + @Override + protected void run() throws IOException { + if (executeCoprocessor && + getMaster().getMasterCoprocessorHost().preCreateNamespace(descriptor)) { + return; + } + LOG.info(getClientIdAuditPrefix() + " creating " + descriptor); + // Execute the operation synchronously - wait for the operation to complete before + // continuing. + long procId = submitProcedure(new CreateNamespaceProcedure( + procedureExecutor.getEnvironment(), descriptor)); + ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); + if (executeCoprocessor) { + getMaster().getMasterCoprocessorHost().postCreateNamespace(descriptor); + } + } + + @Override + protected String getDescription() { + return "CreateNamespaceProcedure"; + } + }); } @Override @@ -2716,21 +2777,28 @@ public class HMaster extends HRegionServer implements MasterServices, Server { final long nonce) throws IOException { TableName.isLegalNamespaceName(Bytes.toBytes(descriptor.getName())); checkNamespaceManagerReady(); - if (cpHost != null) { - if (cpHost.preModifyNamespace(descriptor)) { - return; - } + + MasterProcedureUtil.submitProcedure( + new MasterProcedureUtil.NonceProcedureRunnable(this, nonceGroup, nonce) { + @Override + protected void run() throws IOException { + if (getMaster().getMasterCoprocessorHost().preModifyNamespace(descriptor)) { + return; + } + LOG.info(getClientIdAuditPrefix() + " modify " + descriptor); + // Execute the operation synchronously - wait for the operation to complete before + // continuing. + long procId = submitProcedure(new ModifyNamespaceProcedure( + procedureExecutor.getEnvironment(), descriptor)); + ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); + getMaster().getMasterCoprocessorHost().postModifyNamespace(descriptor); } - LOG.info(getClientIdAuditPrefix() + " modify " + descriptor); - // Execute the operation synchronously - wait for the operation to complete before continuing. - long procId = this.procedureExecutor.submitProcedure( - new ModifyNamespaceProcedure(procedureExecutor.getEnvironment(), descriptor), - nonceGroup, - nonce); - ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); - if (cpHost != null) { - cpHost.postModifyNamespace(descriptor); + + @Override + protected String getDescription() { + return "ModifyNamespace"; } + }); } @Override @@ -2739,21 +2807,28 @@ public class HMaster extends HRegionServer implements MasterServices, Server { final long nonceGroup, final long nonce) throws IOException { checkNamespaceManagerReady(); - if (cpHost != null) { - if (cpHost.preDeleteNamespace(name)) { - return; + + MasterProcedureUtil.submitProcedure( + new MasterProcedureUtil.NonceProcedureRunnable(this, nonceGroup, nonce) { + @Override + protected void run() throws IOException { + if (getMaster().getMasterCoprocessorHost().preDeleteNamespace(name)) { + return; + } + LOG.info(getClientIdAuditPrefix() + " delete " + name); + // Execute the operation synchronously - wait for the operation to complete before + // continuing. + long procId = submitProcedure(new DeleteNamespaceProcedure( + procedureExecutor.getEnvironment(), name)); + ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); + getMaster().getMasterCoprocessorHost().postDeleteNamespace(name); } - } - LOG.info(getClientIdAuditPrefix() + " delete " + name); - // Execute the operation synchronously - wait for the operation to complete before continuing. - long procId = this.procedureExecutor.submitProcedure( - new DeleteNamespaceProcedure(procedureExecutor.getEnvironment(), name), - nonceGroup, - nonce); - ProcedureSyncWait.waitForProcedureToComplete(procedureExecutor, procId); - if (cpHost != null) { - cpHost.postDeleteNamespace(name); - } + + @Override + protected String getDescription() { + return "DeleteNamespaceProcedure"; + } + }); } /** diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterServices.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterServices.java index 015e201a1c0..3fab2cc0477 100644 --- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterServices.java +++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/MasterServices.java @@ -311,7 +311,8 @@ public interface MasterServices extends Server { public void createNamespaceSync( final NamespaceDescriptor descriptor, final long nonceGroup, - final long nonce) throws IOException; + final long nonce, + final boolean executeCoprocessor) throws IOException; /** * Modify an existing namespace 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 afc8a218b02..6bcfb7719bf 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 @@ -242,7 +242,8 @@ public class TableNamespaceManager { masterServices.createNamespaceSync( NamespaceDescriptor.DEFAULT_NAMESPACE, HConstants.NO_NONCE, - HConstants.NO_NONCE); + HConstants.NO_NONCE, + false); } } if (get(nsTable, NamespaceDescriptor.SYSTEM_NAMESPACE.getName()) == null) { @@ -257,7 +258,8 @@ public class TableNamespaceManager { masterServices.createNamespaceSync( NamespaceDescriptor.SYSTEM_NAMESPACE, HConstants.NO_NONCE, - HConstants.NO_NONCE); + HConstants.NO_NONCE, + false); } } diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/MasterProcedureUtil.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/MasterProcedureUtil.java index d7c0b922fad..4759e7ddd4e 100644 --- a/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/MasterProcedureUtil.java +++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/master/procedure/MasterProcedureUtil.java @@ -18,11 +18,18 @@ package org.apache.hadoop.hbase.master.procedure; +import java.io.IOException; + import org.apache.commons.logging.Log; import org.apache.commons.logging.LogFactory; import org.apache.hadoop.hbase.classification.InterfaceAudience; import org.apache.hadoop.hbase.classification.InterfaceStability; +import org.apache.hadoop.hbase.master.MasterServices; +import org.apache.hadoop.hbase.procedure2.Procedure; +import org.apache.hadoop.hbase.procedure2.ProcedureExecutor; import org.apache.hadoop.hbase.protobuf.generated.RPCProtos.UserInformation; +import org.apache.hadoop.hbase.security.User; +import org.apache.hadoop.hbase.util.NonceKey; import org.apache.hadoop.security.UserGroupInformation; @InterfaceAudience.Private @@ -53,4 +60,85 @@ public final class MasterProcedureUtil { } return null; } + + /** + * Helper Runnable used in conjunction with submitProcedure() to deal with + * submitting procs with nonce. + * See submitProcedure() for an example. + */ + public static abstract class NonceProcedureRunnable { + private final MasterServices master; + private final NonceKey nonceKey; + private Long procId; + + public NonceProcedureRunnable(final MasterServices master, + final long nonceGroup, final long nonce) { + this.master = master; + this.nonceKey = getProcedureExecutor().createNonceKey(nonceGroup, nonce); + } + + protected NonceKey getNonceKey() { + return nonceKey; + } + + protected MasterServices getMaster() { + return master; + } + + protected ProcedureExecutor getProcedureExecutor() { + return master.getMasterProcedureExecutor(); + } + + protected long getProcId() { + return procId != null ? procId.longValue() : -1; + } + + protected long setProcId(final long procId) { + this.procId = procId; + return procId; + } + + protected abstract void run() throws IOException; + protected abstract String getDescription(); + + protected long submitProcedure(final Procedure proc) { + assert procId == null : "submitProcedure() was already called, running procId=" + procId; + procId = getProcedureExecutor().submitProcedure(proc, nonceKey); + return procId; + } + } + + /** + * Helper used to deal with submitting procs with nonce. + * Internally the NonceProcedureRunnable.run() will be called only if no one else + * registered the nonce. any Exception thrown by the run() method will be + * collected/handled and rethrown. + * + * long procId = MasterProcedureUtil.submitProcedure( + * new NonceProcedureRunnable(procExec, nonceGroup, nonce) { + * {@literal @}Override + * public void run() { + * cpHost.preOperation(); + * submitProcedure(new MyProc()); + * cpHost.postOperation(); + * } + * }); + * + */ + public static long submitProcedure(final NonceProcedureRunnable runnable) throws IOException { + final ProcedureExecutor procExec = runnable.getProcedureExecutor(); + final long procId = procExec.registerNonce(runnable.getNonceKey()); + if (procId >= 0) return procId; // someone already registered the nonce + try { + runnable.run(); + } catch (IOException e) { + procExec.setFailureResultForNonce(runnable.getNonceKey(), + runnable.getDescription(), + procExec.getEnvironment().getRequestUser(), e); + throw e; + } finally { + procExec.unregisterNonceIfProcedureWasNotSubmitted(runnable.getNonceKey()); + } + return runnable.getProcId(); + } } diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/MockNoopMasterServices.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/MockNoopMasterServices.java index 657d8e2c5ba..a172de5cb02 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/MockNoopMasterServices.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/MockNoopMasterServices.java @@ -176,7 +176,11 @@ public class MockNoopMasterServices implements MasterServices, Server { } @Override - public void createNamespaceSync(NamespaceDescriptor descriptor, long nonceGroup, long nonce) throws IOException { + public void createNamespaceSync( + final NamespaceDescriptor descriptor, + final long nonceGroup, + final long nonce, + final boolean executeCoprocessor) throws IOException { } diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestCatalogJanitor.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestCatalogJanitor.java index 13c0cb1677d..01a2124a9ef 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestCatalogJanitor.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/TestCatalogJanitor.java @@ -409,7 +409,8 @@ public class TestCatalogJanitor { public void createNamespaceSync( final NamespaceDescriptor descriptor, final long nonceGroup, - final long nonce) throws IOException { + final long nonce, + final boolean executeCoprocessor) throws IOException { //To change body of implemented methods use File | Settings | File Templates. } diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestAddColumnFamilyProcedure.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestAddColumnFamilyProcedure.java index 97a287ef71f..b3fa10a0876 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestAddColumnFamilyProcedure.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestAddColumnFamilyProcedure.java @@ -25,7 +25,6 @@ import org.apache.commons.logging.LogFactory; import org.apache.hadoop.conf.Configuration; import org.apache.hadoop.hbase.HBaseTestingUtility; import org.apache.hadoop.hbase.HColumnDescriptor; -import org.apache.hadoop.hbase.HConstants; import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.InvalidFamilyOperationException; import org.apache.hadoop.hbase.ProcedureInfo; @@ -47,9 +46,6 @@ public class TestAddColumnFamilyProcedure { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private static long nonceGroup = HConstants.NO_NONCE; - private static long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1); } @@ -72,9 +68,6 @@ public class TestAddColumnFamilyProcedure { @Before public void setup() throws Exception { ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(getMasterProcedureExecutor(), false); - nonceGroup = - MasterProcedureTestingUtility.generateNonceGroup(UTIL.getHBaseCluster().getMaster()); - nonce = MasterProcedureTestingUtility.generateNonce(UTIL.getHBaseCluster().getMaster()); } @After @@ -99,9 +92,7 @@ public class TestAddColumnFamilyProcedure { // Test 1: Add a column family online long procId1 = procExec.submitProcedure( - new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor1), - nonceGroup, - nonce); + new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor1)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId1); ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); @@ -112,9 +103,7 @@ public class TestAddColumnFamilyProcedure { // Test 2: Add a column family offline UTIL.getHBaseAdmin().disableTable(tableName); long procId2 = procExec.submitProcedure( - new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor2), - nonceGroup + 1, - nonce + 1); + new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor2)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId2); ProcedureTestingUtility.assertProcNotFailed(procExec, procId2); @@ -134,9 +123,7 @@ public class TestAddColumnFamilyProcedure { // add the column family long procId1 = procExec.submitProcedure( - new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup, - nonce); + new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId1); ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); @@ -145,9 +132,7 @@ public class TestAddColumnFamilyProcedure { // add the column family that exists long procId2 = procExec.submitProcedure( - new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup + 1, - nonce + 1); + new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId2); @@ -161,9 +146,7 @@ public class TestAddColumnFamilyProcedure { // Do the same add the existing column family - this time offline UTIL.getHBaseAdmin().disableTable(tableName); long procId3 = procExec.submitProcedure( - new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup + 2, - nonce + 2); + new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId3); @@ -175,37 +158,6 @@ public class TestAddColumnFamilyProcedure { ProcedureTestingUtility.getExceptionCause(result) instanceof InvalidFamilyOperationException); } - @Test(timeout=60000) - public void testAddSameColumnFamilyTwiceWithSameNonce() throws Exception { - final TableName tableName = TableName.valueOf("testAddSameColumnFamilyTwiceWithSameNonce"); - final String cf2 = "cf2"; - final HColumnDescriptor columnDescriptor = new HColumnDescriptor(cf2); - - final ProcedureExecutor procExec = getMasterProcedureExecutor(); - - MasterProcedureTestingUtility.createTable(procExec, tableName, null, "f1"); - - // add the column family - long procId1 = procExec.submitProcedure( - new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup, - nonce); - long procId2 = procExec.submitProcedure( - new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup, - nonce); - // Wait the completion - ProcedureTestingUtility.waitProcedure(procExec, procId1); - ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); - MasterProcedureTestingUtility.validateColumnFamilyAddition(UTIL.getHBaseCluster().getMaster(), - tableName, cf2); - - // Wait the completion and expect not fail - because it is the same proc - ProcedureTestingUtility.waitProcedure(procExec, procId2); - ProcedureTestingUtility.assertProcNotFailed(procExec, procId2); - assertTrue(procId1 == procId2); - } - @Test(timeout = 60000) public void testRecoveryAndDoubleExecutionOffline() throws Exception { final TableName tableName = TableName.valueOf("testRecoveryAndDoubleExecutionOffline"); @@ -221,9 +173,7 @@ public class TestAddColumnFamilyProcedure { // Start the AddColumnFamily procedure && kill the executor long procId = procExec.submitProcedure( - new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup, - nonce); + new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor)); // Restart the executor and execute the step twice int numberOfSteps = AddColumnFamilyState.values().length; @@ -248,9 +198,7 @@ public class TestAddColumnFamilyProcedure { // Start the AddColumnFamily procedure && kill the executor long procId = procExec.submitProcedure( - new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup, - nonce); + new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor)); // Restart the executor and execute the step twice int numberOfSteps = AddColumnFamilyState.values().length; @@ -275,9 +223,7 @@ public class TestAddColumnFamilyProcedure { // Start the AddColumnFamily procedure && kill the executor long procId = procExec.submitProcedure( - new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup, - nonce); + new AddColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor)); int numberOfSteps = AddColumnFamilyState.values().length - 2; // failing in the middle of proc MasterProcedureTestingUtility.testRollbackAndDoubleExecution(procExec, procId, numberOfSteps, diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestCreateNamespaceProcedure.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestCreateNamespaceProcedure.java index e0027ee05c5..e7ddff6a7ac 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestCreateNamespaceProcedure.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestCreateNamespaceProcedure.java @@ -28,7 +28,6 @@ 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.HConstants; import org.apache.hadoop.hbase.NamespaceDescriptor; import org.apache.hadoop.hbase.NamespaceExistException; import org.apache.hadoop.hbase.NamespaceNotFoundException; @@ -51,9 +50,6 @@ public class TestCreateNamespaceProcedure { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private static long nonceGroup = HConstants.NO_NONCE; - private static long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1); } @@ -76,9 +72,6 @@ public class TestCreateNamespaceProcedure { @Before public void setup() throws Exception { ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(getMasterProcedureExecutor(), false); - nonceGroup = - MasterProcedureTestingUtility.generateNonceGroup(UTIL.getHBaseCluster().getMaster()); - nonce = MasterProcedureTestingUtility.generateNonce(UTIL.getHBaseCluster().getMaster()); } @After @@ -92,9 +85,7 @@ public class TestCreateNamespaceProcedure { final ProcedureExecutor procExec = getMasterProcedureExecutor(); long procId = procExec.submitProcedure( - new CreateNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); + new CreateNamespaceProcedure(procExec.getEnvironment(), nsd)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); ProcedureTestingUtility.assertProcNotFailed(procExec, procId); @@ -109,18 +100,14 @@ public class TestCreateNamespaceProcedure { final ProcedureExecutor procExec = getMasterProcedureExecutor(); long procId1 = procExec.submitProcedure( - new CreateNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); + new CreateNamespaceProcedure(procExec.getEnvironment(), nsd)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId1); ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); // Create the namespace that exists long procId2 = procExec.submitProcedure( - new CreateNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup + 1, - nonce + 1); + new CreateNamespaceProcedure(procExec.getEnvironment(), nsd)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId2); @@ -139,9 +126,7 @@ public class TestCreateNamespaceProcedure { final ProcedureExecutor procExec = getMasterProcedureExecutor(); long procId = procExec.submitProcedure( - new CreateNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); + new CreateNamespaceProcedure(procExec.getEnvironment(), nsd)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); ProcedureInfo result = procExec.getResult(procId); @@ -162,9 +147,7 @@ public class TestCreateNamespaceProcedure { nsd.setConfiguration(nsKey, nsValue); long procId = procExec.submitProcedure( - new CreateNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); + new CreateNamespaceProcedure(procExec.getEnvironment(), nsd)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); ProcedureInfo result = procExec.getResult(procId); @@ -184,9 +167,7 @@ public class TestCreateNamespaceProcedure { nsd.setConfiguration(nsKey, nsValue); long procId = procExec.submitProcedure( - new CreateNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); + new CreateNamespaceProcedure(procExec.getEnvironment(), nsd)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); ProcedureInfo result = procExec.getResult(procId); @@ -195,32 +176,6 @@ public class TestCreateNamespaceProcedure { assertTrue(ProcedureTestingUtility.getExceptionCause(result) instanceof ConstraintException); } - @Test(timeout=60000) - public void testCreateSameNamespaceTwiceWithSameNonce() throws Exception { - final NamespaceDescriptor nsd = - NamespaceDescriptor.create("testCreateSameNamespaceTwiceWithSameNonce").build(); - final ProcedureExecutor procExec = getMasterProcedureExecutor(); - - long procId1 = procExec.submitProcedure( - new CreateNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); - long procId2 = procExec.submitProcedure( - new CreateNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); - // Wait the completion - ProcedureTestingUtility.waitProcedure(procExec, procId1); - ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); - - validateNamespaceCreated(nsd); - - // Wait the completion and expect not fail - because it is the same proc - ProcedureTestingUtility.waitProcedure(procExec, procId2); - ProcedureTestingUtility.assertProcNotFailed(procExec, procId2); - assertTrue(procId1 == procId2); - } - @Test(timeout = 60000) public void testRecoveryAndDoubleExecution() throws Exception { final NamespaceDescriptor nsd = @@ -232,9 +187,7 @@ public class TestCreateNamespaceProcedure { // Start the CreateNamespace procedure && kill the executor long procId = procExec.submitProcedure( - new CreateNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); + new CreateNamespaceProcedure(procExec.getEnvironment(), nsd)); // Restart the executor and execute the step twice int numberOfSteps = CreateNamespaceState.values().length; @@ -260,9 +213,7 @@ public class TestCreateNamespaceProcedure { // Start the CreateNamespace procedure && kill the executor long procId = procExec.submitProcedure( - new CreateNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); + new CreateNamespaceProcedure(procExec.getEnvironment(), nsd)); int numberOfSteps = CreateNamespaceState.values().length - 2; // failing in the middle of proc MasterProcedureTestingUtility.testRollbackAndDoubleExecution( diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestCreateTableProcedure.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestCreateTableProcedure.java index 28418474139..8860711b456 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestCreateTableProcedure.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestCreateTableProcedure.java @@ -53,9 +53,6 @@ public class TestCreateTableProcedure { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private static long nonceGroup = HConstants.NO_NONCE; - private static long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1); } @@ -78,9 +75,6 @@ public class TestCreateTableProcedure { @Before public void setup() throws Exception { resetProcExecutorTestingKillFlag(); - nonceGroup = - MasterProcedureTestingUtility.generateNonceGroup(UTIL.getHBaseCluster().getMaster()); - nonce = MasterProcedureTestingUtility.generateNonce(UTIL.getHBaseCluster().getMaster()); } @After @@ -151,14 +145,12 @@ public class TestCreateTableProcedure { // create the table long procId1 = procExec.submitProcedure( - new CreateTableProcedure(procExec.getEnvironment(), htd, regions), nonceGroup, nonce); + new CreateTableProcedure(procExec.getEnvironment(), htd, regions)); // create another with the same name ProcedurePrepareLatch latch2 = new ProcedurePrepareLatch.CompatibilityLatch(); long procId2 = procExec.submitProcedure( - new CreateTableProcedure(procExec.getEnvironment(), htd, regions, latch2), - nonceGroup + 1, - nonce + 1); + new CreateTableProcedure(procExec.getEnvironment(), htd, regions, latch2)); ProcedureTestingUtility.waitProcedure(procExec, procId1); ProcedureTestingUtility.assertProcNotFailed(procExec.getResult(procId1)); @@ -167,29 +159,6 @@ public class TestCreateTableProcedure { latch2.await(); } - @Test(timeout=60000) - public void testCreateTwiceWithSameNonce() throws Exception { - final TableName tableName = TableName.valueOf("testCreateTwiceWithSameNonce"); - final ProcedureExecutor procExec = getMasterProcedureExecutor(); - final HTableDescriptor htd = MasterProcedureTestingUtility.createHTD(tableName, "f"); - final HRegionInfo[] regions = ModifyRegionUtils.createHRegionInfos(htd, null); - - // create the table - long procId1 = procExec.submitProcedure( - new CreateTableProcedure(procExec.getEnvironment(), htd, regions), nonceGroup, nonce); - - // create another with the same name - long procId2 = procExec.submitProcedure( - new CreateTableProcedure(procExec.getEnvironment(), htd, regions), nonceGroup, nonce); - - ProcedureTestingUtility.waitProcedure(procExec, procId1); - ProcedureTestingUtility.assertProcNotFailed(procExec.getResult(procId1)); - - ProcedureTestingUtility.waitProcedure(procExec, procId2); - ProcedureTestingUtility.assertProcNotFailed(procExec.getResult(procId2)); - assertTrue(procId1 == procId2); - } - @Test(timeout=60000) public void testRecoveryAndDoubleExecution() throws Exception { final TableName tableName = TableName.valueOf("testRecoveryAndDoubleExecution"); @@ -203,7 +172,7 @@ public class TestCreateTableProcedure { HTableDescriptor htd = MasterProcedureTestingUtility.createHTD(tableName, "f1", "f2"); HRegionInfo[] regions = ModifyRegionUtils.createHRegionInfos(htd, splitKeys); long procId = procExec.submitProcedure( - new CreateTableProcedure(procExec.getEnvironment(), htd, regions), nonceGroup, nonce); + new CreateTableProcedure(procExec.getEnvironment(), htd, regions)); // Restart the executor and execute the step twice // NOTE: the 6 (number of CreateTableState steps) is hardcoded, @@ -231,7 +200,7 @@ public class TestCreateTableProcedure { htd.setRegionReplication(3); HRegionInfo[] regions = ModifyRegionUtils.createHRegionInfos(htd, splitKeys); long procId = procExec.submitProcedure( - new CreateTableProcedure(procExec.getEnvironment(), htd, regions), nonceGroup, nonce); + new CreateTableProcedure(procExec.getEnvironment(), htd, regions)); // NOTE: the 4 (number of CreateTableState steps) is hardcoded, // so you have to look at this test at least once when you add a new step. @@ -261,7 +230,7 @@ public class TestCreateTableProcedure { HTableDescriptor htd = MasterProcedureTestingUtility.createHTD(tableName, "f1", "f2"); HRegionInfo[] regions = ModifyRegionUtils.createHRegionInfos(htd, splitKeys); long procId = procExec.submitProcedure( - new FaultyCreateTableProcedure(procExec.getEnvironment(), htd, regions), nonceGroup, nonce); + new CreateTableProcedure(procExec.getEnvironment(), htd, regions)); // NOTE: the 4 (number of CreateTableState steps) is hardcoded, // so you have to look at this test at least once when you add a new step. diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDeleteColumnFamilyProcedure.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDeleteColumnFamilyProcedure.java index d5e79cf4699..11d6f149596 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDeleteColumnFamilyProcedure.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDeleteColumnFamilyProcedure.java @@ -24,7 +24,6 @@ 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.HConstants; import org.apache.hadoop.hbase.HRegionInfo; import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.InvalidFamilyOperationException; @@ -47,9 +46,6 @@ public class TestDeleteColumnFamilyProcedure { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private static long nonceGroup = HConstants.NO_NONCE; - private static long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1); } @@ -72,9 +68,6 @@ public class TestDeleteColumnFamilyProcedure { @Before public void setup() throws Exception { ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(getMasterProcedureExecutor(), false); - nonceGroup = - MasterProcedureTestingUtility.generateNonceGroup(UTIL.getHBaseCluster().getMaster()); - nonce = MasterProcedureTestingUtility.generateNonce(UTIL.getHBaseCluster().getMaster()); } @After @@ -97,9 +90,7 @@ public class TestDeleteColumnFamilyProcedure { // Test 1: delete the column family that exists online long procId1 = procExec.submitProcedure( - new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf1.getBytes()), - nonceGroup, - nonce); + new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf1.getBytes())); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId1); ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); @@ -110,9 +101,7 @@ public class TestDeleteColumnFamilyProcedure { // Test 2: delete the column family that exists offline UTIL.getHBaseAdmin().disableTable(tableName); long procId2 = procExec.submitProcedure( - new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf2.getBytes()), - nonceGroup, - nonce); + new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf2.getBytes())); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId2); ProcedureTestingUtility.assertProcNotFailed(procExec, procId2); @@ -129,9 +118,7 @@ public class TestDeleteColumnFamilyProcedure { // delete the column family that exists long procId1 = procExec.submitProcedure( - new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf2.getBytes()), - nonceGroup, - nonce); + new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf2.getBytes())); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId1); // First delete should succeed @@ -142,9 +129,7 @@ public class TestDeleteColumnFamilyProcedure { // delete the column family that does not exist long procId2 = procExec.submitProcedure( - new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf2.getBytes()), - nonceGroup + 1, - nonce + 1); + new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf2.getBytes())); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId2); @@ -159,9 +144,7 @@ public class TestDeleteColumnFamilyProcedure { // Try again, this time with table disabled. UTIL.getHBaseAdmin().disableTable(tableName); long procId3 = procExec.submitProcedure( - new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf2.getBytes()), - nonceGroup + 2, - nonce + 2); + new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf2.getBytes())); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId3); // Expect fail with InvalidFamilyOperationException @@ -172,37 +155,6 @@ public class TestDeleteColumnFamilyProcedure { ProcedureTestingUtility.getExceptionCause(result) instanceof InvalidFamilyOperationException); } - @Test(timeout=60000) - public void testDeleteColumnFamilyTwiceWithSameNonce() throws Exception { - final TableName tableName = TableName.valueOf("testDeleteColumnFamilyTwiceWithSameNonce"); - final ProcedureExecutor procExec = getMasterProcedureExecutor(); - - final String cf2 = "cf2"; - - MasterProcedureTestingUtility.createTable(procExec, tableName, null, "f1", cf2); - - // delete the column family that exists - long procId1 = procExec.submitProcedure( - new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf2.getBytes()), - nonceGroup, - nonce); - long procId2 = procExec.submitProcedure( - new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf2.getBytes()), - nonceGroup, - nonce); - - // Wait the completion - ProcedureTestingUtility.waitProcedure(procExec, procId1); - ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); - MasterProcedureTestingUtility.validateColumnFamilyDeletion(UTIL.getHBaseCluster().getMaster(), - tableName, cf2); - - // Wait the completion and expect not fail - because it is the same proc - ProcedureTestingUtility.waitProcedure(procExec, procId2); - ProcedureTestingUtility.assertProcNotFailed(procExec, procId2); - assertTrue(procId1 == procId2); - } - @Test(timeout=60000) public void testDeleteNonExistingColumnFamily() throws Exception { final TableName tableName = TableName.valueOf("testDeleteNonExistingColumnFamily"); @@ -214,9 +166,7 @@ public class TestDeleteColumnFamilyProcedure { // delete the column family that does not exist long procId1 = procExec.submitProcedure( - new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf3.getBytes()), - nonceGroup, - nonce); + new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf3.getBytes())); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId1); @@ -242,9 +192,7 @@ public class TestDeleteColumnFamilyProcedure { // Start the Delete procedure && kill the executor long procId = procExec.submitProcedure( - new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf4.getBytes()), - nonceGroup, - nonce); + new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf4.getBytes())); // Restart the executor and execute the step twice int numberOfSteps = DeleteColumnFamilyState.values().length; @@ -269,9 +217,7 @@ public class TestDeleteColumnFamilyProcedure { // Start the Delete procedure && kill the executor long procId = procExec.submitProcedure( - new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf5.getBytes()), - nonceGroup, - nonce); + new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf5.getBytes())); // Restart the executor and execute the step twice int numberOfSteps = DeleteColumnFamilyState.values().length; @@ -297,9 +243,7 @@ public class TestDeleteColumnFamilyProcedure { // Start the Delete procedure && kill the executor long procId = procExec.submitProcedure( - new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf5.getBytes()), - nonceGroup, - nonce); + new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf5.getBytes())); // Failing before DELETE_COLUMN_FAMILY_DELETE_FS_LAYOUT we should trigger the rollback // NOTE: the 1 (number before DELETE_COLUMN_FAMILY_DELETE_FS_LAYOUT step) is hardcoded, @@ -330,9 +274,7 @@ public class TestDeleteColumnFamilyProcedure { // Start the Delete procedure && kill the executor long procId = procExec.submitProcedure( - new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf5.getBytes()), - nonceGroup, - nonce); + new DeleteColumnFamilyProcedure(procExec.getEnvironment(), tableName, cf5.getBytes())); // Failing after DELETE_COLUMN_FAMILY_DELETE_FS_LAYOUT we should not trigger the rollback. // NOTE: the 4 (number of DELETE_COLUMN_FAMILY_DELETE_FS_LAYOUT + 1 step) is hardcoded, diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDeleteNamespaceProcedure.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDeleteNamespaceProcedure.java index e9fbddc9f38..b672519879a 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDeleteNamespaceProcedure.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDeleteNamespaceProcedure.java @@ -28,7 +28,6 @@ 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.HConstants; import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.NamespaceDescriptor; import org.apache.hadoop.hbase.NamespaceNotFoundException; @@ -52,9 +51,6 @@ public class TestDeleteNamespaceProcedure { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private static long nonceGroup = HConstants.NO_NONCE; - private static long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1); } @@ -77,9 +73,6 @@ public class TestDeleteNamespaceProcedure { @Before public void setup() throws Exception { ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(getMasterProcedureExecutor(), false); - nonceGroup = - MasterProcedureTestingUtility.generateNonceGroup(UTIL.getHBaseCluster().getMaster()); - nonce = MasterProcedureTestingUtility.generateNonce(UTIL.getHBaseCluster().getMaster()); } @After @@ -99,9 +92,7 @@ public class TestDeleteNamespaceProcedure { createNamespaceForTesting(namespaceName); long procId = procExec.submitProcedure( - new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName), - nonceGroup, - nonce); + new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); ProcedureTestingUtility.assertProcNotFailed(procExec, procId); @@ -117,9 +108,7 @@ public class TestDeleteNamespaceProcedure { validateNamespaceNotExist(namespaceName); long procId = procExec.submitProcedure( - new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName), - nonceGroup, - nonce); + new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); // Expect fail with NamespaceNotFoundException @@ -136,9 +125,7 @@ public class TestDeleteNamespaceProcedure { final ProcedureExecutor procExec = getMasterProcedureExecutor(); long procId = procExec.submitProcedure( - new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName), - nonceGroup, - nonce); + new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); ProcedureInfo result = procExec.getResult(procId); @@ -158,9 +145,7 @@ public class TestDeleteNamespaceProcedure { MasterProcedureTestingUtility.createTable(procExec, tableName, null, "f1"); long procId = procExec.submitProcedure( - new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName), - nonceGroup, - nonce); + new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); ProcedureInfo result = procExec.getResult(procId); @@ -169,33 +154,6 @@ public class TestDeleteNamespaceProcedure { assertTrue(ProcedureTestingUtility.getExceptionCause(result) instanceof ConstraintException); } - @Test(timeout=60000) - public void testDeleteSameNamespaceTwiceWithSameNonce() throws Exception { - final String namespaceName = "testDeleteSameNamespaceTwiceWithSameNonce"; - final ProcedureExecutor procExec = getMasterProcedureExecutor(); - - createNamespaceForTesting(namespaceName); - - long procId1 = procExec.submitProcedure( - new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName), - nonceGroup, - nonce); - long procId2 = procExec.submitProcedure( - new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName), - nonceGroup, - nonce); - // Wait the completion - ProcedureTestingUtility.waitProcedure(procExec, procId1); - ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); - - validateNamespaceNotExist(namespaceName); - - // Wait the completion and expect not fail - because it is the same proc - ProcedureTestingUtility.waitProcedure(procExec, procId2); - ProcedureTestingUtility.assertProcNotFailed(procExec, procId2); - assertTrue(procId1 == procId2); - } - @Test(timeout = 60000) public void testRecoveryAndDoubleExecution() throws Exception { final String namespaceName = "testRecoveryAndDoubleExecution"; @@ -208,10 +166,7 @@ public class TestDeleteNamespaceProcedure { // Start the DeleteNamespace procedure && kill the executor long procId = procExec.submitProcedure( - new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName), - nonceGroup, - nonce); - + new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName)); // Restart the executor and execute the step twice int numberOfSteps = DeleteNamespaceState.values().length; MasterProcedureTestingUtility.testRecoveryAndDoubleExecution( @@ -237,9 +192,7 @@ public class TestDeleteNamespaceProcedure { // Start the DeleteNamespace procedure && kill the executor long procId = procExec.submitProcedure( - new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName), - nonceGroup, - nonce); + new DeleteNamespaceProcedure(procExec.getEnvironment(), namespaceName)); int numberOfSteps = DeleteNamespaceState.values().length - 2; // failing in the middle of proc MasterProcedureTestingUtility.testRollbackAndDoubleExecution( @@ -263,9 +216,7 @@ public class TestDeleteNamespaceProcedure { final ProcedureExecutor procExec = getMasterProcedureExecutor(); long procId = procExec.submitProcedure( - new CreateNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup + 1, - nonce + 1); + new CreateNamespaceProcedure(procExec.getEnvironment(), nsd)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); ProcedureTestingUtility.assertProcNotFailed(procExec, procId); diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDeleteTableProcedure.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDeleteTableProcedure.java index 4a1c4350b84..5ac0ea1d8f8 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDeleteTableProcedure.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDeleteTableProcedure.java @@ -22,7 +22,6 @@ 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.HConstants; import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.HRegionInfo; import org.apache.hadoop.hbase.ProcedureInfo; @@ -49,9 +48,6 @@ public class TestDeleteTableProcedure { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private long nonceGroup = HConstants.NO_NONCE; - private long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1); } @@ -76,10 +72,6 @@ public class TestDeleteTableProcedure { final ProcedureExecutor procExec = getMasterProcedureExecutor(); ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(procExec, false); assertTrue("expected executor to be running", procExec.isRunning()); - - nonceGroup = - MasterProcedureTestingUtility.generateNonceGroup(UTIL.getHBaseCluster().getMaster()); - nonce = MasterProcedureTestingUtility.generateNonce(UTIL.getHBaseCluster().getMaster()); } @After @@ -126,10 +118,10 @@ public class TestDeleteTableProcedure { // delete the table (that exists) long procId1 = procExec.submitProcedure( - new DeleteTableProcedure(procExec.getEnvironment(), tableName), nonceGroup, nonce); + new DeleteTableProcedure(procExec.getEnvironment(), tableName)); // delete the table (that will no longer exist) long procId2 = procExec.submitProcedure( - new DeleteTableProcedure(procExec.getEnvironment(), tableName), nonceGroup + 1, nonce + 1); + new DeleteTableProcedure(procExec.getEnvironment(), tableName)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId1); @@ -147,36 +139,6 @@ public class TestDeleteTableProcedure { assertTrue(ProcedureTestingUtility.getExceptionCause(result) instanceof TableNotFoundException); } - @Test(timeout=60000) - public void testDoubleDeletedTableWithSameNonce() throws Exception { - final TableName tableName = TableName.valueOf("testDoubleDeletedTableWithSameNonce"); - final ProcedureExecutor procExec = getMasterProcedureExecutor(); - - HRegionInfo[] regions = MasterProcedureTestingUtility.createTable( - procExec, tableName, null, "f"); - UTIL.getHBaseAdmin().disableTable(tableName); - - // delete the table (that exists) - long procId1 = procExec.submitProcedure( - new DeleteTableProcedure(procExec.getEnvironment(), tableName), nonceGroup, nonce); - // delete the table (that will no longer exist) - long procId2 = procExec.submitProcedure( - new DeleteTableProcedure(procExec.getEnvironment(), tableName), nonceGroup, nonce); - - // Wait the completion - ProcedureTestingUtility.waitProcedure(procExec, procId1); - ProcedureTestingUtility.waitProcedure(procExec, procId2); - - // First delete should succeed - ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); - MasterProcedureTestingUtility.validateTableDeletion( - UTIL.getHBaseCluster().getMaster(), tableName, regions, "f"); - - // Second delete should not fail, because it is the same delete - ProcedureTestingUtility.assertProcNotFailed(procExec, procId2); - assertTrue(procId1 == procId2); - } - @Test(timeout=60000) public void testSimpleDelete() throws Exception { final TableName tableName = TableName.valueOf("testSimpleDelete"); @@ -223,7 +185,7 @@ public class TestDeleteTableProcedure { // Start the Delete procedure && kill the executor long procId = procExec.submitProcedure( - new DeleteTableProcedure(procExec.getEnvironment(), tableName), nonceGroup, nonce); + new DeleteTableProcedure(procExec.getEnvironment(), tableName)); // Restart the executor and execute the step twice // NOTE: the 6 (number of DeleteTableState steps) is hardcoded, diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDisableTableProcedure.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDisableTableProcedure.java index 078db92193f..72d81130591 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDisableTableProcedure.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestDisableTableProcedure.java @@ -24,7 +24,6 @@ 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.HConstants; import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.ProcedureInfo; import org.apache.hadoop.hbase.TableName; @@ -48,9 +47,6 @@ public class TestDisableTableProcedure { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private static long nonceGroup = HConstants.NO_NONCE; - private static long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1); } @@ -73,9 +69,6 @@ public class TestDisableTableProcedure { @Before public void setup() throws Exception { ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(getMasterProcedureExecutor(), false); - nonceGroup = - MasterProcedureTestingUtility.generateNonceGroup(UTIL.getHBaseCluster().getMaster()); - nonce = MasterProcedureTestingUtility.generateNonce(UTIL.getHBaseCluster().getMaster()); } @After @@ -96,7 +89,7 @@ public class TestDisableTableProcedure { // Disable the table long procId = procExec.submitProcedure( - new DisableTableProcedure(procExec.getEnvironment(), tableName, false), nonceGroup, nonce); + new DisableTableProcedure(procExec.getEnvironment(), tableName, false)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); ProcedureTestingUtility.assertProcNotFailed(procExec, procId); @@ -113,7 +106,7 @@ public class TestDisableTableProcedure { // Disable the table long procId1 = procExec.submitProcedure(new DisableTableProcedure( - procExec.getEnvironment(), tableName, false), nonceGroup, nonce); + procExec.getEnvironment(), tableName, false)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId1); ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); @@ -122,7 +115,7 @@ public class TestDisableTableProcedure { // Disable the table again - expect failure long procId2 = procExec.submitProcedure(new DisableTableProcedure( - procExec.getEnvironment(), tableName, false), nonceGroup + 1, nonce + 1); + procExec.getEnvironment(), tableName, false)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId2); ProcedureInfo result = procExec.getResult(procId2); @@ -136,7 +129,7 @@ public class TestDisableTableProcedure { final ProcedurePrepareLatch prepareLatch = new ProcedurePrepareLatch.CompatibilityLatch(); long procId3 = procExec.submitProcedure(new DisableTableProcedure( - procExec.getEnvironment(), tableName, false, prepareLatch), nonceGroup + 2, nonce + 2); + procExec.getEnvironment(), tableName, false, prepareLatch)); prepareLatch.await(); Assert.fail("Disable should throw exception through latch."); } catch (TableNotEnabledException tnee) { @@ -154,29 +147,6 @@ public class TestDisableTableProcedure { tableName); } - @Test(timeout = 60000) - public void testDisableTableTwiceWithSameNonce() throws Exception { - final TableName tableName = TableName.valueOf("testDisableTableTwiceWithSameNonce"); - final ProcedureExecutor procExec = getMasterProcedureExecutor(); - - MasterProcedureTestingUtility.createTable(procExec, tableName, null, "f1", "f2"); - - // Disable the table - long procId1 = procExec.submitProcedure(new DisableTableProcedure( - procExec.getEnvironment(), tableName, false), nonceGroup, nonce); - long procId2 = procExec.submitProcedure(new DisableTableProcedure( - procExec.getEnvironment(), tableName, false), nonceGroup, nonce); - // Wait the completion - ProcedureTestingUtility.waitProcedure(procExec, procId1); - ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); - MasterProcedureTestingUtility.validateTableIsDisabled(UTIL.getHBaseCluster().getMaster(), - tableName); - - ProcedureTestingUtility.waitProcedure(procExec, procId2); - ProcedureTestingUtility.assertProcNotFailed(procExec, procId2); - assertTrue(procId1 == procId2); - } - @Test(timeout=60000) public void testRecoveryAndDoubleExecution() throws Exception { final TableName tableName = TableName.valueOf("testRecoveryAndDoubleExecution"); @@ -191,7 +161,7 @@ public class TestDisableTableProcedure { // Start the Disable procedure && kill the executor long procId = procExec.submitProcedure( - new DisableTableProcedure(procExec.getEnvironment(), tableName, false), nonceGroup, nonce); + new DisableTableProcedure(procExec.getEnvironment(), tableName, false)); // Restart the executor and execute the step twice int numberOfSteps = DisableTableState.values().length; diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestEnableTableProcedure.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestEnableTableProcedure.java index 820024678b4..baaa64d25c0 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestEnableTableProcedure.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestEnableTableProcedure.java @@ -24,7 +24,6 @@ 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.HConstants; import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.ProcedureInfo; import org.apache.hadoop.hbase.TableName; @@ -48,9 +47,6 @@ public class TestEnableTableProcedure { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private static long nonceGroup = HConstants.NO_NONCE; - private static long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1); } @@ -73,9 +69,6 @@ public class TestEnableTableProcedure { @Before public void setup() throws Exception { ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(getMasterProcedureExecutor(), false); - nonceGroup = - MasterProcedureTestingUtility.generateNonceGroup(UTIL.getHBaseCluster().getMaster()); - nonce = MasterProcedureTestingUtility.generateNonce(UTIL.getHBaseCluster().getMaster()); } @After @@ -97,7 +90,7 @@ public class TestEnableTableProcedure { // Enable the table long procId = procExec.submitProcedure( - new EnableTableProcedure(procExec.getEnvironment(), tableName, false), nonceGroup, nonce); + new EnableTableProcedure(procExec.getEnvironment(), tableName, false)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); ProcedureTestingUtility.assertProcNotFailed(procExec, procId); @@ -105,29 +98,6 @@ public class TestEnableTableProcedure { tableName); } - @Test(timeout = 60000) - public void testEnableTableTwiceWithSameNonce() throws Exception { - final TableName tableName = TableName.valueOf("testEnableTableTwiceWithSameNonce"); - final ProcedureExecutor procExec = getMasterProcedureExecutor(); - - MasterProcedureTestingUtility.createTable(procExec, tableName, null, "f1", "f2"); - UTIL.getHBaseAdmin().disableTable(tableName); - - // Enable the table - long procId1 = procExec.submitProcedure( - new EnableTableProcedure(procExec.getEnvironment(), tableName, false), nonceGroup, nonce); - long procId2 = procExec.submitProcedure( - new EnableTableProcedure(procExec.getEnvironment(), tableName, false), nonceGroup, nonce); - - // Wait the completion - ProcedureTestingUtility.waitProcedure(procExec, procId1); - ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); - // The second proc should succeed too - because it is the same proc. - ProcedureTestingUtility.waitProcedure(procExec, procId2); - ProcedureTestingUtility.assertProcNotFailed(procExec, procId2); - assertTrue(procId1 == procId2); - } - @Test(timeout=60000, expected=TableNotDisabledException.class) public void testEnableNonDisabledTable() throws Exception { final TableName tableName = TableName.valueOf("testEnableNonExistingTable"); @@ -137,7 +107,7 @@ public class TestEnableTableProcedure { // Enable the table - expect failure long procId1 = procExec.submitProcedure( - new EnableTableProcedure(procExec.getEnvironment(), tableName, false), nonceGroup, nonce); + new EnableTableProcedure(procExec.getEnvironment(), tableName, false)); ProcedureTestingUtility.waitProcedure(procExec, procId1); ProcedureInfo result = procExec.getResult(procId1); @@ -148,9 +118,7 @@ public class TestEnableTableProcedure { // Enable the table with skipping table state check flag (simulate recovery scenario) long procId2 = procExec.submitProcedure( - new EnableTableProcedure(procExec.getEnvironment(), tableName, true), - nonceGroup + 1, - nonce + 1); + new EnableTableProcedure(procExec.getEnvironment(), tableName, true)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId2); ProcedureTestingUtility.assertProcNotFailed(procExec, procId2); @@ -158,9 +126,7 @@ public class TestEnableTableProcedure { // Enable the table - expect failure from ProcedurePrepareLatch final ProcedurePrepareLatch prepareLatch = new ProcedurePrepareLatch.CompatibilityLatch(); long procId3 = procExec.submitProcedure( - new EnableTableProcedure(procExec.getEnvironment(), tableName, false, prepareLatch), - nonceGroup + 2, - nonce + 2); + new EnableTableProcedure(procExec.getEnvironment(), tableName, false, prepareLatch)); prepareLatch.await(); Assert.fail("Enable should throw exception through latch."); } @@ -180,7 +146,7 @@ public class TestEnableTableProcedure { // Start the Enable procedure && kill the executor long procId = procExec.submitProcedure( - new EnableTableProcedure(procExec.getEnvironment(), tableName, false), nonceGroup, nonce); + new EnableTableProcedure(procExec.getEnvironment(), tableName, false)); // Restart the executor and execute the step twice int numberOfSteps = EnableTableState.values().length; @@ -208,7 +174,7 @@ public class TestEnableTableProcedure { // Start the Enable procedure && kill the executor long procId = procExec.submitProcedure( - new EnableTableProcedure(procExec.getEnvironment(), tableName, false), nonceGroup, nonce); + new EnableTableProcedure(procExec.getEnvironment(), tableName, false)); int numberOfSteps = EnableTableState.values().length - 2; // failing in the middle of proc MasterProcedureTestingUtility.testRollbackAndDoubleExecution( diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestMasterProcedureEvents.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestMasterProcedureEvents.java index adf3e5c422e..c87794b621b 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestMasterProcedureEvents.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestMasterProcedureEvents.java @@ -18,13 +18,10 @@ package org.apache.hadoop.hbase.master.procedure; -import java.io.IOException; - 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.HConstants; import org.apache.hadoop.hbase.HRegionInfo; import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.HColumnDescriptor; @@ -38,9 +35,7 @@ import org.apache.hadoop.hbase.procedure2.store.wal.WALProcedureStore; import org.apache.hadoop.hbase.regionserver.HRegionServer; import org.apache.hadoop.hbase.testclassification.MasterTests; import org.apache.hadoop.hbase.testclassification.MediumTests; -import org.junit.After; import org.junit.AfterClass; -import org.junit.Before; import org.junit.BeforeClass; import org.junit.Test; import org.junit.experimental.categories.Category; @@ -54,9 +49,6 @@ public class TestMasterProcedureEvents { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private static long nonceGroup = HConstants.NO_NONCE; - private static long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 8); conf.setBoolean(WALProcedureStore.USE_HSYNC_CONF_KEY, false); @@ -98,7 +90,7 @@ public class TestMasterProcedureEvents { long pollCalls = procSched.getPollCalls(); long nullPollCalls = procSched.getNullPollCalls(); - long procId = procExec.submitProcedure(proc, HConstants.NO_NONCE, HConstants.NO_NONCE); + long procId = procExec.submitProcedure(proc); for (int i = 0; i < 10; ++i) { Thread.sleep(100); assertEquals(pollCalls + 1, procSched.getPollCalls()); diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestModifyColumnFamilyProcedure.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestModifyColumnFamilyProcedure.java index 12b2ad8cae7..4ccf24b6afa 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestModifyColumnFamilyProcedure.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestModifyColumnFamilyProcedure.java @@ -25,7 +25,6 @@ import org.apache.commons.logging.LogFactory; import org.apache.hadoop.conf.Configuration; import org.apache.hadoop.hbase.HBaseTestingUtility; import org.apache.hadoop.hbase.HColumnDescriptor; -import org.apache.hadoop.hbase.HConstants; import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.InvalidFamilyOperationException; import org.apache.hadoop.hbase.ProcedureInfo; @@ -47,9 +46,6 @@ public class TestModifyColumnFamilyProcedure { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private static long nonceGroup = HConstants.NO_NONCE; - private static long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1); } @@ -72,9 +68,6 @@ public class TestModifyColumnFamilyProcedure { @Before public void setup() throws Exception { ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(getMasterProcedureExecutor(), false); - nonceGroup = - MasterProcedureTestingUtility.generateNonceGroup(UTIL.getHBaseCluster().getMaster()); - nonce = MasterProcedureTestingUtility.generateNonce(UTIL.getHBaseCluster().getMaster()); } @After @@ -101,9 +94,7 @@ public class TestModifyColumnFamilyProcedure { // Test 1: modify the column family online columnDescriptor.setBlocksize(newBlockSize); long procId1 = procExec.submitProcedure( - new ModifyColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup, - nonce); + new ModifyColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId1); ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); @@ -114,9 +105,7 @@ public class TestModifyColumnFamilyProcedure { UTIL.getHBaseAdmin().disableTable(tableName); columnDescriptor.setBlocksize(newBlockSize * 2); long procId2 = procExec.submitProcedure( - new ModifyColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup + 1, - nonce + 1); + new ModifyColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId2); ProcedureTestingUtility.assertProcNotFailed(procExec, procId2); @@ -139,9 +128,7 @@ public class TestModifyColumnFamilyProcedure { // Modify the column family that does not exist columnDescriptor.setBlocksize(newBlockSize); long procId1 = procExec.submitProcedure( - new ModifyColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup, - nonce); + new ModifyColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId1); @@ -171,9 +158,7 @@ public class TestModifyColumnFamilyProcedure { // Start the Modify procedure && kill the executor columnDescriptor.setBlocksize(newBlockSize); long procId = procExec.submitProcedure( - new ModifyColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup, - nonce); + new ModifyColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor)); // Restart the executor and execute the step twice int numberOfSteps = ModifyColumnFamilyState.values().length; @@ -205,9 +190,7 @@ public class TestModifyColumnFamilyProcedure { // Start the Modify procedure && kill the executor columnDescriptor.setBlocksize(newBlockSize); long procId = procExec.submitProcedure( - new ModifyColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup, - nonce); + new ModifyColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor)); // Restart the executor and execute the step twice int numberOfSteps = ModifyColumnFamilyState.values().length; @@ -236,9 +219,7 @@ public class TestModifyColumnFamilyProcedure { // Start the Modify procedure && kill the executor columnDescriptor.setBlocksize(newBlockSize); long procId = procExec.submitProcedure( - new ModifyColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor), - nonceGroup, - nonce); + new ModifyColumnFamilyProcedure(procExec.getEnvironment(), tableName, columnDescriptor)); // Failing in the middle of proc int numberOfSteps = ModifyColumnFamilyState.values().length - 2; diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestModifyNamespaceProcedure.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestModifyNamespaceProcedure.java index c38eab88234..327afc4f29e 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestModifyNamespaceProcedure.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestModifyNamespaceProcedure.java @@ -18,14 +18,12 @@ package org.apache.hadoop.hbase.master.procedure; - import static org.junit.Assert.*; 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.HConstants; import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.NamespaceDescriptor; import org.apache.hadoop.hbase.NamespaceNotFoundException; @@ -48,9 +46,6 @@ public class TestModifyNamespaceProcedure { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private static long nonceGroup = HConstants.NO_NONCE; - private static long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1); } @@ -73,9 +68,6 @@ public class TestModifyNamespaceProcedure { @Before public void setup() throws Exception { ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(getMasterProcedureExecutor(), false); - nonceGroup = - MasterProcedureTestingUtility.generateNonceGroup(UTIL.getHBaseCluster().getMaster()); - nonce = MasterProcedureTestingUtility.generateNonce(UTIL.getHBaseCluster().getMaster()); } @After @@ -112,9 +104,7 @@ public class TestModifyNamespaceProcedure { nsd.setConfiguration(nsKey2, nsValue2); long procId1 = procExec.submitProcedure( - new ModifyNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); + new ModifyNamespaceProcedure(procExec.getEnvironment(), nsd)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId1); ProcedureTestingUtility.assertProcNotFailed(procExec, procId1); @@ -142,9 +132,7 @@ public class TestModifyNamespaceProcedure { final NamespaceDescriptor nsd = NamespaceDescriptor.create(namespaceName).build(); long procId = procExec.submitProcedure( - new ModifyNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); + new ModifyNamespaceProcedure(procExec.getEnvironment(), nsd)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); @@ -170,9 +158,7 @@ public class TestModifyNamespaceProcedure { nsd.setConfiguration(nsKey, nsValue); long procId = procExec.submitProcedure( - new ModifyNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); + new ModifyNamespaceProcedure(procExec.getEnvironment(), nsd)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); ProcedureInfo result = procExec.getResult(procId); @@ -195,9 +181,7 @@ public class TestModifyNamespaceProcedure { nsd.setConfiguration(nsKey, nsValue); long procId = procExec.submitProcedure( - new ModifyNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); + new ModifyNamespaceProcedure(procExec.getEnvironment(), nsd)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); ProcedureInfo result = procExec.getResult(procId); @@ -223,9 +207,7 @@ public class TestModifyNamespaceProcedure { // Start the Modify procedure && kill the executor long procId = procExec.submitProcedure( - new ModifyNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); + new ModifyNamespaceProcedure(procExec.getEnvironment(), nsd)); // Restart the executor and execute the step twice int numberOfSteps = ModifyNamespaceState.values().length; @@ -259,9 +241,7 @@ public class TestModifyNamespaceProcedure { // Start the Modify procedure && kill the executor long procId = procExec.submitProcedure( - new ModifyNamespaceProcedure(procExec.getEnvironment(), nsd), - nonceGroup, - nonce); + new ModifyNamespaceProcedure(procExec.getEnvironment(), nsd)); // Failing in the middle of proc int numberOfSteps = ModifyNamespaceState.values().length - 2; @@ -285,9 +265,7 @@ public class TestModifyNamespaceProcedure { final ProcedureExecutor procExec = getMasterProcedureExecutor(); long procId = procExec.submitProcedure( - new CreateNamespaceProcedure(procExec.getEnvironment(), nsDescriptor), - nonceGroup + 1, - nonce + 1); + new CreateNamespaceProcedure(procExec.getEnvironment(), nsDescriptor)); // Wait the completion ProcedureTestingUtility.waitProcedure(procExec, procId); ProcedureTestingUtility.assertProcNotFailed(procExec, procId); diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestModifyTableProcedure.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestModifyTableProcedure.java index ae43867e538..ac3b84a0ff9 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestModifyTableProcedure.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestModifyTableProcedure.java @@ -28,7 +28,6 @@ import org.apache.hadoop.conf.Configuration; import org.apache.hadoop.hbase.DoNotRetryIOException; import org.apache.hadoop.hbase.HBaseTestingUtility; import org.apache.hadoop.hbase.HColumnDescriptor; -import org.apache.hadoop.hbase.HConstants; import org.apache.hadoop.hbase.HRegionInfo; import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.ProcedureInfo; @@ -50,9 +49,6 @@ public class TestModifyTableProcedure { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private static long nonceGroup = HConstants.NO_NONCE; - private static long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1); } @@ -75,9 +71,6 @@ public class TestModifyTableProcedure { @Before public void setup() throws Exception { ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(getMasterProcedureExecutor(), false); - nonceGroup = - MasterProcedureTestingUtility.generateNonceGroup(UTIL.getHBaseCluster().getMaster()); - nonce = MasterProcedureTestingUtility.generateNonce(UTIL.getHBaseCluster().getMaster()); } @After @@ -251,7 +244,7 @@ public class TestModifyTableProcedure { // Start the Modify procedure && kill the executor long procId = procExec.submitProcedure( - new ModifyTableProcedure(procExec.getEnvironment(), htd), nonceGroup, nonce); + new ModifyTableProcedure(procExec.getEnvironment(), htd)); // Restart the executor and execute the step twice int numberOfSteps = ModifyTableState.values().length; @@ -293,7 +286,7 @@ public class TestModifyTableProcedure { // Start the Modify procedure && kill the executor long procId = procExec.submitProcedure( - new ModifyTableProcedure(procExec.getEnvironment(), htd), nonceGroup, nonce); + new ModifyTableProcedure(procExec.getEnvironment(), htd)); // Restart the executor and execute the step twice int numberOfSteps = ModifyTableState.values().length; @@ -331,7 +324,7 @@ public class TestModifyTableProcedure { // Start the Modify procedure && kill the executor long procId = procExec.submitProcedure( - new ModifyTableProcedure(procExec.getEnvironment(), htd), nonceGroup, nonce); + new ModifyTableProcedure(procExec.getEnvironment(), htd)); // Restart the executor and rollback the step twice int numberOfSteps = ModifyTableState.values().length - 4; // failing in the middle of proc @@ -368,7 +361,7 @@ public class TestModifyTableProcedure { // Start the Modify procedure && kill the executor long procId = procExec.submitProcedure( - new ModifyTableProcedure(procExec.getEnvironment(), htd), nonceGroup, nonce); + new ModifyTableProcedure(procExec.getEnvironment(), htd)); // Restart the executor and rollback the step twice int numberOfSteps = ModifyTableState.values().length - 4; // failing in the middle of proc @@ -406,7 +399,7 @@ public class TestModifyTableProcedure { // Start the Modify procedure && kill the executor long procId = procExec.submitProcedure( - new ModifyTableProcedure(procExec.getEnvironment(), htd), nonceGroup, nonce); + new ModifyTableProcedure(procExec.getEnvironment(), htd)); // Failing after MODIFY_TABLE_DELETE_FS_LAYOUT we should not trigger the rollback. // NOTE: the 5 (number of MODIFY_TABLE_DELETE_FS_LAYOUT + 1 step) is hardcoded, diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestProcedureAdmin.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestProcedureAdmin.java index 8097ae5aa9f..4dd85071a30 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestProcedureAdmin.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestProcedureAdmin.java @@ -25,7 +25,6 @@ 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.HConstants; import org.apache.hadoop.hbase.HRegionInfo; import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.ProcedureInfo; @@ -49,9 +48,6 @@ public class TestProcedureAdmin { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private long nonceGroup = HConstants.NO_NONCE; - private long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1); } @@ -76,10 +72,6 @@ public class TestProcedureAdmin { final ProcedureExecutor procExec = getMasterProcedureExecutor(); ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(procExec, false); assertTrue("expected executor to be running", procExec.isRunning()); - - nonceGroup = - MasterProcedureTestingUtility.generateNonceGroup(UTIL.getHBaseCluster().getMaster()); - nonce = MasterProcedureTestingUtility.generateNonce(UTIL.getHBaseCluster().getMaster()); } @After @@ -102,7 +94,7 @@ public class TestProcedureAdmin { ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(procExec, true); // Submit an abortable procedure long procId = procExec.submitProcedure( - new DisableTableProcedure(procExec.getEnvironment(), tableName, false), nonceGroup, nonce); + new DisableTableProcedure(procExec.getEnvironment(), tableName, false)); // Wait for one step to complete ProcedureTestingUtility.waitProcedure(procExec, procId); @@ -129,7 +121,7 @@ public class TestProcedureAdmin { ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(procExec, true); // Submit an un-abortable procedure long procId = procExec.submitProcedure( - new DeleteTableProcedure(procExec.getEnvironment(), tableName), nonceGroup, nonce); + new DeleteTableProcedure(procExec.getEnvironment(), tableName)); // Wait for one step to complete ProcedureTestingUtility.waitProcedure(procExec, procId); @@ -155,7 +147,7 @@ public class TestProcedureAdmin { ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(procExec, true); // Submit a procedure long procId = procExec.submitProcedure( - new DisableTableProcedure(procExec.getEnvironment(), tableName, true), nonceGroup, nonce); + new DisableTableProcedure(procExec.getEnvironment(), tableName, true)); // Wait for one step to complete ProcedureTestingUtility.waitProcedure(procExec, procId); @@ -196,7 +188,7 @@ public class TestProcedureAdmin { ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(procExec, true); long procId = procExec.submitProcedure( - new DisableTableProcedure(procExec.getEnvironment(), tableName, false), nonceGroup, nonce); + new DisableTableProcedure(procExec.getEnvironment(), tableName, false)); // Wait for one step to complete ProcedureTestingUtility.waitProcedure(procExec, procId); diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestTruncateTableProcedure.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestTruncateTableProcedure.java index 7a3df0f01c5..8797a4fad2f 100644 --- a/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestTruncateTableProcedure.java +++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/master/procedure/TestTruncateTableProcedure.java @@ -22,7 +22,6 @@ 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.HConstants; import org.apache.hadoop.hbase.HTableDescriptor; import org.apache.hadoop.hbase.HRegionInfo; import org.apache.hadoop.hbase.ProcedureInfo; @@ -50,9 +49,6 @@ public class TestTruncateTableProcedure { protected static final HBaseTestingUtility UTIL = new HBaseTestingUtility(); - private static long nonceGroup = HConstants.NO_NONCE; - private static long nonce = HConstants.NO_NONCE; - private static void setupConf(Configuration conf) { conf.setInt(MasterProcedureConstants.MASTER_PROCEDURE_THREADS, 1); } @@ -77,10 +73,6 @@ public class TestTruncateTableProcedure { final ProcedureExecutor procExec = getMasterProcedureExecutor(); ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(procExec, false); assertTrue("expected executor to be running", procExec.isRunning()); - - nonceGroup = - MasterProcedureTestingUtility.generateNonceGroup(UTIL.getHBaseCluster().getMaster()); - nonce = MasterProcedureTestingUtility.generateNonce(UTIL.getHBaseCluster().getMaster()); } @After @@ -215,9 +207,7 @@ public class TestTruncateTableProcedure { // Start the Truncate procedure && kill the executor long procId = procExec.submitProcedure( - new TruncateTableProcedure(procExec.getEnvironment(), tableName, preserveSplits), - nonceGroup, - nonce); + new TruncateTableProcedure(procExec.getEnvironment(), tableName, preserveSplits)); // Restart the executor and execute the step twice // NOTE: the 7 (number of TruncateTableState steps) is hardcoded,