001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.client;
019
020import static org.junit.Assert.assertEquals;
021import static org.junit.Assert.assertFalse;
022import static org.junit.Assert.assertNotNull;
023import static org.junit.Assert.assertThrows;
024import static org.junit.Assert.assertTrue;
025import static org.junit.Assert.fail;
026
027import java.io.IOException;
028import java.util.ArrayList;
029import java.util.List;
030import java.util.concurrent.ExecutionException;
031import java.util.concurrent.TimeUnit;
032import java.util.concurrent.atomic.AtomicInteger;
033import org.apache.hadoop.hbase.HBaseClassTestRule;
034import org.apache.hadoop.hbase.HConstants;
035import org.apache.hadoop.hbase.HRegionLocation;
036import org.apache.hadoop.hbase.MetaTableAccessor;
037import org.apache.hadoop.hbase.ServerName;
038import org.apache.hadoop.hbase.TableName;
039import org.apache.hadoop.hbase.TableNotFoundException;
040import org.apache.hadoop.hbase.exceptions.MergeRegionException;
041import org.apache.hadoop.hbase.master.HMaster;
042import org.apache.hadoop.hbase.master.janitor.CatalogJanitor;
043import org.apache.hadoop.hbase.regionserver.DisabledRegionSplitPolicy;
044import org.apache.hadoop.hbase.regionserver.HRegion;
045import org.apache.hadoop.hbase.regionserver.HStore;
046import org.apache.hadoop.hbase.regionserver.HStoreFile;
047import org.apache.hadoop.hbase.testclassification.ClientTests;
048import org.apache.hadoop.hbase.testclassification.LargeTests;
049import org.apache.hadoop.hbase.util.Bytes;
050import org.apache.hadoop.hbase.util.CommonFSUtils;
051import org.apache.hadoop.hbase.util.FutureUtils;
052import org.apache.hadoop.hbase.util.Pair;
053import org.apache.hadoop.hbase.util.Threads;
054import org.junit.ClassRule;
055import org.junit.Test;
056import org.junit.experimental.categories.Category;
057import org.slf4j.Logger;
058import org.slf4j.LoggerFactory;
059
060import org.apache.hadoop.hbase.shaded.protobuf.RequestConverter;
061import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.MergeTableRegionsRequest;
062
063/**
064 * Class to test HBaseAdmin. Spins up the minicluster once at test start and then takes it down
065 * afterward. Add any testing of HBaseAdmin functionality here.
066 */
067@Category({ LargeTests.class, ClientTests.class })
068public class TestAdmin1 extends TestAdminBase {
069
070  @ClassRule
071  public static final HBaseClassTestRule CLASS_RULE = HBaseClassTestRule.forClass(TestAdmin1.class);
072
073  private static final Logger LOG = LoggerFactory.getLogger(TestAdmin1.class);
074
075  @Test
076  public void testSplitFlushCompactUnknownTable() throws InterruptedException {
077    final TableName unknowntable = TableName.valueOf(name.getMethodName());
078    Exception exception = null;
079    try {
080      ADMIN.compact(unknowntable);
081    } catch (IOException e) {
082      exception = e;
083    }
084    assertTrue(exception instanceof TableNotFoundException);
085
086    exception = null;
087    try {
088      ADMIN.flush(unknowntable);
089    } catch (IOException e) {
090      exception = e;
091    }
092    assertTrue(exception instanceof TableNotFoundException);
093
094    exception = null;
095    try {
096      ADMIN.split(unknowntable);
097    } catch (IOException e) {
098      exception = e;
099    }
100    assertTrue(exception instanceof TableNotFoundException);
101  }
102
103  @Test
104  public void testCompactATableWithSuperLongTableName() throws Exception {
105    TableName tableName = TableName.valueOf(name.getMethodName());
106    TableDescriptor htd = TableDescriptorBuilder.newBuilder(tableName)
107      .setColumnFamily(ColumnFamilyDescriptorBuilder.of("fam1")).build();
108    try {
109      ADMIN.createTable(htd);
110      assertThrows(IllegalArgumentException.class,
111        () -> ADMIN.majorCompactRegion(tableName.getName()));
112
113      assertThrows(IllegalArgumentException.class,
114        () -> ADMIN.majorCompactRegion(Bytes.toBytes("abcd")));
115    } finally {
116      ADMIN.disableTable(tableName);
117      ADMIN.deleteTable(tableName);
118    }
119  }
120
121  @Test
122  public void testCompactionTimestamps() throws Exception {
123    TableName tableName = TableName.valueOf(name.getMethodName());
124    TableDescriptor htd = TableDescriptorBuilder.newBuilder(tableName)
125      .setColumnFamily(ColumnFamilyDescriptorBuilder.of("fam1")).build();
126    ADMIN.createTable(htd);
127    Table table = TEST_UTIL.getConnection().getTable(htd.getTableName());
128    long ts = ADMIN.getLastMajorCompactionTimestamp(tableName);
129    assertEquals(0, ts);
130    Put p = new Put(Bytes.toBytes("row1"));
131    p.addColumn(Bytes.toBytes("fam1"), Bytes.toBytes("fam1"), Bytes.toBytes("fam1"));
132    table.put(p);
133    ts = ADMIN.getLastMajorCompactionTimestamp(tableName);
134    // no files written -> no data
135    assertEquals(0, ts);
136
137    ADMIN.flush(tableName);
138    ts = ADMIN.getLastMajorCompactionTimestamp(tableName);
139    // still 0, we flushed a file, but no major compaction happened
140    assertEquals(0, ts);
141
142    byte[] regionName;
143    try (RegionLocator l = TEST_UTIL.getConnection().getRegionLocator(tableName)) {
144      regionName = l.getAllRegionLocations().get(0).getRegion().getRegionName();
145    }
146    long ts1 = ADMIN.getLastMajorCompactionTimestampForRegion(regionName);
147    assertEquals(ts, ts1);
148    p = new Put(Bytes.toBytes("row2"));
149    p.addColumn(Bytes.toBytes("fam1"), Bytes.toBytes("fam1"), Bytes.toBytes("fam1"));
150    table.put(p);
151    ADMIN.flush(tableName);
152    ts = ADMIN.getLastMajorCompactionTimestamp(tableName);
153    // make sure the region API returns the same value, as the old file is still around
154    assertEquals(ts1, ts);
155
156    TEST_UTIL.compact(tableName, true);
157    table.put(p);
158    // forces a wait for the compaction
159    ADMIN.flush(tableName);
160    ts = ADMIN.getLastMajorCompactionTimestamp(tableName);
161    // after a compaction our earliest timestamp will have progressed forward
162    assertTrue(ts > ts1);
163
164    // region api still the same
165    ts1 = ADMIN.getLastMajorCompactionTimestampForRegion(regionName);
166    assertEquals(ts, ts1);
167    table.put(p);
168    ADMIN.flush(tableName);
169    ts = ADMIN.getLastMajorCompactionTimestamp(tableName);
170    assertEquals(ts, ts1);
171    table.close();
172  }
173
174  @Test(expected = IllegalArgumentException.class)
175  public void testColumnValidName() {
176    ColumnFamilyDescriptorBuilder.of("\\test\\abc");
177  }
178
179  @Test
180  public void testTableExist() throws IOException {
181    final TableName table = TableName.valueOf(name.getMethodName());
182    boolean exist;
183    exist = ADMIN.tableExists(table);
184    assertEquals(false, exist);
185    TEST_UTIL.createTable(table, HConstants.CATALOG_FAMILY);
186    exist = ADMIN.tableExists(table);
187    assertEquals(true, exist);
188  }
189
190  /**
191   * Tests forcing split from client and having scanners successfully ride over split.
192   */
193  @Test
194  public void testForceSplit() throws Exception {
195    byte[][] familyNames = new byte[][] { Bytes.toBytes("cf") };
196    int[] rowCounts = new int[] { 6000 };
197    int numVersions = ColumnFamilyDescriptorBuilder.DEFAULT_MAX_VERSIONS;
198    int blockSize = 256;
199    splitTest(null, familyNames, rowCounts, numVersions, blockSize, true);
200
201    byte[] splitKey = Bytes.toBytes(3500);
202    splitTest(splitKey, familyNames, rowCounts, numVersions, blockSize, true);
203    // test regionSplitSync
204    splitTest(splitKey, familyNames, rowCounts, numVersions, blockSize, false);
205  }
206
207  /**
208   * Multi-family scenario. Tests forcing split from client and having scanners successfully ride
209   * over split.
210   */
211  @Test
212  public void testForceSplitMultiFamily() throws Exception {
213    int numVersions = ColumnFamilyDescriptorBuilder.DEFAULT_MAX_VERSIONS;
214
215    // use small HFile block size so that we can have lots of blocks in HFile
216    // Otherwise, if there is only one block,
217    // HFileBlockIndex.midKey()'s value == startKey
218    int blockSize = 256;
219    byte[][] familyNames = new byte[][] { Bytes.toBytes("cf1"), Bytes.toBytes("cf2") };
220
221    // one of the column families isn't splittable
222    int[] rowCounts = new int[] { 6000, 1 };
223    splitTest(null, familyNames, rowCounts, numVersions, blockSize, true);
224
225    rowCounts = new int[] { 1, 6000 };
226    splitTest(null, familyNames, rowCounts, numVersions, blockSize, true);
227
228    // one column family has much smaller data than the other
229    // the split key should be based on the largest column family
230    rowCounts = new int[] { 6000, 300 };
231    splitTest(null, familyNames, rowCounts, numVersions, blockSize, true);
232
233    rowCounts = new int[] { 300, 6000 };
234    splitTest(null, familyNames, rowCounts, numVersions, blockSize, true);
235  }
236
237  private int count(ResultScanner scanner) throws IOException {
238    int rows = 0;
239    while (scanner.next() != null) {
240      rows++;
241    }
242    return rows;
243  }
244
245  private void splitTest(byte[] splitPoint, byte[][] familyNames, int[] rowCounts, int numVersions,
246    int blockSize, boolean async) throws Exception {
247    TableName tableName = TableName.valueOf("testForceSplit");
248    StringBuilder sb = new StringBuilder();
249    // Add tail to String so can see better in logs where a test is running.
250    for (int i = 0; i < rowCounts.length; i++) {
251      sb.append("_").append(Integer.toString(rowCounts[i]));
252    }
253    assertFalse(ADMIN.tableExists(tableName));
254    try (final Table table = TEST_UTIL.createTable(tableName, familyNames, numVersions, blockSize);
255      final RegionLocator locator = TEST_UTIL.getConnection().getRegionLocator(tableName)) {
256
257      int rowCount = 0;
258      byte[] q = new byte[0];
259
260      // insert rows into column families. The number of rows that have values
261      // in a specific column family is decided by rowCounts[familyIndex]
262      for (int index = 0; index < familyNames.length; index++) {
263        ArrayList<Put> puts = new ArrayList<>(rowCounts[index]);
264        for (int i = 0; i < rowCounts[index]; i++) {
265          byte[] k = Bytes.toBytes(i);
266          Put put = new Put(k);
267          put.addColumn(familyNames[index], q, k);
268          puts.add(put);
269        }
270        table.put(puts);
271
272        if (rowCount < rowCounts[index]) {
273          rowCount = rowCounts[index];
274        }
275      }
276
277      // get the initial layout (should just be one region)
278      List<HRegionLocation> m = locator.getAllRegionLocations();
279      LOG.info("Initial regions (" + m.size() + "): " + m);
280      assertTrue(m.size() == 1);
281
282      // Verify row count
283      Scan scan = new Scan();
284      int rows;
285      try (ResultScanner scanner = table.getScanner(scan)) {
286        rows = count(scanner);
287      }
288      assertEquals(rowCount, rows);
289
290      // Have an outstanding scan going on to make sure we can scan over splits.
291      scan = new Scan();
292      try (ResultScanner scanner = table.getScanner(scan)) {
293        // Scan first row so we are into first region before split happens.
294        scanner.next();
295
296        // Split the table
297        if (async) {
298          if (splitPoint != null) {
299            ADMIN.split(tableName, splitPoint);
300          } else {
301            ADMIN.split(tableName);
302          }
303          final AtomicInteger count = new AtomicInteger(0);
304          Thread t = new Thread("CheckForSplit") {
305            @Override
306            public void run() {
307              for (int i = 0; i < 45; i++) {
308                try {
309                  sleep(1000);
310                } catch (InterruptedException e) {
311                  continue;
312                }
313                // check again
314                List<HRegionLocation> regions = null;
315                try {
316                  regions = locator.getAllRegionLocations();
317                } catch (IOException e) {
318                  LOG.warn("get location failed", e);
319                }
320                if (regions == null) {
321                  continue;
322                }
323                count.set(regions.size());
324                if (count.get() >= 2) {
325                  LOG.info("Found: " + regions);
326                  break;
327                }
328                LOG.debug("Cycle waiting on split");
329              }
330              LOG.debug("CheckForSplit thread exited, current region count: " + count.get());
331            }
332          };
333          t.setPriority(Thread.NORM_PRIORITY - 2);
334          t.start();
335          t.join();
336        } else {
337          // Sync split region, no need to create a thread to check
338          ADMIN.splitRegionAsync(m.get(0).getRegion().getRegionName(), splitPoint).get();
339        }
340        // Verify row count
341        rows = 1 + count(scanner); // We counted one row above.
342      }
343      assertEquals(rowCount, rows);
344
345      List<HRegionLocation> regions = null;
346      try {
347        regions = locator.getAllRegionLocations();
348      } catch (IOException e) {
349        e.printStackTrace();
350      }
351      assertEquals(2, regions.size());
352      if (splitPoint != null) {
353        // make sure the split point matches our explicit configuration
354        assertEquals(Bytes.toString(splitPoint),
355          Bytes.toString(regions.get(0).getRegion().getEndKey()));
356        assertEquals(Bytes.toString(splitPoint),
357          Bytes.toString(regions.get(1).getRegion().getStartKey()));
358        LOG.debug("Properly split on " + Bytes.toString(splitPoint));
359      } else {
360        if (familyNames.length > 1) {
361          int splitKey = Bytes.toInt(regions.get(0).getRegion().getEndKey());
362          // check if splitKey is based on the largest column family
363          // in terms of it store size
364          int deltaForLargestFamily = Math.abs(rowCount / 2 - splitKey);
365          LOG.debug("SplitKey=" + splitKey + "&deltaForLargestFamily=" + deltaForLargestFamily
366            + ", r=" + regions.get(0).getRegion());
367          for (int index = 0; index < familyNames.length; index++) {
368            int delta = Math.abs(rowCounts[index] / 2 - splitKey);
369            if (delta < deltaForLargestFamily) {
370              assertTrue("Delta " + delta + " for family " + index + " should be at least "
371                + "deltaForLargestFamily " + deltaForLargestFamily, false);
372            }
373          }
374        }
375      }
376      TEST_UTIL.deleteTable(tableName);
377    }
378  }
379
380  @Test
381  public void testSplitAndMergeWithReplicaTable() throws Exception {
382    // The test tries to directly split replica regions and directly merge replica regions. These
383    // are not allowed. The test validates that. Then the test does a valid split/merge of allowed
384    // regions.
385    // Set up a table with 3 regions and replication set to 3
386    TableName tableName = TableName.valueOf(name.getMethodName());
387    byte[] cf = Bytes.toBytes("f");
388    TableDescriptor desc = TableDescriptorBuilder.newBuilder(tableName).setRegionReplication(3)
389      .setColumnFamily(ColumnFamilyDescriptorBuilder.of(cf)).build();
390    byte[][] splitRows = new byte[2][];
391    splitRows[0] = new byte[] { (byte) '4' };
392    splitRows[1] = new byte[] { (byte) '7' };
393    TEST_UTIL.getAdmin().createTable(desc, splitRows);
394    List<HRegion> oldRegions;
395    do {
396      oldRegions = TEST_UTIL.getHBaseCluster().getRegions(tableName);
397      Thread.sleep(10);
398    } while (oldRegions.size() != 9); // 3 regions * 3 replicas
399    // write some data to the table
400    Table ht = TEST_UTIL.getConnection().getTable(tableName);
401    List<Put> puts = new ArrayList<>();
402    byte[] qualifier = Bytes.toBytes("c");
403    Put put = new Put(new byte[] { (byte) '1' });
404    put.addColumn(cf, qualifier, Bytes.toBytes("100"));
405    puts.add(put);
406    put = new Put(new byte[] { (byte) '6' });
407    put.addColumn(cf, qualifier, Bytes.toBytes("100"));
408    puts.add(put);
409    put = new Put(new byte[] { (byte) '8' });
410    put.addColumn(cf, qualifier, Bytes.toBytes("100"));
411    puts.add(put);
412    ht.put(puts);
413    ht.close();
414    List<Pair<RegionInfo, ServerName>> regions =
415      MetaTableAccessor.getTableRegionsAndLocations(TEST_UTIL.getConnection(), tableName);
416    boolean gotException = false;
417    // the element at index 1 would be a replica (since the metareader gives us ordered
418    // regions). Try splitting that region via the split API . Should fail
419    try {
420      FutureUtils
421        .get(TEST_UTIL.getAdmin().splitRegionAsync(regions.get(1).getFirst().getRegionName()));
422    } catch (IllegalArgumentException ex) {
423      gotException = true;
424    }
425    assertTrue(gotException);
426    gotException = false;
427    // the element at index 1 would be a replica (since the metareader gives us ordered
428    // regions). Try splitting that region via a different split API (the difference is
429    // this API goes direct to the regionserver skipping any checks in the admin). Should fail
430    try {
431      FutureUtils.get(TEST_UTIL.getAdmin().splitRegionAsync(
432        regions.get(1).getFirst().getEncodedNameAsBytes(), new byte[] { (byte) '1' }));
433    } catch (IllegalArgumentException ex) {
434      gotException = true;
435    }
436    assertTrue(gotException);
437
438    gotException = false;
439    // testing Sync split operation
440    try {
441      FutureUtils.get(TEST_UTIL.getAdmin()
442        .splitRegionAsync(regions.get(1).getFirst().getRegionName(), new byte[] { (byte) '1' }));
443    } catch (IllegalArgumentException ex) {
444      gotException = true;
445    }
446    assertTrue(gotException);
447
448    gotException = false;
449    // Try merging a replica with another. Should fail.
450    try {
451      FutureUtils.get(
452        TEST_UTIL.getAdmin().mergeRegionsAsync(regions.get(1).getFirst().getEncodedNameAsBytes(),
453          regions.get(2).getFirst().getEncodedNameAsBytes(), true));
454    } catch (IllegalArgumentException m) {
455      gotException = true;
456    }
457    assertTrue(gotException);
458    // Try going to the master directly (that will skip the check in admin)
459    try {
460      byte[][] nameofRegionsToMerge = new byte[2][];
461      nameofRegionsToMerge[0] = regions.get(1).getFirst().getEncodedNameAsBytes();
462      nameofRegionsToMerge[1] = regions.get(2).getFirst().getEncodedNameAsBytes();
463      MergeTableRegionsRequest request = RequestConverter.buildMergeTableRegionsRequest(
464        nameofRegionsToMerge, true, HConstants.NO_NONCE, HConstants.NO_NONCE);
465      TEST_UTIL.getMiniHBaseCluster().getMaster().getMasterRpcServices().mergeTableRegions(null,
466        request);
467    } catch (org.apache.hbase.thirdparty.com.google.protobuf.ServiceException m) {
468      Throwable t = m.getCause();
469      do {
470        if (t instanceof MergeRegionException) {
471          gotException = true;
472          break;
473        }
474        t = t.getCause();
475      } while (t != null);
476    }
477    assertTrue(gotException);
478  }
479
480  @Test(expected = IllegalArgumentException.class)
481  public void testInvalidColumnDescriptor() throws IOException {
482    ColumnFamilyDescriptorBuilder.of("/cfamily/name");
483  }
484
485  /**
486   * Test DFS replication for column families, where one CF has default replication(3) and the other
487   * is set to 1.
488   */
489  @Test
490  public void testHFileReplication() throws Exception {
491    final TableName tableName = TableName.valueOf(this.name.getMethodName());
492    String fn1 = "rep1";
493    String fn = "defaultRep";
494    TableDescriptor htd = TableDescriptorBuilder.newBuilder(tableName)
495      .setColumnFamily(ColumnFamilyDescriptorBuilder.of(fn))
496      .setColumnFamily(ColumnFamilyDescriptorBuilder.newBuilder(Bytes.toBytes(fn1))
497        .setDFSReplication((short) 1).build())
498      .build();
499    Table table = TEST_UTIL.createTable(htd, null);
500    TEST_UTIL.waitTableAvailable(tableName);
501    Put p = new Put(Bytes.toBytes("defaultRep_rk"));
502    byte[] q1 = Bytes.toBytes("q1");
503    byte[] v1 = Bytes.toBytes("v1");
504    p.addColumn(Bytes.toBytes(fn), q1, v1);
505    List<Put> puts = new ArrayList<>(2);
506    puts.add(p);
507    p = new Put(Bytes.toBytes("rep1_rk"));
508    p.addColumn(Bytes.toBytes(fn1), q1, v1);
509    puts.add(p);
510    try {
511      table.put(puts);
512      ADMIN.flush(tableName);
513
514      List<HRegion> regions = TEST_UTIL.getMiniHBaseCluster().getRegions(tableName);
515      for (HRegion r : regions) {
516        HStore store = r.getStore(Bytes.toBytes(fn));
517        for (HStoreFile sf : store.getStorefiles()) {
518          assertTrue(sf.toString().contains(fn));
519          assertTrue("Column family " + fn + " should have 3 copies",
520            CommonFSUtils.getDefaultReplication(TEST_UTIL.getTestFileSystem(), sf.getPath())
521                == (sf.getFileInfo().getFileStatus().getReplication()));
522        }
523
524        store = r.getStore(Bytes.toBytes(fn1));
525        for (HStoreFile sf : store.getStorefiles()) {
526          assertTrue(sf.toString().contains(fn1));
527          assertTrue("Column family " + fn1 + " should have only 1 copy",
528            1 == sf.getFileInfo().getFileStatus().getReplication());
529        }
530      }
531    } finally {
532      if (ADMIN.isTableEnabled(tableName)) {
533        ADMIN.disableTable(tableName);
534        ADMIN.deleteTable(tableName);
535      }
536    }
537  }
538
539  @Test
540  public void testMergeRegions() throws Exception {
541    final TableName tableName = TableName.valueOf(name.getMethodName());
542    TableDescriptor td = TableDescriptorBuilder.newBuilder(tableName)
543      .setColumnFamily(ColumnFamilyDescriptorBuilder.of("d")).build();
544    byte[][] splitRows = new byte[2][];
545    splitRows[0] = new byte[] { (byte) '3' };
546    splitRows[1] = new byte[] { (byte) '6' };
547    try {
548      TEST_UTIL.createTable(td, splitRows);
549      TEST_UTIL.waitTableAvailable(tableName);
550
551      List<RegionInfo> tableRegions;
552      RegionInfo regionA;
553      RegionInfo regionB;
554      RegionInfo regionC;
555      RegionInfo mergedChildRegion = null;
556
557      // merge with full name
558      tableRegions = ADMIN.getRegions(tableName);
559      assertEquals(3, ADMIN.getRegions(tableName).size());
560      regionA = tableRegions.get(0);
561      regionB = tableRegions.get(1);
562      regionC = tableRegions.get(2);
563      // TODO convert this to version that is synchronous (See HBASE-16668)
564      ADMIN.mergeRegionsAsync(regionA.getRegionName(), regionB.getRegionName(), false).get(60,
565        TimeUnit.SECONDS);
566
567      tableRegions = ADMIN.getRegions(tableName);
568
569      assertEquals(2, tableRegions.size());
570      for (RegionInfo ri : tableRegions) {
571        if (regionC.compareTo(ri) != 0) {
572          mergedChildRegion = ri;
573          break;
574        }
575      }
576
577      assertNotNull(mergedChildRegion);
578      // Need to wait GC for merged child region is done.
579      HMaster services = TEST_UTIL.getHBaseCluster().getMaster();
580      CatalogJanitor cj = services.getCatalogJanitor();
581      assertTrue(cj.scan() > 0);
582      // Wait until all procedures settled down
583      while (!services.getMasterProcedureExecutor().getActiveProcIds().isEmpty()) {
584        Thread.sleep(200);
585      }
586
587      // TODO convert this to version that is synchronous (See HBASE-16668)
588      ADMIN.mergeRegionsAsync(regionC.getEncodedNameAsBytes(),
589        mergedChildRegion.getEncodedNameAsBytes(), false).get(60, TimeUnit.SECONDS);
590
591      assertEquals(1, ADMIN.getRegions(tableName).size());
592    } finally {
593      ADMIN.disableTable(tableName);
594      ADMIN.deleteTable(tableName);
595    }
596  }
597
598  @Test
599  public void testMergeRegionsInvalidRegionCount()
600    throws IOException, InterruptedException, ExecutionException {
601    TableName tableName = TableName.valueOf(name.getMethodName());
602    TableDescriptor td = TableDescriptorBuilder.newBuilder(tableName)
603      .setColumnFamily(ColumnFamilyDescriptorBuilder.of("d")).build();
604    byte[][] splitRows = new byte[2][];
605    splitRows[0] = new byte[] { (byte) '3' };
606    splitRows[1] = new byte[] { (byte) '6' };
607    try {
608      TEST_UTIL.createTable(td, splitRows);
609      TEST_UTIL.waitTableAvailable(tableName);
610
611      List<RegionInfo> tableRegions = ADMIN.getRegions(tableName);
612      // 0
613      try {
614        FutureUtils.get(ADMIN.mergeRegionsAsync(new byte[0][0], false));
615        fail();
616      } catch (IllegalArgumentException e) {
617        // expected
618      }
619      // 1
620      try {
621        FutureUtils.get(ADMIN
622          .mergeRegionsAsync(new byte[][] { tableRegions.get(0).getEncodedNameAsBytes() }, false));
623        fail();
624      } catch (IllegalArgumentException e) {
625        // expected
626      }
627    } finally {
628      ADMIN.disableTable(tableName);
629      ADMIN.deleteTable(tableName);
630    }
631  }
632
633  @Test
634  public void testSplitShouldNotHappenIfSplitIsDisabledForTable() throws Exception {
635    final TableName tableName = TableName.valueOf(name.getMethodName());
636    TableDescriptor htd = TableDescriptorBuilder.newBuilder(tableName)
637      .setRegionSplitPolicyClassName(DisabledRegionSplitPolicy.class.getName())
638      .setColumnFamily(ColumnFamilyDescriptorBuilder.of("f")).build();
639    Table table = TEST_UTIL.createTable(htd, null);
640    for (int i = 0; i < 10; i++) {
641      Put p = new Put(Bytes.toBytes("row" + i));
642      byte[] q1 = Bytes.toBytes("q1");
643      byte[] v1 = Bytes.toBytes("v1");
644      p.addColumn(Bytes.toBytes("f"), q1, v1);
645      table.put(p);
646    }
647    ADMIN.flush(tableName);
648    try {
649      ADMIN.split(tableName, Bytes.toBytes("row5"));
650      Threads.sleep(10000);
651    } catch (Exception e) {
652      // Nothing to do.
653    }
654    // Split should not happen.
655    List<RegionInfo> allRegions =
656      MetaTableAccessor.getTableRegions(ADMIN.getConnection(), tableName, true);
657    assertEquals(1, allRegions.size());
658  }
659}