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.master.normalizer;
019
020import static org.hamcrest.Matchers.comparesEqualTo;
021import static org.hamcrest.Matchers.greaterThanOrEqualTo;
022import static org.hamcrest.Matchers.lessThanOrEqualTo;
023import static org.hamcrest.Matchers.not;
024import static org.junit.Assert.assertEquals;
025import static org.junit.Assert.assertFalse;
026import static org.junit.Assert.assertNotNull;
027import static org.junit.Assert.assertTrue;
028
029import java.io.IOException;
030import java.util.ArrayList;
031import java.util.Collections;
032import java.util.Comparator;
033import java.util.List;
034import java.util.concurrent.TimeUnit;
035import org.apache.hadoop.hbase.HBaseClassTestRule;
036import org.apache.hadoop.hbase.HBaseTestingUtil;
037import org.apache.hadoop.hbase.HConstants;
038import org.apache.hadoop.hbase.MatcherPredicate;
039import org.apache.hadoop.hbase.NamespaceDescriptor;
040import org.apache.hadoop.hbase.RegionMetrics;
041import org.apache.hadoop.hbase.ServerName;
042import org.apache.hadoop.hbase.Size;
043import org.apache.hadoop.hbase.TableName;
044import org.apache.hadoop.hbase.Waiter.ExplainingPredicate;
045import org.apache.hadoop.hbase.client.AsyncAdmin;
046import org.apache.hadoop.hbase.client.NormalizeTableFilterParams;
047import org.apache.hadoop.hbase.client.Put;
048import org.apache.hadoop.hbase.client.RegionInfo;
049import org.apache.hadoop.hbase.client.RegionLocator;
050import org.apache.hadoop.hbase.client.Table;
051import org.apache.hadoop.hbase.client.TableDescriptor;
052import org.apache.hadoop.hbase.client.TableDescriptorBuilder;
053import org.apache.hadoop.hbase.master.HMaster;
054import org.apache.hadoop.hbase.master.MasterServices;
055import org.apache.hadoop.hbase.master.TableNamespaceManager;
056import org.apache.hadoop.hbase.master.normalizer.NormalizationPlan.PlanType;
057import org.apache.hadoop.hbase.namespace.TestNamespaceAuditor;
058import org.apache.hadoop.hbase.quotas.QuotaUtil;
059import org.apache.hadoop.hbase.regionserver.HRegion;
060import org.apache.hadoop.hbase.regionserver.Region;
061import org.apache.hadoop.hbase.testclassification.MasterTests;
062import org.apache.hadoop.hbase.testclassification.MediumTests;
063import org.apache.hadoop.hbase.util.Bytes;
064import org.apache.hadoop.hbase.util.LoadTestKVGenerator;
065import org.hamcrest.Matcher;
066import org.hamcrest.Matchers;
067import org.junit.AfterClass;
068import org.junit.Before;
069import org.junit.BeforeClass;
070import org.junit.ClassRule;
071import org.junit.Rule;
072import org.junit.Test;
073import org.junit.experimental.categories.Category;
074import org.junit.rules.TestName;
075import org.slf4j.Logger;
076import org.slf4j.LoggerFactory;
077
078/**
079 * Testing {@link SimpleRegionNormalizer} on minicluster.
080 */
081@Category({ MasterTests.class, MediumTests.class })
082public class TestSimpleRegionNormalizerOnCluster {
083  private static final Logger LOG =
084    LoggerFactory.getLogger(TestSimpleRegionNormalizerOnCluster.class);
085
086  @ClassRule
087  public static final HBaseClassTestRule CLASS_RULE =
088    HBaseClassTestRule.forClass(TestSimpleRegionNormalizerOnCluster.class);
089
090  private static final HBaseTestingUtil TEST_UTIL = new HBaseTestingUtil();
091  private static final byte[] FAMILY_NAME = Bytes.toBytes("fam");
092
093  private static AsyncAdmin admin;
094  private static HMaster master;
095
096  @Rule
097  public TestName name = new TestName();
098
099  @BeforeClass
100  public static void beforeAllTests() throws Exception {
101    // we will retry operations when PleaseHoldException is thrown
102    TEST_UTIL.getConfiguration().setInt(HConstants.HBASE_CLIENT_RETRIES_NUMBER, 3);
103    TEST_UTIL.getConfiguration().setBoolean(QuotaUtil.QUOTA_CONF_KEY, true);
104
105    // no way for the test to set the regionId on a created region, so disable this feature.
106    TEST_UTIL.getConfiguration().setInt("hbase.normalizer.merge.min_region_age.days", 0);
107
108    // disable the normalizer coming along and running via Chore
109    TEST_UTIL.getConfiguration().setInt("hbase.normalizer.period", Integer.MAX_VALUE);
110
111    TEST_UTIL.startMiniCluster(1);
112    TestNamespaceAuditor.waitForQuotaInitialize(TEST_UTIL);
113    admin = TEST_UTIL.getAsyncConnection().getAdmin();
114    master = TEST_UTIL.getHBaseCluster().getMaster();
115    assertNotNull(master);
116  }
117
118  @AfterClass
119  public static void afterAllTests() throws Exception {
120    TEST_UTIL.shutdownMiniCluster();
121  }
122
123  @Before
124  public void before() throws Exception {
125    // disable the normalizer ahead of time, let the test enable it when its ready.
126    admin.normalizerSwitch(false).get();
127  }
128
129  @Test
130  public void testHonorsNormalizerSwitch() throws Exception {
131    assertFalse(admin.isNormalizerEnabled().get());
132    assertFalse(admin.normalize().get());
133    assertFalse(admin.normalizerSwitch(true).get());
134    assertTrue(admin.normalize().get());
135  }
136
137  /**
138   * Test that disabling normalizer via table configuration is honored. There's no side-effect to
139   * look for (other than a log message), so normalize two tables, one with the disabled setting,
140   * and look for change in one and no change in the other.
141   */
142  @Test
143  public void testHonorsNormalizerTableSetting() throws Exception {
144    final TableName tn1 = TableName.valueOf(name.getMethodName() + "1");
145    final TableName tn2 = TableName.valueOf(name.getMethodName() + "2");
146    final TableName tn3 = TableName.valueOf(name.getMethodName() + "3");
147
148    try {
149      final int tn1RegionCount = createTableBegsSplit(tn1, true, false);
150      final int tn2RegionCount = createTableBegsSplit(tn2, false, false);
151      final int tn3RegionCount = createTableBegsSplit(tn3, true, true);
152
153      assertFalse(admin.normalizerSwitch(true).get());
154      assertTrue(admin.normalize().get());
155      waitForTableRegionCount(tn1, greaterThanOrEqualTo(tn1RegionCount + 1));
156
157      // confirm that tn1 has (tn1RegionCount + 1) number of regions.
158      // tn2 has tn2RegionCount number of regions because normalizer has not been enabled on it.
159      // tn3 has tn3RegionCount number of regions because two plans are run:
160      // 1. split one region to two
161      // 2. merge two regions into one
162      // and hence, total number of regions for tn3 remains same
163      assertEquals(tn1 + " should have split.", tn1RegionCount + 1, getRegionCount(tn1));
164      assertEquals(tn2 + " should not have split.", tn2RegionCount, getRegionCount(tn2));
165      LOG.debug("waiting for t3 to settle...");
166      waitForTableRegionCount(tn3, comparesEqualTo(tn3RegionCount));
167    } finally {
168      dropIfExists(tn1);
169      dropIfExists(tn2);
170      dropIfExists(tn3);
171    }
172  }
173
174  @Test
175  public void testRegionNormalizationSplitWithoutQuotaLimit() throws Exception {
176    testRegionNormalizationSplit(false);
177  }
178
179  @Test
180  public void testRegionNormalizationSplitWithQuotaLimit() throws Exception {
181    testRegionNormalizationSplit(true);
182  }
183
184  void testRegionNormalizationSplit(boolean limitedByQuota) throws Exception {
185    TableName tableName = null;
186    try {
187      tableName = limitedByQuota
188        ? buildTableNameForQuotaTest(name.getMethodName())
189        : TableName.valueOf(name.getMethodName());
190
191      final int currentRegionCount = createTableBegsSplit(tableName, true, false);
192      final long existingSkippedSplitCount =
193        master.getRegionNormalizerManager().getSkippedCount(PlanType.SPLIT);
194      assertFalse(admin.normalizerSwitch(true).get());
195      assertTrue(admin.normalize().get());
196      if (limitedByQuota) {
197        waitForSkippedSplits(master, existingSkippedSplitCount);
198        assertEquals(tableName + " should not have split.", currentRegionCount,
199          getRegionCount(tableName));
200      } else {
201        waitForTableRegionCount(tableName, greaterThanOrEqualTo(currentRegionCount + 1));
202        assertEquals(tableName + " should have split.", currentRegionCount + 1,
203          getRegionCount(tableName));
204      }
205    } finally {
206      dropIfExists(tableName);
207    }
208  }
209
210  @Test
211  public void testRegionNormalizationMerge() throws Exception {
212    final TableName tableName = TableName.valueOf(name.getMethodName());
213    try {
214      final int currentRegionCount = createTableBegsMerge(tableName);
215      assertFalse(admin.normalizerSwitch(true).get());
216      assertTrue(admin.normalize().get());
217      waitForTableRegionCount(tableName, lessThanOrEqualTo(currentRegionCount - 1));
218      assertEquals(tableName + " should have merged.", currentRegionCount - 1,
219        getRegionCount(tableName));
220    } finally {
221      dropIfExists(tableName);
222    }
223  }
224
225  @Test
226  public void testHonorsNamespaceFilter() throws Exception {
227    final NamespaceDescriptor namespaceDescriptor = NamespaceDescriptor.create("ns").build();
228    final TableName tn1 = TableName.valueOf("ns", name.getMethodName());
229    final TableName tn2 = TableName.valueOf(name.getMethodName());
230
231    try {
232      admin.createNamespace(namespaceDescriptor).get();
233      final int tn1RegionCount = createTableBegsSplit(tn1, true, false);
234      final int tn2RegionCount = createTableBegsSplit(tn2, true, false);
235      final NormalizeTableFilterParams ntfp =
236        new NormalizeTableFilterParams.Builder().namespace("ns").build();
237
238      assertFalse(admin.normalizerSwitch(true).get());
239      assertTrue(admin.normalize(ntfp).get());
240      waitForTableRegionCount(tn1, greaterThanOrEqualTo(tn1RegionCount + 1));
241
242      // confirm that tn1 has (tn1RegionCount + 1) number of regions.
243      // tn2 has tn2RegionCount number of regions because it's not a member of the target namespace.
244      assertEquals(tn1 + " should have split.", tn1RegionCount + 1, getRegionCount(tn1));
245      waitForTableRegionCount(tn2, comparesEqualTo(tn2RegionCount));
246    } finally {
247      dropIfExists(tn1);
248      dropIfExists(tn2);
249    }
250  }
251
252  @Test
253  public void testHonorsPatternFilter() throws Exception {
254    final TableName tn1 = TableName.valueOf(name.getMethodName() + "1");
255    final TableName tn2 = TableName.valueOf(name.getMethodName() + "2");
256
257    try {
258      final int tn1RegionCount = createTableBegsSplit(tn1, true, false);
259      final int tn2RegionCount = createTableBegsSplit(tn2, true, false);
260      final NormalizeTableFilterParams ntfp =
261        new NormalizeTableFilterParams.Builder().regex(".*[1]").build();
262
263      assertFalse(admin.normalizerSwitch(true).get());
264      assertTrue(admin.normalize(ntfp).get());
265      waitForTableRegionCount(tn1, greaterThanOrEqualTo(tn1RegionCount + 1));
266
267      // confirm that tn1 has (tn1RegionCount + 1) number of regions.
268      // tn2 has tn2RegionCount number of regions because it fails filter.
269      assertEquals(tn1 + " should have split.", tn1RegionCount + 1, getRegionCount(tn1));
270      waitForTableRegionCount(tn2, comparesEqualTo(tn2RegionCount));
271    } finally {
272      dropIfExists(tn1);
273      dropIfExists(tn2);
274    }
275  }
276
277  @Test
278  public void testHonorsNameFilter() throws Exception {
279    final TableName tn1 = TableName.valueOf(name.getMethodName() + "1");
280    final TableName tn2 = TableName.valueOf(name.getMethodName() + "2");
281
282    try {
283      final int tn1RegionCount = createTableBegsSplit(tn1, true, false);
284      final int tn2RegionCount = createTableBegsSplit(tn2, true, false);
285      final NormalizeTableFilterParams ntfp =
286        new NormalizeTableFilterParams.Builder().tableNames(Collections.singletonList(tn1)).build();
287
288      assertFalse(admin.normalizerSwitch(true).get());
289      assertTrue(admin.normalize(ntfp).get());
290      waitForTableRegionCount(tn1, greaterThanOrEqualTo(tn1RegionCount + 1));
291
292      // confirm that tn1 has (tn1RegionCount + 1) number of regions.
293      // tn2 has tn3RegionCount number of regions because it fails filter:
294      assertEquals(tn1 + " should have split.", tn1RegionCount + 1, getRegionCount(tn1));
295      waitForTableRegionCount(tn2, comparesEqualTo(tn2RegionCount));
296    } finally {
297      dropIfExists(tn1);
298      dropIfExists(tn2);
299    }
300  }
301
302  /**
303   * A test for when a region is the target of both a split and a merge plan. Does not define
304   * expected behavior, only that some change is applied to the table.
305   */
306  @Test
307  public void testTargetOfSplitAndMerge() throws Exception {
308    final TableName tn = TableName.valueOf(name.getMethodName());
309    try {
310      final int tnRegionCount = createTableTargetOfSplitAndMerge(tn);
311      assertFalse(admin.normalizerSwitch(true).get());
312      assertTrue(admin.normalize().get());
313      TEST_UTIL.waitFor(TimeUnit.MINUTES.toMillis(5),
314        new MatcherPredicate<>("expected " + tn + " to split or merge (probably split)",
315          () -> getRegionCountUnchecked(tn), not(comparesEqualTo(tnRegionCount))));
316    } finally {
317      dropIfExists(tn);
318    }
319  }
320
321  private static TableName buildTableNameForQuotaTest(final String methodName) throws Exception {
322    String nsp = "np2";
323    NamespaceDescriptor nspDesc =
324      NamespaceDescriptor.create(nsp).addConfiguration(TableNamespaceManager.KEY_MAX_REGIONS, "5")
325        .addConfiguration(TableNamespaceManager.KEY_MAX_TABLES, "2").build();
326    admin.createNamespace(nspDesc).get();
327    return TableName.valueOf(nsp, methodName);
328  }
329
330  private static void waitForSkippedSplits(final HMaster master,
331    final long existingSkippedSplitCount) {
332    TEST_UTIL.waitFor(TimeUnit.MINUTES.toMillis(5),
333      new MatcherPredicate<>("waiting to observe split attempt and skipped.",
334        () -> master.getRegionNormalizerManager().getSkippedCount(PlanType.SPLIT),
335        Matchers.greaterThan(existingSkippedSplitCount)));
336  }
337
338  private static void waitForTableRegionCount(final TableName tableName,
339    Matcher<? super Integer> matcher) {
340    TEST_UTIL.waitFor(TimeUnit.MINUTES.toMillis(5),
341      new MatcherPredicate<>("region count for table " + tableName + " does not match expected",
342        () -> getRegionCountUnchecked(tableName), matcher));
343  }
344
345  private static List<HRegion> generateTestData(final TableName tableName,
346    final int... regionSizesMb) throws IOException {
347    final List<HRegion> generatedRegions;
348    final int numRegions = regionSizesMb.length;
349    LOG.debug("generating test data into {}, {} regions of sizes (mb) {}", tableName, numRegions,
350      regionSizesMb);
351    try (Table ignored = TEST_UTIL.createMultiRegionTable(tableName, FAMILY_NAME, numRegions)) {
352      // Need to get sorted list of regions here
353      generatedRegions = new ArrayList<>(TEST_UTIL.getHBaseCluster().getRegions(tableName));
354      generatedRegions.sort(Comparator.comparing(HRegion::getRegionInfo, RegionInfo.COMPARATOR));
355      assertEquals(numRegions, generatedRegions.size());
356      for (int i = 0; i < numRegions; i++) {
357        HRegion region = generatedRegions.get(i);
358        generateTestData(region, regionSizesMb[i]);
359        region.flush(true);
360      }
361    }
362    return generatedRegions;
363  }
364
365  private static void generateTestData(Region region, int numRows) throws IOException {
366    // generating 1Mb values
367    LOG.debug("writing {}mb to {}", numRows, region);
368    LoadTestKVGenerator dataGenerator = new LoadTestKVGenerator(1024 * 1024, 1024 * 1024);
369    for (int i = 0; i < numRows; ++i) {
370      byte[] key = Bytes.add(region.getRegionInfo().getStartKey(), Bytes.toBytes(i));
371      for (int j = 0; j < 1; ++j) {
372        Put put = new Put(key);
373        byte[] col = Bytes.toBytes(String.valueOf(j));
374        byte[] value = dataGenerator.generateRandomSizeValue(key, col);
375        put.addColumn(FAMILY_NAME, col, value);
376        region.put(put);
377      }
378    }
379  }
380
381  private static double getRegionSizeMB(final MasterServices masterServices,
382    final RegionInfo regionInfo) {
383    final ServerName sn =
384      masterServices.getAssignmentManager().getRegionStates().getRegionServerOfRegion(regionInfo);
385    final RegionMetrics regionLoad = masterServices.getServerManager().getLoad(sn)
386      .getRegionMetrics().get(regionInfo.getRegionName());
387    if (regionLoad == null) {
388      LOG.debug("{} was not found in RegionsLoad", regionInfo.getRegionNameAsString());
389      return -1;
390    }
391    return regionLoad.getStoreFileSize().get(Size.Unit.MEGABYTE);
392  }
393
394  /**
395   * create a table with 5 regions, having region sizes so as to provoke a split of the largest
396   * region.
397   * <ul>
398   * <li>total table size: 12</li>
399   * <li>average region size: 2.4</li>
400   * <li>split threshold: 2.4 * 2 = 4.8</li>
401   * </ul>
402   */
403  private static int createTableBegsSplit(final TableName tableName,
404    final boolean normalizerEnabled, final boolean isMergeEnabled) throws Exception {
405    final List<HRegion> generatedRegions = generateTestData(tableName, 1, 1, 2, 3, 5);
406    assertEquals(5, getRegionCount(tableName));
407    admin.flush(tableName).get();
408
409    final TableDescriptor td =
410      TableDescriptorBuilder.newBuilder(admin.getDescriptor(tableName).get())
411        .setNormalizationEnabled(normalizerEnabled).setMergeEnabled(isMergeEnabled).build();
412    admin.modifyTable(td).get();
413
414    // make sure relatively accurate region statistics are available for the test table. use
415    // the last/largest region as clue.
416    TEST_UTIL.waitFor(TimeUnit.MINUTES.toMillis(1), new ExplainingPredicate<IOException>() {
417      @Override
418      public String explainFailure() {
419        return "expected largest region to be >= 4mb.";
420      }
421
422      @Override
423      public boolean evaluate() {
424        return generatedRegions.stream()
425          .mapToDouble(val -> getRegionSizeMB(master, val.getRegionInfo())).allMatch(val -> val > 0)
426          && getRegionSizeMB(master, generatedRegions.get(4).getRegionInfo()) >= 4.0;
427      }
428    });
429    return 5;
430  }
431
432  /**
433   * create a table with 5 regions, having region sizes so as to provoke a merge of the smallest
434   * regions.
435   * <ul>
436   * <li>total table size: 13</li>
437   * <li>average region size: 2.6</li>
438   * <li>sum of sizes of first two regions < average</li>
439   * </ul>
440   */
441  private static int createTableBegsMerge(final TableName tableName) throws Exception {
442    // create 5 regions with sizes to trigger merge of small regions
443    final List<HRegion> generatedRegions = generateTestData(tableName, 1, 1, 3, 3, 5);
444    assertEquals(5, getRegionCount(tableName));
445    admin.flush(tableName).get();
446
447    final TableDescriptor td = TableDescriptorBuilder
448      .newBuilder(admin.getDescriptor(tableName).get()).setNormalizationEnabled(true).build();
449    admin.modifyTable(td).get();
450
451    // make sure relatively accurate region statistics are available for the test table. use
452    // the last/largest region as clue.
453    LOG.debug("waiting for region statistics to settle.");
454    TEST_UTIL.waitFor(TimeUnit.MINUTES.toMillis(1), new ExplainingPredicate<IOException>() {
455      @Override
456      public String explainFailure() {
457        return "expected largest region to be >= 4mb.";
458      }
459
460      @Override
461      public boolean evaluate() {
462        return generatedRegions.stream()
463          .mapToDouble(val -> getRegionSizeMB(master, val.getRegionInfo())).allMatch(val -> val > 0)
464          && getRegionSizeMB(master, generatedRegions.get(4).getRegionInfo()) >= 4.0;
465      }
466    });
467    return 5;
468  }
469
470  /**
471   * Create a table with 4 regions, having region sizes so as to provoke a split of the largest
472   * region and a merge of an empty region into the largest.
473   * <ul>
474   * <li>total table size: 14</li>
475   * <li>average region size: 3.5</li>
476   * </ul>
477   */
478  private static int createTableTargetOfSplitAndMerge(final TableName tableName) throws Exception {
479    final int[] regionSizesMb = { 10, 0, 2, 2 };
480    final List<HRegion> generatedRegions = generateTestData(tableName, regionSizesMb);
481    assertEquals(4, getRegionCount(tableName));
482    admin.flush(tableName).get();
483
484    final TableDescriptor td = TableDescriptorBuilder
485      .newBuilder(admin.getDescriptor(tableName).get()).setNormalizationEnabled(true).build();
486    admin.modifyTable(td).get();
487
488    // make sure relatively accurate region statistics are available for the test table. use
489    // the last/largest region as clue.
490    LOG.debug("waiting for region statistics to settle.");
491    TEST_UTIL.waitFor(TimeUnit.MINUTES.toMillis(5), new ExplainingPredicate<IOException>() {
492      @Override
493      public String explainFailure() {
494        return "expected largest region to be >= 10mb.";
495      }
496
497      @Override
498      public boolean evaluate() {
499        for (int i = 0; i < generatedRegions.size(); i++) {
500          final RegionInfo regionInfo = generatedRegions.get(i).getRegionInfo();
501          if (!(getRegionSizeMB(master, regionInfo) >= regionSizesMb[i])) {
502            return false;
503          }
504        }
505        return true;
506      }
507    });
508    return 4;
509  }
510
511  private static void dropIfExists(final TableName tableName) throws Exception {
512    if (tableName != null && admin.tableExists(tableName).get()) {
513      if (admin.isTableEnabled(tableName).get()) {
514        admin.disableTable(tableName).get();
515      }
516      admin.deleteTable(tableName).get();
517    }
518  }
519
520  private static int getRegionCount(TableName tableName) throws IOException {
521    try (RegionLocator locator = TEST_UTIL.getConnection().getRegionLocator(tableName)) {
522      return locator.getAllRegionLocations().size();
523    }
524  }
525
526  private static int getRegionCountUnchecked(final TableName tableName) {
527    try {
528      return getRegionCount(tableName);
529    } catch (IOException e) {
530      throw new RuntimeException(e);
531    }
532  }
533}