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; 019 020import static org.junit.Assert.assertEquals; 021import static org.junit.Assert.assertNotNull; 022import static org.junit.Assert.assertNull; 023import static org.junit.Assert.assertTrue; 024import static org.mockito.ArgumentMatchers.any; 025import static org.mockito.Mockito.doReturn; 026import static org.mockito.Mockito.mock; 027import static org.mockito.Mockito.reset; 028import static org.mockito.Mockito.times; 029import static org.mockito.Mockito.verify; 030 031import java.io.IOException; 032import java.util.Collections; 033import java.util.List; 034import java.util.Random; 035import java.util.concurrent.ThreadLocalRandom; 036import org.apache.hadoop.conf.Configuration; 037import org.apache.hadoop.hbase.client.Connection; 038import org.apache.hadoop.hbase.client.ConnectionFactory; 039import org.apache.hadoop.hbase.client.Get; 040import org.apache.hadoop.hbase.client.RegionInfo; 041import org.apache.hadoop.hbase.client.RegionInfoBuilder; 042import org.apache.hadoop.hbase.client.RegionLocator; 043import org.apache.hadoop.hbase.client.Result; 044import org.apache.hadoop.hbase.client.Table; 045import org.apache.hadoop.hbase.ipc.CallRunner; 046import org.apache.hadoop.hbase.ipc.DelegatingRpcScheduler; 047import org.apache.hadoop.hbase.ipc.PriorityFunction; 048import org.apache.hadoop.hbase.ipc.RpcScheduler; 049import org.apache.hadoop.hbase.master.HMaster; 050import org.apache.hadoop.hbase.regionserver.HRegion; 051import org.apache.hadoop.hbase.regionserver.HRegionServer; 052import org.apache.hadoop.hbase.regionserver.SimpleRpcSchedulerFactory; 053import org.apache.hadoop.hbase.testclassification.MediumTests; 054import org.apache.hadoop.hbase.testclassification.MiscTests; 055import org.apache.hadoop.hbase.util.Bytes; 056import org.apache.hadoop.hbase.util.EnvironmentEdgeManager; 057import org.apache.hadoop.hbase.util.Pair; 058import org.junit.AfterClass; 059import org.junit.BeforeClass; 060import org.junit.ClassRule; 061import org.junit.Rule; 062import org.junit.Test; 063import org.junit.experimental.categories.Category; 064import org.junit.rules.TestName; 065import org.slf4j.Logger; 066import org.slf4j.LoggerFactory; 067 068import org.apache.hbase.thirdparty.com.google.common.collect.Lists; 069 070/** 071 * Test {@link org.apache.hadoop.hbase.MetaTableAccessor}. 072 */ 073@Category({ MiscTests.class, MediumTests.class }) 074@SuppressWarnings("deprecation") 075public class TestMetaTableAccessor { 076 @ClassRule 077 public static final HBaseClassTestRule CLASS_RULE = 078 HBaseClassTestRule.forClass(TestMetaTableAccessor.class); 079 080 private static final Logger LOG = LoggerFactory.getLogger(TestMetaTableAccessor.class); 081 private static final HBaseTestingUtil UTIL = new HBaseTestingUtil(); 082 private static Connection connection; 083 084 @Rule 085 public TestName name = new TestName(); 086 087 @BeforeClass 088 public static void beforeClass() throws Exception { 089 UTIL.startMiniCluster(3); 090 091 Configuration c = new Configuration(UTIL.getConfiguration()); 092 // Tests to 4 retries every 5 seconds. Make it try every 1 second so more 093 // responsive. 1 second is default as is ten retries. 094 c.setLong("hbase.client.pause", 1000); 095 c.setInt(HConstants.HBASE_CLIENT_RETRIES_NUMBER, 10); 096 connection = ConnectionFactory.createConnection(c); 097 } 098 099 @AfterClass 100 public static void afterClass() throws Exception { 101 connection.close(); 102 UTIL.shutdownMiniCluster(); 103 } 104 105 @Test 106 public void testIsMetaWhenAllHealthy() throws InterruptedException { 107 HMaster m = UTIL.getMiniHBaseCluster().getMaster(); 108 assertTrue(m.waitForMetaOnline()); 109 } 110 111 @Test 112 public void testIsMetaWhenMetaGoesOffline() throws InterruptedException { 113 HMaster m = UTIL.getMiniHBaseCluster().getMaster(); 114 int index = UTIL.getMiniHBaseCluster().getServerWithMeta(); 115 HRegionServer rsWithMeta = UTIL.getMiniHBaseCluster().getRegionServer(index); 116 rsWithMeta.abort("TESTING"); 117 assertTrue(m.waitForMetaOnline()); 118 } 119 120 /** 121 * Does {@link MetaTableAccessor#getRegion(Connection, byte[])} and a write against hbase:meta 122 * while its hosted server is restarted to prove our retrying works. 123 */ 124 @Test 125 public void testRetrying() throws IOException, InterruptedException { 126 final TableName tableName = TableName.valueOf(name.getMethodName()); 127 LOG.info("Started " + tableName); 128 Table t = UTIL.createMultiRegionTable(tableName, HConstants.CATALOG_FAMILY); 129 int regionCount = -1; 130 try (RegionLocator r = UTIL.getConnection().getRegionLocator(tableName)) { 131 regionCount = r.getStartKeys().length; 132 } 133 // Test it works getting a region from just made user table. 134 final List<RegionInfo> regions = testGettingTableRegions(connection, tableName, regionCount); 135 MetaTask reader = new MetaTask(connection, "reader") { 136 @Override 137 void metaTask() throws Throwable { 138 testGetRegion(connection, regions.get(0)); 139 LOG.info("Read " + regions.get(0).getEncodedName()); 140 } 141 }; 142 MetaTask writer = new MetaTask(connection, "writer") { 143 144 @Override 145 void metaTask() throws IOException { 146 MetaTableAccessor.addRegionsToMeta(connection, Collections.singletonList(regions.get(0)), 147 1); 148 LOG.info("Wrote " + regions.get(0).getEncodedName()); 149 } 150 }; 151 reader.start(); 152 writer.start(); 153 154 // We're gonna check how it takes. If it takes too long, we will consider 155 // it as a fail. We can't put that in the @Test tag as we want to close 156 // the threads nicely 157 final long timeOut = 180000; 158 long startTime = EnvironmentEdgeManager.currentTime(); 159 160 try { 161 // Make sure reader and writer are working. 162 assertTrue(reader.isProgressing()); 163 assertTrue(writer.isProgressing()); 164 165 // Kill server hosting meta -- twice . See if our reader/writer ride over the 166 // meta moves. They'll need to retry. 167 for (int i = 0; i < 2; i++) { 168 LOG.info("Restart=" + i); 169 UTIL.ensureSomeRegionServersAvailable(2); 170 int index = -1; 171 do { 172 index = UTIL.getMiniHBaseCluster().getServerWithMeta(); 173 } while (index == -1 && startTime + timeOut < EnvironmentEdgeManager.currentTime()); 174 175 if (index != -1) { 176 UTIL.getMiniHBaseCluster().abortRegionServer(index); 177 UTIL.getMiniHBaseCluster().waitOnRegionServer(index); 178 } 179 } 180 181 assertTrue("reader: " + reader.toString(), reader.isProgressing()); 182 assertTrue("writer: " + writer.toString(), writer.isProgressing()); 183 } catch (IOException e) { 184 throw e; 185 } finally { 186 reader.stop = true; 187 writer.stop = true; 188 reader.join(); 189 writer.join(); 190 t.close(); 191 } 192 long exeTime = EnvironmentEdgeManager.currentTime() - startTime; 193 assertTrue("Timeout: test took " + exeTime / 1000 + " sec", exeTime < timeOut); 194 } 195 196 /** 197 * Thread that runs a MetaTableAccessor task until asked stop. 198 */ 199 abstract static class MetaTask extends Thread { 200 boolean stop = false; 201 int count = 0; 202 Throwable t = null; 203 final Connection connection; 204 205 MetaTask(final Connection connection, final String name) { 206 super(name); 207 this.connection = connection; 208 } 209 210 @Override 211 public void run() { 212 try { 213 while (!this.stop) { 214 LOG.info("Before " + this.getName() + ", count=" + this.count); 215 metaTask(); 216 this.count += 1; 217 LOG.info("After " + this.getName() + ", count=" + this.count); 218 Thread.sleep(100); 219 } 220 } catch (Throwable t) { 221 LOG.info(this.getName() + " failed", t); 222 this.t = t; 223 } 224 } 225 226 boolean isProgressing() throws InterruptedException { 227 int currentCount = this.count; 228 while (currentCount == this.count) { 229 if (!isAlive()) return false; 230 if (this.t != null) return false; 231 Thread.sleep(10); 232 } 233 return true; 234 } 235 236 @Override 237 public String toString() { 238 return "count=" + this.count + ", t=" + (this.t == null ? "null" : this.t.toString()); 239 } 240 241 abstract void metaTask() throws Throwable; 242 } 243 244 @Test 245 public void testGetRegion() throws IOException, InterruptedException { 246 final String name = this.name.getMethodName(); 247 LOG.info("Started " + name); 248 // Test get on non-existent region. 249 Pair<RegionInfo, ServerName> pair = 250 MetaTableAccessor.getRegion(connection, Bytes.toBytes("nonexistent-region")); 251 assertNull(pair); 252 LOG.info("Finished " + name); 253 } 254 255 // Test for the optimization made in HBASE-3650 256 @Test 257 public void testScanMetaForTable() throws IOException, InterruptedException { 258 final TableName tableName = TableName.valueOf(name.getMethodName()); 259 LOG.info("Started " + tableName); 260 261 /** 262 * Create 2 tables - testScanMetaForTable - testScanMetaForTablf 263 **/ 264 265 UTIL.createTable(tableName, HConstants.CATALOG_FAMILY); 266 // name that is +1 greater than the first one (e+1=f) 267 TableName greaterName = TableName.valueOf("testScanMetaForTablf"); 268 UTIL.createTable(greaterName, HConstants.CATALOG_FAMILY); 269 270 // Now make sure we only get the regions from 1 of the tables at a time 271 272 assertEquals(1, MetaTableAccessor.getTableRegions(connection, tableName).size()); 273 assertEquals(1, MetaTableAccessor.getTableRegions(connection, greaterName).size()); 274 } 275 276 private static List<RegionInfo> testGettingTableRegions(final Connection connection, 277 final TableName name, final int regionCount) throws IOException, InterruptedException { 278 List<RegionInfo> regions = MetaTableAccessor.getTableRegions(connection, name); 279 assertEquals(regionCount, regions.size()); 280 Pair<RegionInfo, ServerName> pair = 281 MetaTableAccessor.getRegion(connection, regions.get(0).getRegionName()); 282 assertEquals(regions.get(0).getEncodedName(), pair.getFirst().getEncodedName()); 283 return regions; 284 } 285 286 private static void testGetRegion(final Connection connection, final RegionInfo region) 287 throws IOException, InterruptedException { 288 Pair<RegionInfo, ServerName> pair = 289 MetaTableAccessor.getRegion(connection, region.getRegionName()); 290 assertEquals(region.getEncodedName(), pair.getFirst().getEncodedName()); 291 } 292 293 @Test 294 public void testMetaLocationsForRegionReplicas() throws IOException { 295 Random rand = ThreadLocalRandom.current(); 296 297 ServerName serverName0 = ServerName.valueOf("foo", 60010, rand.nextLong()); 298 ServerName serverName1 = ServerName.valueOf("bar", 60010, rand.nextLong()); 299 ServerName serverName100 = ServerName.valueOf("baz", 60010, rand.nextLong()); 300 301 long regionId = EnvironmentEdgeManager.currentTime(); 302 RegionInfo primary = RegionInfoBuilder.newBuilder(TableName.valueOf(name.getMethodName())) 303 .setStartKey(HConstants.EMPTY_START_ROW).setEndKey(HConstants.EMPTY_END_ROW).setSplit(false) 304 .setRegionId(regionId).setReplicaId(0).build(); 305 RegionInfo replica1 = RegionInfoBuilder.newBuilder(TableName.valueOf(name.getMethodName())) 306 .setStartKey(HConstants.EMPTY_START_ROW).setEndKey(HConstants.EMPTY_END_ROW).setSplit(false) 307 .setRegionId(regionId).setReplicaId(1).build(); 308 RegionInfo replica100 = RegionInfoBuilder.newBuilder(TableName.valueOf(name.getMethodName())) 309 .setStartKey(HConstants.EMPTY_START_ROW).setEndKey(HConstants.EMPTY_END_ROW).setSplit(false) 310 .setRegionId(regionId).setReplicaId(100).build(); 311 312 long seqNum0 = rand.nextLong(); 313 long seqNum1 = rand.nextLong(); 314 long seqNum100 = rand.nextLong(); 315 316 try (Table meta = MetaTableAccessor.getMetaHTable(connection)) { 317 MetaTableAccessor.updateRegionLocation(connection, primary, serverName0, seqNum0, 318 EnvironmentEdgeManager.currentTime()); 319 320 // assert that the server, startcode and seqNum columns are there for the primary region 321 assertMetaLocation(meta, primary.getRegionName(), serverName0, seqNum0, 0, true); 322 323 // add replica = 1 324 MetaTableAccessor.updateRegionLocation(connection, replica1, serverName1, seqNum1, 325 EnvironmentEdgeManager.currentTime()); 326 // check whether the primary is still there 327 assertMetaLocation(meta, primary.getRegionName(), serverName0, seqNum0, 0, true); 328 // now check for replica 1 329 assertMetaLocation(meta, primary.getRegionName(), serverName1, seqNum1, 1, true); 330 331 // add replica = 1 332 MetaTableAccessor.updateRegionLocation(connection, replica100, serverName100, seqNum100, 333 EnvironmentEdgeManager.currentTime()); 334 // check whether the primary is still there 335 assertMetaLocation(meta, primary.getRegionName(), serverName0, seqNum0, 0, true); 336 // check whether the replica 1 is still there 337 assertMetaLocation(meta, primary.getRegionName(), serverName1, seqNum1, 1, true); 338 // now check for replica 1 339 assertMetaLocation(meta, primary.getRegionName(), serverName100, seqNum100, 100, true); 340 } 341 } 342 343 public static void assertMetaLocation(Table meta, byte[] row, ServerName serverName, long seqNum, 344 int replicaId, boolean checkSeqNum) throws IOException { 345 Get get = new Get(row); 346 Result result = meta.get(get); 347 assertTrue(Bytes.equals( 348 result.getValue(HConstants.CATALOG_FAMILY, CatalogFamilyFormat.getServerColumn(replicaId)), 349 Bytes.toBytes(serverName.getAddress().toString()))); 350 assertTrue(Bytes.equals( 351 result.getValue(HConstants.CATALOG_FAMILY, CatalogFamilyFormat.getStartCodeColumn(replicaId)), 352 Bytes.toBytes(serverName.getStartcode()))); 353 if (checkSeqNum) { 354 assertTrue(Bytes.equals( 355 result.getValue(HConstants.CATALOG_FAMILY, CatalogFamilyFormat.getSeqNumColumn(replicaId)), 356 Bytes.toBytes(seqNum))); 357 } 358 } 359 360 public static void assertEmptyMetaLocation(Table meta, byte[] row, int replicaId) 361 throws IOException { 362 Get get = new Get(row); 363 Result result = meta.get(get); 364 Cell serverCell = result.getColumnLatestCell(HConstants.CATALOG_FAMILY, 365 CatalogFamilyFormat.getServerColumn(replicaId)); 366 Cell startCodeCell = result.getColumnLatestCell(HConstants.CATALOG_FAMILY, 367 CatalogFamilyFormat.getStartCodeColumn(replicaId)); 368 assertNotNull(serverCell); 369 assertNotNull(startCodeCell); 370 assertEquals(0, serverCell.getValueLength()); 371 assertEquals(0, startCodeCell.getValueLength()); 372 } 373 374 @Test 375 public void testMetaLocationForRegionReplicasIsAddedAtTableCreation() throws IOException { 376 long regionId = EnvironmentEdgeManager.currentTime(); 377 RegionInfo primary = RegionInfoBuilder.newBuilder(TableName.valueOf(name.getMethodName())) 378 .setStartKey(HConstants.EMPTY_START_ROW).setEndKey(HConstants.EMPTY_END_ROW).setSplit(false) 379 .setRegionId(regionId).setReplicaId(0).build(); 380 381 Table meta = MetaTableAccessor.getMetaHTable(connection); 382 try { 383 List<RegionInfo> regionInfos = Lists.newArrayList(primary); 384 MetaTableAccessor.addRegionsToMeta(connection, regionInfos, 3); 385 386 assertEmptyMetaLocation(meta, primary.getRegionName(), 1); 387 assertEmptyMetaLocation(meta, primary.getRegionName(), 2); 388 } finally { 389 meta.close(); 390 } 391 } 392 393 @Test 394 public void testMetaScanner() throws Exception { 395 LOG.info("Starting " + name.getMethodName()); 396 397 final TableName tableName = TableName.valueOf(name.getMethodName()); 398 final byte[] FAMILY = Bytes.toBytes("family"); 399 final byte[][] SPLIT_KEYS = 400 new byte[][] { Bytes.toBytes("region_a"), Bytes.toBytes("region_b") }; 401 402 UTIL.createTable(tableName, FAMILY, SPLIT_KEYS); 403 Table table = connection.getTable(tableName); 404 // Make sure all the regions are deployed 405 HBaseTestingUtil.countRows(table); 406 407 ClientMetaTableAccessor.Visitor visitor = mock(ClientMetaTableAccessor.Visitor.class); 408 doReturn(true).when(visitor).visit(any()); 409 410 // Scanning the entire table should give us three rows 411 MetaTableAccessor.scanMetaForTableRegions(connection, visitor, tableName); 412 verify(visitor, times(3)).visit(any()); 413 414 // Scanning the table with a specified empty start row should also 415 // give us three hbase:meta rows 416 reset(visitor); 417 doReturn(true).when(visitor).visit(any()); 418 MetaTableAccessor.scanMeta(connection, visitor, tableName, null, 1000); 419 verify(visitor, times(3)).visit(any()); 420 421 // Scanning the table starting in the middle should give us two rows: 422 // region_a and region_b 423 reset(visitor); 424 doReturn(true).when(visitor).visit(any()); 425 MetaTableAccessor.scanMeta(connection, visitor, tableName, Bytes.toBytes("region_ac"), 1000); 426 verify(visitor, times(2)).visit(any()); 427 428 // Scanning with a limit of 1 should only give us one row 429 reset(visitor); 430 doReturn(true).when(visitor).visit(any()); 431 MetaTableAccessor.scanMeta(connection, visitor, tableName, Bytes.toBytes("region_ac"), 1); 432 verify(visitor, times(1)).visit(any()); 433 table.close(); 434 } 435 436 /** 437 * Tests whether maximum of masters system time versus RSs local system time is used 438 */ 439 @Test 440 public void testMastersSystemTimeIsUsedInUpdateLocations() throws IOException { 441 long regionId = EnvironmentEdgeManager.currentTime(); 442 RegionInfo regionInfo = RegionInfoBuilder.newBuilder(TableName.valueOf(name.getMethodName())) 443 .setStartKey(HConstants.EMPTY_START_ROW).setEndKey(HConstants.EMPTY_END_ROW).setSplit(false) 444 .setRegionId(regionId).setReplicaId(0).build(); 445 446 ServerName sn = ServerName.valueOf("bar", 0, 0); 447 try (Table meta = MetaTableAccessor.getMetaHTable(connection)) { 448 List<RegionInfo> regionInfos = Lists.newArrayList(regionInfo); 449 MetaTableAccessor.addRegionsToMeta(connection, regionInfos, 1); 450 451 long masterSystemTime = EnvironmentEdgeManager.currentTime() + 123456789; 452 MetaTableAccessor.updateRegionLocation(connection, regionInfo, sn, 1, masterSystemTime); 453 454 Get get = new Get(regionInfo.getRegionName()); 455 Result result = meta.get(get); 456 Cell serverCell = result.getColumnLatestCell(HConstants.CATALOG_FAMILY, 457 CatalogFamilyFormat.getServerColumn(0)); 458 Cell startCodeCell = result.getColumnLatestCell(HConstants.CATALOG_FAMILY, 459 CatalogFamilyFormat.getStartCodeColumn(0)); 460 Cell seqNumCell = result.getColumnLatestCell(HConstants.CATALOG_FAMILY, 461 CatalogFamilyFormat.getSeqNumColumn(0)); 462 assertNotNull(serverCell); 463 assertNotNull(startCodeCell); 464 assertNotNull(seqNumCell); 465 assertTrue(serverCell.getValueLength() > 0); 466 assertTrue(startCodeCell.getValueLength() > 0); 467 assertTrue(seqNumCell.getValueLength() > 0); 468 assertEquals(masterSystemTime, serverCell.getTimestamp()); 469 assertEquals(masterSystemTime, startCodeCell.getTimestamp()); 470 assertEquals(masterSystemTime, seqNumCell.getTimestamp()); 471 } 472 } 473 474 public static class SpyingRpcSchedulerFactory extends SimpleRpcSchedulerFactory { 475 @Override 476 public RpcScheduler create(Configuration conf, PriorityFunction priority, Abortable server) { 477 final RpcScheduler delegate = super.create(conf, priority, server); 478 return new SpyingRpcScheduler(delegate); 479 } 480 } 481 482 public static class SpyingRpcScheduler extends DelegatingRpcScheduler { 483 long numPriorityCalls = 0; 484 485 public SpyingRpcScheduler(RpcScheduler delegate) { 486 super(delegate); 487 } 488 489 @Override 490 public boolean dispatch(CallRunner task) { 491 int priority = task.getRpcCall().getPriority(); 492 493 if (priority > HConstants.QOS_THRESHOLD) { 494 numPriorityCalls++; 495 } 496 return super.dispatch(task); 497 } 498 } 499 500 @Test 501 public void testScanByRegionEncodedNameExistingRegion() throws Exception { 502 final TableName tableName = TableName.valueOf("testScanByRegionEncodedNameExistingRegion"); 503 UTIL.createTable(tableName, "cf"); 504 final List<HRegion> regions = UTIL.getHBaseCluster().getRegions(tableName); 505 final String encodedName = regions.get(0).getRegionInfo().getEncodedName(); 506 final Result result = 507 MetaTableAccessor.scanByRegionEncodedName(UTIL.getConnection(), encodedName); 508 assertNotNull(result); 509 assertTrue(result.advance()); 510 final String resultingRowKey = CellUtil.getCellKeyAsString(result.current()); 511 assertTrue(resultingRowKey.contains(encodedName)); 512 UTIL.deleteTable(tableName); 513 } 514 515 @Test 516 public void testScanByRegionEncodedNameNonExistingRegion() throws Exception { 517 final String encodedName = "nonexistingregion"; 518 final Result result = 519 MetaTableAccessor.scanByRegionEncodedName(UTIL.getConnection(), encodedName); 520 assertNull(result); 521 } 522}