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.rsgroup;
019
020import static org.junit.Assert.assertEquals;
021import static org.junit.Assert.assertFalse;
022import static org.junit.Assert.assertNull;
023import static org.junit.Assert.assertTrue;
024import static org.junit.Assert.fail;
025
026import java.io.IOException;
027import java.util.Iterator;
028import java.util.List;
029import java.util.Map;
030import java.util.Set;
031import java.util.SortedSet;
032
033import org.apache.hadoop.hbase.HBaseClassTestRule;
034import org.apache.hadoop.hbase.NamespaceDescriptor;
035import org.apache.hadoop.hbase.ServerName;
036import org.apache.hadoop.hbase.TableExistsException;
037import org.apache.hadoop.hbase.TableName;
038import org.apache.hadoop.hbase.TableNotFoundException;
039import org.apache.hadoop.hbase.Waiter;
040import org.apache.hadoop.hbase.client.ColumnFamilyDescriptor;
041import org.apache.hadoop.hbase.client.ColumnFamilyDescriptorBuilder;
042import org.apache.hadoop.hbase.client.TableDescriptor;
043import org.apache.hadoop.hbase.client.TableDescriptorBuilder;
044import org.apache.hadoop.hbase.constraint.ConstraintException;
045import org.apache.hadoop.hbase.master.ServerManager;
046import org.apache.hadoop.hbase.master.TableNamespaceManager;
047import org.apache.hadoop.hbase.master.snapshot.SnapshotManager;
048import org.apache.hadoop.hbase.net.Address;
049import org.apache.hadoop.hbase.quotas.QuotaUtil;
050import org.apache.hadoop.hbase.testclassification.MediumTests;
051import org.apache.hadoop.hbase.util.Bytes;
052import org.junit.After;
053import org.junit.AfterClass;
054import org.junit.Assert;
055import org.junit.Before;
056import org.junit.BeforeClass;
057import org.junit.ClassRule;
058import org.junit.Test;
059import org.junit.experimental.categories.Category;
060import org.slf4j.Logger;
061import org.slf4j.LoggerFactory;
062
063import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
064
065@Category({ MediumTests.class })
066public class TestRSGroupsAdmin1 extends TestRSGroupsBase {
067
068  @ClassRule
069  public static final HBaseClassTestRule CLASS_RULE =
070    HBaseClassTestRule.forClass(TestRSGroupsAdmin1.class);
071
072  protected static final Logger LOG = LoggerFactory.getLogger(TestRSGroupsAdmin1.class);
073
074  @BeforeClass
075  public static void setUp() throws Exception {
076    setUpTestBeforeClass();
077  }
078
079  @AfterClass
080  public static void tearDown() throws Exception {
081    tearDownAfterClass();
082  }
083
084  @Before
085  public void beforeMethod() throws Exception {
086    setUpBeforeMethod();
087  }
088
089  @After
090  public void afterMethod() throws Exception {
091    tearDownAfterMethod();
092  }
093
094  @Test
095  public void testValidGroupNames() throws IOException {
096    String[] badNames = { "foo*", "foo@", "-" };
097    String[] goodNames = { "foo_123" };
098
099    for (String entry : badNames) {
100      try {
101        rsGroupAdmin.addRSGroup(entry);
102        fail("Expected a constraint exception for: " + entry);
103      } catch (ConstraintException ex) {
104        // expected
105      }
106    }
107
108    for (String entry : goodNames) {
109      rsGroupAdmin.addRSGroup(entry);
110    }
111  }
112
113  @Test
114  public void testBogusArgs() throws Exception {
115    assertNull(rsGroupAdmin.getRSGroupInfoOfTable(TableName.valueOf("nonexistent")));
116    assertNull(rsGroupAdmin.getRSGroupOfServer(Address.fromParts("bogus", 123)));
117    assertNull(rsGroupAdmin.getRSGroupInfo("bogus"));
118
119    try {
120      rsGroupAdmin.removeRSGroup("bogus");
121      fail("Expected removing bogus group to fail");
122    } catch (ConstraintException ex) {
123      // expected
124    }
125
126    try {
127      rsGroupAdmin.moveTables(Sets.newHashSet(TableName.valueOf("bogustable")), "bogus");
128      fail("Expected move with bogus group to fail");
129    } catch (ConstraintException | TableNotFoundException ex) {
130      // expected
131    }
132
133    try {
134      rsGroupAdmin.moveServers(Sets.newHashSet(Address.fromParts("bogus", 123)), "bogus");
135      fail("Expected move with bogus group to fail");
136    } catch (ConstraintException ex) {
137      // expected
138    }
139
140    try {
141      admin.balancerSwitch(true, true);
142      rsGroupAdmin.balanceRSGroup("bogus");
143      admin.balancerSwitch(false, true);
144      fail("Expected move with bogus group to fail");
145    } catch (ConstraintException ex) {
146      // expected
147    }
148  }
149
150  @Test
151  public void testNamespaceConstraint() throws Exception {
152    String nsName = tablePrefix + "_foo";
153    String groupName = tablePrefix + "_foo";
154    LOG.info("testNamespaceConstraint");
155    rsGroupAdmin.addRSGroup(groupName);
156    assertTrue(observer.preAddRSGroupCalled);
157    assertTrue(observer.postAddRSGroupCalled);
158
159    admin.createNamespace(NamespaceDescriptor.create(nsName)
160      .addConfiguration(RSGroupInfo.NAMESPACE_DESC_PROP_GROUP, groupName).build());
161    // test removing a referenced group
162    try {
163      rsGroupAdmin.removeRSGroup(groupName);
164      fail("Expected a constraint exception");
165    } catch (IOException ex) {
166    }
167    // test modify group
168    // changing with the same name is fine
169    admin.modifyNamespace(NamespaceDescriptor.create(nsName)
170      .addConfiguration(RSGroupInfo.NAMESPACE_DESC_PROP_GROUP, groupName).build());
171    String anotherGroup = tablePrefix + "_anotherGroup";
172    rsGroupAdmin.addRSGroup(anotherGroup);
173    // test add non-existent group
174    admin.deleteNamespace(nsName);
175    rsGroupAdmin.removeRSGroup(groupName);
176    assertTrue(observer.preRemoveRSGroupCalled);
177    assertTrue(observer.postRemoveRSGroupCalled);
178    try {
179      admin.createNamespace(NamespaceDescriptor.create(nsName)
180        .addConfiguration(RSGroupInfo.NAMESPACE_DESC_PROP_GROUP, "foo").build());
181      fail("Expected a constraint exception");
182    } catch (IOException ex) {
183    }
184  }
185
186  @Test
187  public void testGroupInfoMultiAccessing() throws Exception {
188    RSGroupInfoManager manager = rsGroupAdminEndpoint.getGroupInfoManager();
189    RSGroupInfo defaultGroup = manager.getRSGroup("default");
190    // getRSGroup updates default group's server list
191    // this process must not affect other threads iterating the list
192    Iterator<Address> it = defaultGroup.getServers().iterator();
193    manager.getRSGroup("default");
194    it.next();
195  }
196
197  @Test
198  public void testFailRemoveGroup() throws IOException, InterruptedException {
199    int initNumGroups = rsGroupAdmin.listRSGroups().size();
200    addGroup("bar", 3);
201    TEST_UTIL.createTable(tableName, Bytes.toBytes("f"));
202    rsGroupAdmin.moveTables(Sets.newHashSet(tableName), "bar");
203    RSGroupInfo barGroup = rsGroupAdmin.getRSGroupInfo("bar");
204    // group is not empty therefore it should fail
205    try {
206      rsGroupAdmin.removeRSGroup(barGroup.getName());
207      fail("Expected remove group to fail");
208    } catch (IOException e) {
209    }
210    // group cannot lose all it's servers therefore it should fail
211    try {
212      rsGroupAdmin.moveServers(barGroup.getServers(), RSGroupInfo.DEFAULT_GROUP);
213      fail("Expected move servers to fail");
214    } catch (IOException e) {
215    }
216
217    rsGroupAdmin.moveTables(barGroup.getTables(), RSGroupInfo.DEFAULT_GROUP);
218    try {
219      rsGroupAdmin.removeRSGroup(barGroup.getName());
220      fail("Expected move servers to fail");
221    } catch (IOException e) {
222    }
223
224    rsGroupAdmin.moveServers(barGroup.getServers(), RSGroupInfo.DEFAULT_GROUP);
225    rsGroupAdmin.removeRSGroup(barGroup.getName());
226
227    Assert.assertEquals(initNumGroups, rsGroupAdmin.listRSGroups().size());
228  }
229
230  @Test
231  public void testMultiTableMove() throws Exception {
232    final TableName tableNameA = TableName.valueOf(tablePrefix + name.getMethodName() + "A");
233    final TableName tableNameB = TableName.valueOf(tablePrefix + name.getMethodName() + "B");
234    final byte[] familyNameBytes = Bytes.toBytes("f");
235    String newGroupName = getGroupName(name.getMethodName());
236    final RSGroupInfo newGroup = addGroup(newGroupName, 1);
237
238    TEST_UTIL.createTable(tableNameA, familyNameBytes);
239    TEST_UTIL.createTable(tableNameB, familyNameBytes);
240    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
241      @Override
242      public boolean evaluate() throws Exception {
243        List<String> regionsA = getTableRegionMap().get(tableNameA);
244        if (regionsA == null) {
245          return false;
246        }
247        List<String> regionsB = getTableRegionMap().get(tableNameB);
248        if (regionsB == null) {
249          return false;
250        }
251        return getTableRegionMap().get(tableNameA).size() >= 1 &&
252          getTableRegionMap().get(tableNameB).size() >= 1;
253      }
254    });
255
256    RSGroupInfo tableGrpA = rsGroupAdmin.getRSGroupInfoOfTable(tableNameA);
257    assertTrue(tableGrpA.getName().equals(RSGroupInfo.DEFAULT_GROUP));
258
259    RSGroupInfo tableGrpB = rsGroupAdmin.getRSGroupInfoOfTable(tableNameB);
260    assertTrue(tableGrpB.getName().equals(RSGroupInfo.DEFAULT_GROUP));
261    // change table's group
262    LOG.info("Moving table [" + tableNameA + "," + tableNameB + "] to " + newGroup.getName());
263    rsGroupAdmin.moveTables(Sets.newHashSet(tableNameA, tableNameB), newGroup.getName());
264
265    // verify group change
266    Assert.assertEquals(newGroup.getName(),
267      rsGroupAdmin.getRSGroupInfoOfTable(tableNameA).getName());
268
269    Assert.assertEquals(newGroup.getName(),
270      rsGroupAdmin.getRSGroupInfoOfTable(tableNameB).getName());
271
272    // verify tables' not exist in old group
273    Set<TableName> DefaultTables =
274      rsGroupAdmin.getRSGroupInfo(RSGroupInfo.DEFAULT_GROUP).getTables();
275    assertFalse(DefaultTables.contains(tableNameA));
276    assertFalse(DefaultTables.contains(tableNameB));
277
278    // verify tables' exist in new group
279    Set<TableName> newGroupTables = rsGroupAdmin.getRSGroupInfo(newGroupName).getTables();
280    assertTrue(newGroupTables.contains(tableNameA));
281    assertTrue(newGroupTables.contains(tableNameB));
282  }
283
284  @Test
285  public void testTableMoveTruncateAndDrop() throws Exception {
286    final byte[] familyNameBytes = Bytes.toBytes("f");
287    String newGroupName = getGroupName(name.getMethodName());
288    final RSGroupInfo newGroup = addGroup(newGroupName, 2);
289
290    TEST_UTIL.createMultiRegionTable(tableName, familyNameBytes, 5);
291    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
292      @Override
293      public boolean evaluate() throws Exception {
294        List<String> regions = getTableRegionMap().get(tableName);
295        if (regions == null) {
296          return false;
297        }
298
299        return getTableRegionMap().get(tableName).size() >= 5;
300      }
301    });
302
303    RSGroupInfo tableGrp = rsGroupAdmin.getRSGroupInfoOfTable(tableName);
304    assertTrue(tableGrp.getName().equals(RSGroupInfo.DEFAULT_GROUP));
305
306    // change table's group
307    LOG.info("Moving table " + tableName + " to " + newGroup.getName());
308    rsGroupAdmin.moveTables(Sets.newHashSet(tableName), newGroup.getName());
309
310    // verify group change
311    Assert.assertEquals(newGroup.getName(),
312      rsGroupAdmin.getRSGroupInfoOfTable(tableName).getName());
313
314    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
315      @Override
316      public boolean evaluate() throws Exception {
317        Map<ServerName, List<String>> serverMap = getTableServerRegionMap().get(tableName);
318        int count = 0;
319        if (serverMap != null) {
320          for (ServerName rs : serverMap.keySet()) {
321            if (newGroup.containsServer(rs.getAddress())) {
322              count += serverMap.get(rs).size();
323            }
324          }
325        }
326        return count == 5;
327      }
328    });
329
330    // test truncate
331    admin.disableTable(tableName);
332    admin.truncateTable(tableName, true);
333    Assert.assertEquals(1, rsGroupAdmin.getRSGroupInfo(newGroup.getName()).getTables().size());
334    Assert.assertEquals(tableName,
335      rsGroupAdmin.getRSGroupInfo(newGroup.getName()).getTables().first());
336
337    // verify removed table is removed from group
338    TEST_UTIL.deleteTable(tableName);
339    Assert.assertEquals(0, rsGroupAdmin.getRSGroupInfo(newGroup.getName()).getTables().size());
340
341    assertTrue(observer.preMoveTablesCalled);
342    assertTrue(observer.postMoveTablesCalled);
343  }
344
345  @Test
346  public void testDisabledTableMove() throws Exception {
347    final byte[] familyNameBytes = Bytes.toBytes("f");
348    String newGroupName = getGroupName(name.getMethodName());
349    final RSGroupInfo newGroup = addGroup(newGroupName, 2);
350
351    TEST_UTIL.createMultiRegionTable(tableName, familyNameBytes, 5);
352    TEST_UTIL.waitFor(WAIT_TIMEOUT, new Waiter.Predicate<Exception>() {
353      @Override
354      public boolean evaluate() throws Exception {
355        List<String> regions = getTableRegionMap().get(tableName);
356        if (regions == null) {
357          return false;
358        }
359        return getTableRegionMap().get(tableName).size() >= 5;
360      }
361    });
362
363    RSGroupInfo tableGrp = rsGroupAdmin.getRSGroupInfoOfTable(tableName);
364    assertTrue(tableGrp.getName().equals(RSGroupInfo.DEFAULT_GROUP));
365
366    // test disable table
367    admin.disableTable(tableName);
368
369    // change table's group
370    LOG.info("Moving table " + tableName + " to " + newGroup.getName());
371    rsGroupAdmin.moveTables(Sets.newHashSet(tableName), newGroup.getName());
372
373    // verify group change
374    Assert.assertEquals(newGroup.getName(),
375      rsGroupAdmin.getRSGroupInfoOfTable(tableName).getName());
376  }
377
378  @Test
379  public void testNonExistentTableMove() throws Exception {
380    TableName tableName = TableName.valueOf(tablePrefix + name.getMethodName());
381
382    RSGroupInfo tableGrp = rsGroupAdmin.getRSGroupInfoOfTable(tableName);
383    assertNull(tableGrp);
384
385    // test if table exists already.
386    boolean exist = admin.tableExists(tableName);
387    assertFalse(exist);
388
389    LOG.info("Moving table " + tableName + " to " + RSGroupInfo.DEFAULT_GROUP);
390    try {
391      rsGroupAdmin.moveTables(Sets.newHashSet(tableName), RSGroupInfo.DEFAULT_GROUP);
392      fail("Table " + tableName + " shouldn't have been successfully moved.");
393    } catch (IOException ex) {
394      assertTrue(ex instanceof TableNotFoundException);
395    }
396
397    try {
398      rsGroupAdmin.moveServersAndTables(Sets.newHashSet(Address.fromParts("bogus", 123)),
399        Sets.newHashSet(tableName), RSGroupInfo.DEFAULT_GROUP);
400      fail("Table " + tableName + " shouldn't have been successfully moved.");
401    } catch (IOException ex) {
402      assertTrue(ex instanceof TableNotFoundException);
403    }
404    // verify group change
405    assertNull(rsGroupAdmin.getRSGroupInfoOfTable(tableName));
406  }
407
408  @Test
409  public void testRSGroupListDoesNotContainFailedTableCreation() throws Exception {
410    toggleQuotaCheckAndRestartMiniCluster(true);
411    String nsp = "np1";
412    NamespaceDescriptor nspDesc =
413      NamespaceDescriptor.create(nsp).addConfiguration(TableNamespaceManager.KEY_MAX_REGIONS, "5")
414        .addConfiguration(TableNamespaceManager.KEY_MAX_TABLES, "2").build();
415    admin.createNamespace(nspDesc);
416    assertEquals(3, admin.listNamespaceDescriptors().length);
417    ColumnFamilyDescriptor fam1 = ColumnFamilyDescriptorBuilder.of("fam1");
418    TableDescriptor tableDescOne = TableDescriptorBuilder
419      .newBuilder(TableName.valueOf(nsp + TableName.NAMESPACE_DELIM + "table1"))
420      .setColumnFamily(fam1).build();
421    admin.createTable(tableDescOne);
422
423    TableDescriptor tableDescTwo = TableDescriptorBuilder
424      .newBuilder(TableName.valueOf(nsp + TableName.NAMESPACE_DELIM + "table2"))
425      .setColumnFamily(fam1).build();
426    boolean constraintViolated = false;
427
428    try {
429      admin.createTable(tableDescTwo, Bytes.toBytes("AAA"), Bytes.toBytes("ZZZ"), 6);
430      Assert.fail("Creation table should fail because of quota violation.");
431    } catch (Exception exp) {
432      assertTrue(exp instanceof IOException);
433      constraintViolated = true;
434    } finally {
435      assertTrue("Constraint not violated for table " + tableDescTwo.getTableName(),
436        constraintViolated);
437    }
438    List<RSGroupInfo> rsGroupInfoList = rsGroupAdmin.listRSGroups();
439    boolean foundTable2 = false;
440    boolean foundTable1 = false;
441    for (int i = 0; i < rsGroupInfoList.size(); i++) {
442      if (rsGroupInfoList.get(i).getTables().contains(tableDescTwo.getTableName())) {
443        foundTable2 = true;
444      }
445      if (rsGroupInfoList.get(i).getTables().contains(tableDescOne.getTableName())) {
446        foundTable1 = true;
447      }
448    }
449    assertFalse("Found table2 in rsgroup list.", foundTable2);
450    assertTrue("Did not find table1 in rsgroup list", foundTable1);
451
452    TEST_UTIL.deleteTable(tableDescOne.getTableName());
453    admin.deleteNamespace(nspDesc.getName());
454    toggleQuotaCheckAndRestartMiniCluster(false);
455
456  }
457
458  @Test
459  public void testNotMoveTableToNullRSGroupWhenCreatingExistingTable()
460          throws Exception {
461    // Trigger
462    TableName tn1 = TableName.valueOf("t1");
463    TEST_UTIL.createTable(tn1, "cf1");
464    try {
465      // Create an existing table to trigger HBASE-21866
466      TEST_UTIL.createTable(tn1, "cf1");
467    } catch (TableExistsException teex) {
468      // Ignore
469    }
470
471    // Wait then verify
472    //   Could not verify until the rollback of CreateTableProcedure is done
473    //   (that is, the coprocessor finishes its work),
474    //   or the table is still in the "default" rsgroup even though HBASE-21866
475    //   is not fixed.
476    TEST_UTIL.waitFor(5000, new Waiter.Predicate<Exception>() {
477      @Override
478      public boolean evaluate() throws Exception {
479        return
480                (master.getMasterProcedureExecutor().getActiveExecutorCount() == 0);
481      }
482    });
483    SortedSet<TableName> tables
484            = rsGroupAdmin.getRSGroupInfo(RSGroupInfo.DEFAULT_GROUP).getTables();
485    assertTrue("Table 't1' must be in 'default' rsgroup", tables.contains(tn1));
486
487    // Cleanup
488    TEST_UTIL.deleteTable(tn1);
489  }
490
491  private void toggleQuotaCheckAndRestartMiniCluster(boolean enable) throws Exception {
492    TEST_UTIL.shutdownMiniCluster();
493    TEST_UTIL.getConfiguration().setBoolean(QuotaUtil.QUOTA_CONF_KEY, enable);
494    TEST_UTIL.startMiniCluster(NUM_SLAVES_BASE - 1);
495    TEST_UTIL.getConfiguration().setInt(ServerManager.WAIT_ON_REGIONSERVERS_MINTOSTART,
496      NUM_SLAVES_BASE - 1);
497    TEST_UTIL.getConfiguration().setBoolean(SnapshotManager.HBASE_SNAPSHOT_ENABLED, true);
498    initialize();
499  }
500
501  @Test
502  public void testRenameRSGroup() throws Exception {
503    // Add rsgroup, and assign 2 servers and a table to it.
504    RSGroupInfo oldgroup = addGroup("oldgroup", 2);
505    final TableName tb1 = TableName.valueOf("testRename");
506    TEST_UTIL.createTable(tb1, "tr");
507    rsGroupAdmin.moveTables(Sets.newHashSet(tb1), oldgroup.getName());
508    TEST_UTIL.waitFor(1000,
509      (Waiter.Predicate<Exception>) () ->
510        rsGroupAdmin.getRSGroupInfoOfTable(tb1).getServers().size() == 2);
511    oldgroup = rsGroupAdmin.getRSGroupInfo(oldgroup.getName());
512    assertEquals(2, oldgroup.getServers().size());
513    assertEquals(oldgroup.getName(), rsGroupAdmin.getRSGroupInfoOfTable(tb1).getName());
514    assertTrue(oldgroup.getTables().contains(tb1));
515
516    // Another rsgroup and table for verification
517    // that they are unchanged during we're renaming oldgroup.
518    RSGroupInfo normal = addGroup("normal", 1);
519    final TableName tb2 = TableName.valueOf("unmovedTable");
520    TEST_UTIL.createTable(tb2, "ut");
521    rsGroupAdmin.moveTables(Sets.newHashSet(tb2), normal.getName());
522    TEST_UTIL.waitFor(1000,
523      (Waiter.Predicate<Exception>) () ->
524        rsGroupAdmin.getRSGroupInfoOfTable(tb2).getServers().size() == 1);
525    normal = rsGroupAdmin.getRSGroupInfo(normal.getName());
526    assertEquals(1, normal.getServers().size());
527    assertEquals(normal.getName(), rsGroupAdmin.getRSGroupInfoOfTable(tb2).getName());
528    assertTrue(normal.containsTable(tb2));
529
530
531    // Rename rsgroup
532    rsGroupAdmin.renameRSGroup(oldgroup.getName(), "newgroup");
533    Set<Address> servers = oldgroup.getServers();
534    RSGroupInfo newgroup = rsGroupAdmin.getRSGroupInfo("newgroup");
535    assertEquals(servers.size(), newgroup.getServers().size());
536    for (Address server : servers) {
537      assertTrue(newgroup.containsServer(server));
538    }
539    assertEquals(newgroup.getName(), rsGroupAdmin.getRSGroupInfoOfTable(tb1).getName());
540    assertTrue(newgroup.containsTable(tb1));
541    assertEquals(normal.getName(), rsGroupAdmin.getRSGroupInfoOfTable(tb2).getName());
542  }
543}