1 /**
2  * Licensed to the Apache Software Foundation (ASF) under one
3  * or more contributor license agreements.  See the NOTICE file
4  * distributed with this work for additional information
5  * regarding copyright ownership.  The ASF licenses this file
6  * to you under the Apache License, Version 2.0 (the
7  * "License"); you may not use this file except in compliance
8  * with the License.  You may obtain a copy of the License at
9  *
10  *     http://www.apache.org/licenses/LICENSE-2.0
11  *
12  * Unless required by applicable law or agreed to in writing, software
13  * distributed under the License is distributed on an "AS IS" BASIS,
14  * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15  * See the License for the specific language governing permissions and
16  * limitations under the License.
17  */
18 
19 package org.apache.hadoop.hbase.master.procedure;
20 
21 import java.io.IOException;
22 import java.util.concurrent.atomic.AtomicInteger;
23 import java.util.List;
24 
25 import org.apache.commons.logging.Log;
26 import org.apache.commons.logging.LogFactory;
27 import org.apache.hadoop.fs.FileSystem;
28 import org.apache.hadoop.fs.Path;
29 import org.apache.hadoop.hbase.HColumnDescriptor;
30 import org.apache.hadoop.hbase.HRegionInfo;
31 import org.apache.hadoop.hbase.HRegionLocation;
32 import org.apache.hadoop.hbase.HTableDescriptor;
33 import org.apache.hadoop.hbase.MetaTableAccessor;
34 import org.apache.hadoop.hbase.RegionLocations;
35 import org.apache.hadoop.hbase.ServerName;
36 import org.apache.hadoop.hbase.TableName;
37 import org.apache.hadoop.hbase.TableStateManager;
38 import org.apache.hadoop.hbase.client.BufferedMutator;
39 import org.apache.hadoop.hbase.client.Connection;
40 import org.apache.hadoop.hbase.client.Durability;
41 import org.apache.hadoop.hbase.client.NonceGenerator;
42 import org.apache.hadoop.hbase.client.Put;
43 import org.apache.hadoop.hbase.client.MetaScanner;
44 import org.apache.hadoop.hbase.client.MetaScanner.MetaScannerVisitor;
45 import org.apache.hadoop.hbase.client.MetaScanner.MetaScannerVisitorBase;
46 import org.apache.hadoop.hbase.client.Result;
47 import org.apache.hadoop.hbase.master.HMaster;
48 import org.apache.hadoop.hbase.procedure2.ProcedureExecutor;
49 import org.apache.hadoop.hbase.procedure2.ProcedureTestingUtility;
50 import org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos;
51 import org.apache.hadoop.hbase.util.ModifyRegionUtils;
52 import org.apache.hadoop.hbase.util.Bytes;
53 import org.apache.hadoop.hbase.util.FSUtils;
54 import org.apache.hadoop.hbase.util.MD5Hash;
55 
56 import static org.junit.Assert.assertEquals;
57 import static org.junit.Assert.assertFalse;
58 import static org.junit.Assert.assertTrue;
59 import static org.junit.Assert.fail;
60 
61 public class MasterProcedureTestingUtility {
62   private static final Log LOG = LogFactory.getLog(MasterProcedureTestingUtility.class);
63 
MasterProcedureTestingUtility()64   private MasterProcedureTestingUtility() {
65   }
66 
createHTD(final TableName tableName, final String... family)67   public static HTableDescriptor createHTD(final TableName tableName, final String... family) {
68     HTableDescriptor htd = new HTableDescriptor(tableName);
69     for (int i = 0; i < family.length; ++i) {
70       htd.addFamily(new HColumnDescriptor(family[i]));
71     }
72     return htd;
73   }
74 
createTable(final ProcedureExecutor<MasterProcedureEnv> procExec, final TableName tableName, final byte[][] splitKeys, String... family)75   public static HRegionInfo[] createTable(final ProcedureExecutor<MasterProcedureEnv> procExec,
76       final TableName tableName, final byte[][] splitKeys, String... family) throws IOException {
77     HTableDescriptor htd = createHTD(tableName, family);
78     HRegionInfo[] regions = ModifyRegionUtils.createHRegionInfos(htd, splitKeys);
79     long procId = ProcedureTestingUtility.submitAndWait(procExec,
80       new CreateTableProcedure(procExec.getEnvironment(), htd, regions));
81     ProcedureTestingUtility.assertProcNotFailed(procExec.getResult(procId));
82     return regions;
83   }
84 
validateTableCreation(final HMaster master, final TableName tableName, final HRegionInfo[] regions, String... family)85   public static void validateTableCreation(final HMaster master, final TableName tableName,
86       final HRegionInfo[] regions, String... family) throws IOException {
87     validateTableCreation(master, tableName, regions, true, family);
88   }
89 
validateTableCreation(final HMaster master, final TableName tableName, final HRegionInfo[] regions, boolean hasFamilyDirs, String... family)90   public static void validateTableCreation(final HMaster master, final TableName tableName,
91       final HRegionInfo[] regions, boolean hasFamilyDirs, String... family) throws IOException {
92     // check filesystem
93     final FileSystem fs = master.getMasterFileSystem().getFileSystem();
94     final Path tableDir = FSUtils.getTableDir(master.getMasterFileSystem().getRootDir(), tableName);
95     assertTrue(fs.exists(tableDir));
96     FSUtils.logFileSystemState(fs, tableDir, LOG);
97     List<Path> allRegionDirs = FSUtils.getRegionDirs(fs, tableDir);
98     for (int i = 0; i < regions.length; ++i) {
99       Path regionDir = new Path(tableDir, regions[i].getEncodedName());
100       assertTrue(regions[i] + " region dir does not exist", fs.exists(regionDir));
101       assertTrue(allRegionDirs.remove(regionDir));
102       List<Path> allFamilyDirs = FSUtils.getFamilyDirs(fs, regionDir);
103       for (int j = 0; j < family.length; ++j) {
104         final Path familyDir = new Path(regionDir, family[j]);
105         if (hasFamilyDirs) {
106           assertTrue(family[j] + " family dir does not exist", fs.exists(familyDir));
107           assertTrue(allFamilyDirs.remove(familyDir));
108         } else {
109           // TODO: WARN: Modify Table/Families does not create a family dir
110           if (!fs.exists(familyDir)) {
111             LOG.warn(family[j] + " family dir does not exist");
112           }
113           allFamilyDirs.remove(familyDir);
114         }
115       }
116       assertTrue("found extraneous families: " + allFamilyDirs, allFamilyDirs.isEmpty());
117     }
118     assertTrue("found extraneous regions: " + allRegionDirs, allRegionDirs.isEmpty());
119 
120     // check meta
121     assertTrue(MetaTableAccessor.tableExists(master.getConnection(), tableName));
122     assertEquals(regions.length, countMetaRegions(master, tableName));
123 
124     // check htd
125     HTableDescriptor htd = master.getTableDescriptors().get(tableName);
126     assertTrue("table descriptor not found", htd != null);
127     for (int i = 0; i < family.length; ++i) {
128       assertTrue("family not found " + family[i], htd.getFamily(Bytes.toBytes(family[i])) != null);
129     }
130     assertEquals(family.length, htd.getFamilies().size());
131   }
132 
validateTableDeletion(final HMaster master, final TableName tableName, final HRegionInfo[] regions, String... family)133   public static void validateTableDeletion(final HMaster master, final TableName tableName,
134       final HRegionInfo[] regions, String... family) throws IOException {
135     // check filesystem
136     final FileSystem fs = master.getMasterFileSystem().getFileSystem();
137     final Path tableDir = FSUtils.getTableDir(master.getMasterFileSystem().getRootDir(), tableName);
138     assertFalse(fs.exists(tableDir));
139 
140     // check meta
141     assertFalse(MetaTableAccessor.tableExists(master.getConnection(), tableName));
142     assertEquals(0, countMetaRegions(master, tableName));
143 
144     // check htd
145     assertTrue("found htd of deleted table",
146       master.getTableDescriptors().get(tableName) == null);
147   }
148 
countMetaRegions(final HMaster master, final TableName tableName)149   private static int countMetaRegions(final HMaster master, final TableName tableName)
150       throws IOException {
151     final AtomicInteger actualRegCount = new AtomicInteger(0);
152     final MetaScannerVisitor visitor = new MetaScannerVisitorBase() {
153       @Override
154       public boolean processRow(Result rowResult) throws IOException {
155         RegionLocations list = MetaTableAccessor.getRegionLocations(rowResult);
156         if (list == null) {
157           LOG.warn("No serialized HRegionInfo in " + rowResult);
158           return true;
159         }
160         HRegionLocation l = list.getRegionLocation();
161         if (l == null) {
162           return true;
163         }
164         if (!l.getRegionInfo().getTable().equals(tableName)) {
165           return false;
166         }
167         if (l.getRegionInfo().isOffline() || l.getRegionInfo().isSplit()) return true;
168         HRegionLocation[] locations = list.getRegionLocations();
169         for (HRegionLocation location : locations) {
170           if (location == null) continue;
171           ServerName serverName = location.getServerName();
172           // Make sure that regions are assigned to server
173           if (serverName != null && serverName.getHostAndPort() != null) {
174             actualRegCount.incrementAndGet();
175           }
176         }
177         return true;
178       }
179     };
180     MetaScanner.metaScan(master.getConnection(), visitor, tableName);
181     return actualRegCount.get();
182   }
183 
validateTableIsEnabled(final HMaster master, final TableName tableName)184   public static void validateTableIsEnabled(final HMaster master, final TableName tableName)
185       throws IOException {
186     TableStateManager tsm = master.getAssignmentManager().getTableStateManager();
187     assertTrue(tsm.isTableState(tableName, ZooKeeperProtos.Table.State.ENABLED));
188   }
189 
validateTableIsDisabled(final HMaster master, final TableName tableName)190   public static void validateTableIsDisabled(final HMaster master, final TableName tableName)
191       throws IOException {
192     TableStateManager tsm = master.getAssignmentManager().getTableStateManager();
193     assertTrue(tsm.isTableState(tableName, ZooKeeperProtos.Table.State.DISABLED));
194   }
195 
196   /**
197    * Run through all procedure flow states TWICE while also restarting procedure executor at each
198    * step; i.e force a reread of procedure store.
199    *
200    *<p>It does
201    * <ol><li>Execute step N - kill the executor before store update
202    * <li>Restart executor/store
203    * <li>Execute step N - and then save to store
204    * </ol>
205    *
206    *<p>This is a good test for finding state that needs persisting and steps that are not
207    * idempotent. Use this version of the test when a procedure executes all flow steps from start to
208    * finish.
209    * @see #testRecoveryAndDoubleExecution(ProcedureExecutor, long)
210    */
testRecoveryAndDoubleExecution( final ProcedureExecutor<MasterProcedureEnv> procExec, final long procId, final int numSteps, final TState[] states)211   public static <TState> void testRecoveryAndDoubleExecution(
212       final ProcedureExecutor<MasterProcedureEnv> procExec, final long procId,
213       final int numSteps, final TState[] states) throws Exception {
214     ProcedureTestingUtility.waitProcedure(procExec, procId);
215     assertEquals(false, procExec.isRunning());
216     for (int i = 0; i < numSteps; ++i) {
217       LOG.info("Restart "+ i +" exec state: " + states[i]);
218       ProcedureTestingUtility.assertProcNotYetCompleted(procExec, procId);
219       ProcedureTestingUtility.restart(procExec);
220       ProcedureTestingUtility.waitProcedure(procExec, procId);
221     }
222     assertEquals(true, procExec.isRunning());
223     ProcedureTestingUtility.assertProcNotFailed(procExec, procId);
224   }
225 
226   /**
227    * Run through all procedure flow states TWICE while also restarting procedure executor at each
228    * step; i.e force a reread of procedure store.
229    *
230    *<p>It does
231    * <ol><li>Execute step N - kill the executor before store update
232    * <li>Restart executor/store
233    * <li>Execute step N - and then save to store
234    * </ol>
235    *
236    *<p>This is a good test for finding state that needs persisting and steps that are not
237    * idempotent. Use this version of the test when the order in which flow steps are executed is
238    * not start to finish; where the procedure may vary the flow steps dependent on circumstance
239    * found.
240    * @see #testRecoveryAndDoubleExecution(ProcedureExecutor, long, int, Object[])
241    */
testRecoveryAndDoubleExecution( final ProcedureExecutor<MasterProcedureEnv> procExec, final long procId)242   public static <TState> void testRecoveryAndDoubleExecution(
243       final ProcedureExecutor<MasterProcedureEnv> procExec, final long procId)
244   throws Exception {
245     ProcedureTestingUtility.waitProcedure(procExec, procId);
246     assertEquals(false, procExec.isRunning());
247     while (!procExec.isFinished(procId)) {
248       ProcedureTestingUtility.restart(procExec);
249       ProcedureTestingUtility.waitProcedure(procExec, procId);
250     }
251     assertEquals(true, procExec.isRunning());
252     ProcedureTestingUtility.assertProcNotFailed(procExec, procId);
253   }
254 
testRollbackAndDoubleExecution( final ProcedureExecutor<MasterProcedureEnv> procExec, final long procId, final int lastStep, final TState[] states)255   public static <TState> void testRollbackAndDoubleExecution(
256       final ProcedureExecutor<MasterProcedureEnv> procExec, final long procId,
257       final int lastStep, final TState[] states) throws Exception {
258     ProcedureTestingUtility.waitProcedure(procExec, procId);
259 
260     // Restart the executor and execute the step twice
261     //   execute step N - kill before store update
262     //   restart executor/store
263     //   execute step N - save on store
264     for (int i = 0; i < lastStep; ++i) {
265       LOG.info("Restart "+ i +" exec state: " + states[i]);
266       ProcedureTestingUtility.assertProcNotYetCompleted(procExec, procId);
267       ProcedureTestingUtility.restart(procExec);
268       ProcedureTestingUtility.waitProcedure(procExec, procId);
269     }
270 
271     // Restart the executor and rollback the step twice
272     //   rollback step N - kill before store update
273     //   restart executor/store
274     //   rollback step N - save on store
275     MasterProcedureTestingUtility.InjectAbortOnLoadListener abortListener =
276       new MasterProcedureTestingUtility.InjectAbortOnLoadListener(procExec);
277     procExec.registerListener(abortListener);
278     try {
279       for (int i = lastStep + 1; i >= 0; --i) {
280         LOG.info("Restart " + i +" rollback state: "+ states[i]);
281         ProcedureTestingUtility.assertProcNotYetCompleted(procExec, procId);
282         ProcedureTestingUtility.restart(procExec);
283         ProcedureTestingUtility.waitProcedure(procExec, procId);
284       }
285     } finally {
286       assertTrue(procExec.unregisterListener(abortListener));
287     }
288 
289     ProcedureTestingUtility.assertIsAbortException(procExec.getResult(procId));
290   }
291 
testRollbackAndDoubleExecutionAfterPONR( final ProcedureExecutor<MasterProcedureEnv> procExec, final long procId, final int lastStep, final TState[] states)292   public static <TState> void testRollbackAndDoubleExecutionAfterPONR(
293       final ProcedureExecutor<MasterProcedureEnv> procExec, final long procId,
294       final int lastStep, final TState[] states) throws Exception {
295     ProcedureTestingUtility.waitProcedure(procExec, procId);
296 
297     // Restart the executor and execute the step twice
298     //   execute step N - kill before store update
299     //   restart executor/store
300     //   execute step N - save on store
301     for (int i = 0; i < lastStep; ++i) {
302       LOG.info("Restart "+ i +" exec state: " + states[i]);
303       ProcedureTestingUtility.assertProcNotYetCompleted(procExec, procId);
304       ProcedureTestingUtility.restart(procExec);
305       ProcedureTestingUtility.waitProcedure(procExec, procId);
306     }
307 
308     // try to inject the abort
309     ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(procExec, false);
310     MasterProcedureTestingUtility.InjectAbortOnLoadListener abortListener =
311       new MasterProcedureTestingUtility.InjectAbortOnLoadListener(procExec);
312     procExec.registerListener(abortListener);
313     try {
314       ProcedureTestingUtility.assertProcNotYetCompleted(procExec, procId);
315       ProcedureTestingUtility.restart(procExec);
316       LOG.info("Restart and execute");
317       ProcedureTestingUtility.waitProcedure(procExec, procId);
318     } finally {
319       assertTrue(procExec.unregisterListener(abortListener));
320     }
321 
322     assertEquals(true, procExec.isRunning());
323     ProcedureTestingUtility.assertProcNotFailed(procExec, procId);
324   }
325 
testRollbackRetriableFailure( final ProcedureExecutor<MasterProcedureEnv> procExec, final long procId, final int lastStep, final TState[] states)326   public static <TState> void testRollbackRetriableFailure(
327       final ProcedureExecutor<MasterProcedureEnv> procExec, final long procId,
328       final int lastStep, final TState[] states) throws Exception {
329     ProcedureTestingUtility.waitProcedure(procExec, procId);
330 
331     // Restart the executor and execute the step twice
332     //   execute step N - kill before store update
333     //   restart executor/store
334     //   execute step N - save on store
335     for (int i = 0; i < lastStep; ++i) {
336       LOG.info("Restart "+ i +" exec state: " + states[i]);
337       ProcedureTestingUtility.assertProcNotYetCompleted(procExec, procId);
338       ProcedureTestingUtility.restart(procExec);
339       ProcedureTestingUtility.waitProcedure(procExec, procId);
340     }
341 
342     // execute the rollback
343     ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(procExec, false);
344     MasterProcedureTestingUtility.InjectAbortOnLoadListener abortListener =
345       new MasterProcedureTestingUtility.InjectAbortOnLoadListener(procExec);
346     procExec.registerListener(abortListener);
347     try {
348       ProcedureTestingUtility.assertProcNotYetCompleted(procExec, procId);
349       ProcedureTestingUtility.restart(procExec);
350       LOG.info("Restart and rollback");
351       ProcedureTestingUtility.waitProcedure(procExec, procId);
352     } finally {
353       assertTrue(procExec.unregisterListener(abortListener));
354     }
355 
356     ProcedureTestingUtility.assertIsAbortException(procExec.getResult(procId));
357   }
358 
validateColumnFamilyAddition(final HMaster master, final TableName tableName, final String family)359   public static void validateColumnFamilyAddition(final HMaster master, final TableName tableName,
360       final String family) throws IOException {
361     HTableDescriptor htd = master.getTableDescriptors().get(tableName);
362     assertTrue(htd != null);
363     assertTrue(htd.hasFamily(family.getBytes()));
364   }
365 
validateColumnFamilyDeletion(final HMaster master, final TableName tableName, final String family)366   public static void validateColumnFamilyDeletion(final HMaster master, final TableName tableName,
367       final String family) throws IOException {
368     // verify htd
369     HTableDescriptor htd = master.getTableDescriptors().get(tableName);
370     assertTrue(htd != null);
371     assertFalse(htd.hasFamily(family.getBytes()));
372 
373     // verify fs
374     final FileSystem fs = master.getMasterFileSystem().getFileSystem();
375     final Path tableDir = FSUtils.getTableDir(master.getMasterFileSystem().getRootDir(), tableName);
376     for (Path regionDir: FSUtils.getRegionDirs(fs, tableDir)) {
377       final Path familyDir = new Path(regionDir, family);
378       assertFalse(family + " family dir should not exist", fs.exists(familyDir));
379     }
380   }
381 
validateColumnFamilyModification(final HMaster master, final TableName tableName, final String family, HColumnDescriptor columnDescriptor)382   public static void validateColumnFamilyModification(final HMaster master,
383       final TableName tableName, final String family, HColumnDescriptor columnDescriptor)
384       throws IOException {
385     HTableDescriptor htd = master.getTableDescriptors().get(tableName);
386     assertTrue(htd != null);
387 
388     HColumnDescriptor hcfd = htd.getFamily(family.getBytes());
389     assertTrue(hcfd.equals(columnDescriptor));
390   }
391 
loadData(final Connection connection, final TableName tableName, int rows, final byte[][] splitKeys, final String... sfamilies)392   public static void loadData(final Connection connection, final TableName tableName,
393       int rows, final byte[][] splitKeys,  final String... sfamilies) throws IOException {
394     byte[][] families = new byte[sfamilies.length][];
395     for (int i = 0; i < families.length; ++i) {
396       families[i] = Bytes.toBytes(sfamilies[i]);
397     }
398 
399     BufferedMutator mutator = connection.getBufferedMutator(tableName);
400 
401     // Ensure one row per region
402     assertTrue(rows >= splitKeys.length);
403     for (byte[] k: splitKeys) {
404       byte[] value = Bytes.add(Bytes.toBytes(System.currentTimeMillis()), k);
405       byte[] key = Bytes.add(k, Bytes.toBytes(MD5Hash.getMD5AsHex(value)));
406       mutator.mutate(createPut(families, key, value));
407       rows--;
408     }
409 
410     // Add other extra rows. more rows, more files
411     while (rows-- > 0) {
412       byte[] value = Bytes.add(Bytes.toBytes(System.currentTimeMillis()), Bytes.toBytes(rows));
413       byte[] key = Bytes.toBytes(MD5Hash.getMD5AsHex(value));
414       mutator.mutate(createPut(families, key, value));
415     }
416     mutator.flush();
417   }
418 
createPut(final byte[][] families, final byte[] key, final byte[] value)419   private static Put createPut(final byte[][] families, final byte[] key, final byte[] value) {
420     byte[] q = Bytes.toBytes("q");
421     Put put = new Put(key);
422     put.setDurability(Durability.SKIP_WAL);
423     for (byte[] family: families) {
424       put.add(family, q, value);
425     }
426     return put;
427   }
428 
generateNonceGroup(final HMaster master)429   public static long generateNonceGroup(final HMaster master) {
430     return master.getConnection().getNonceGenerator().getNonceGroup();
431   }
432 
generateNonce(final HMaster master)433   public static long generateNonce(final HMaster master) {
434     return master.getConnection().getNonceGenerator().newNonce();
435   }
436 
437   public static class InjectAbortOnLoadListener
438       implements ProcedureExecutor.ProcedureExecutorListener {
439     private final ProcedureExecutor<MasterProcedureEnv> procExec;
440 
InjectAbortOnLoadListener(final ProcedureExecutor<MasterProcedureEnv> procExec)441     public InjectAbortOnLoadListener(final ProcedureExecutor<MasterProcedureEnv> procExec) {
442       this.procExec = procExec;
443     }
444 
445     @Override
procedureLoaded(long procId)446     public void procedureLoaded(long procId) {
447       procExec.abort(procId);
448     }
449 
450     @Override
procedureAdded(long procId)451     public void procedureAdded(long procId) { /* no-op */ }
452 
453     @Override
procedureFinished(long procId)454     public void procedureFinished(long procId) { /* no-op */ }
455   }
456 }
457