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.replication.regionserver;
019
020import static org.apache.hadoop.hbase.client.RegionLocator.LOCATOR_META_REPLICAS_MODE;
021import static org.junit.Assert.assertArrayEquals;
022import static org.junit.Assert.assertEquals;
023import static org.junit.Assert.assertFalse;
024import static org.junit.Assert.assertNotNull;
025import static org.junit.Assert.assertTrue;
026
027import java.io.IOException;
028import java.util.ArrayList;
029import java.util.Arrays;
030import java.util.List;
031import java.util.Objects;
032import org.apache.hadoop.conf.Configuration;
033import org.apache.hadoop.hbase.Cell;
034import org.apache.hadoop.hbase.CellScanner;
035import org.apache.hadoop.hbase.CellUtil;
036import org.apache.hadoop.hbase.HBaseClassTestRule;
037import org.apache.hadoop.hbase.HBaseTestingUtility;
038import org.apache.hadoop.hbase.HConstants;
039import org.apache.hadoop.hbase.HRegionLocation;
040import org.apache.hadoop.hbase.MetaTableAccessor;
041import org.apache.hadoop.hbase.MiniHBaseCluster;
042import org.apache.hadoop.hbase.TableName;
043import org.apache.hadoop.hbase.Waiter;
044import org.apache.hadoop.hbase.client.Connection;
045import org.apache.hadoop.hbase.client.ConnectionFactory;
046import org.apache.hadoop.hbase.client.Get;
047import org.apache.hadoop.hbase.client.RegionInfo;
048import org.apache.hadoop.hbase.client.RegionLocator;
049import org.apache.hadoop.hbase.client.Result;
050import org.apache.hadoop.hbase.client.Scan;
051import org.apache.hadoop.hbase.client.Table;
052import org.apache.hadoop.hbase.regionserver.HRegion;
053import org.apache.hadoop.hbase.regionserver.HRegionServer;
054import org.apache.hadoop.hbase.regionserver.Region;
055import org.apache.hadoop.hbase.regionserver.RegionScanner;
056import org.apache.hadoop.hbase.testclassification.LargeTests;
057import org.apache.hadoop.hbase.util.Bytes;
058import org.apache.hadoop.hbase.util.ServerRegionReplicaUtil;
059import org.junit.After;
060import org.junit.Before;
061import org.junit.ClassRule;
062import org.junit.Rule;
063import org.junit.Test;
064import org.junit.experimental.categories.Category;
065import org.junit.rules.TestName;
066import org.slf4j.Logger;
067import org.slf4j.LoggerFactory;
068
069/**
070 * Tests RegionReplicaReplicationEndpoint class for hbase:meta by setting up region replicas and
071 * verifying async wal replication replays the edits to the secondary region in various scenarios.
072 * @see TestRegionReplicaReplicationEndpoint
073 */
074@Category({ LargeTests.class })
075public class TestMetaRegionReplicaReplicationEndpoint {
076  @ClassRule
077  public static final HBaseClassTestRule CLASS_RULE =
078    HBaseClassTestRule.forClass(TestMetaRegionReplicaReplicationEndpoint.class);
079  private static final Logger LOG =
080    LoggerFactory.getLogger(TestMetaRegionReplicaReplicationEndpoint.class);
081  private static final int NB_SERVERS = 4;
082  private final HBaseTestingUtility HTU = new HBaseTestingUtility();
083  private int numOfMetaReplica = NB_SERVERS - 1;
084  private static byte[] VALUE = Bytes.toBytes("value");
085
086  @Rule
087  public TestName name = new TestName();
088
089  @Before
090  public void before() throws Exception {
091    Configuration conf = HTU.getConfiguration();
092    conf.setFloat("hbase.regionserver.logroll.multiplier", 0.0003f);
093    conf.setInt("replication.source.size.capacity", 10240);
094    conf.setLong("replication.source.sleepforretries", 100);
095    conf.setInt("hbase.regionserver.maxlogs", 10);
096    conf.setLong("hbase.master.logcleaner.ttl", 10);
097    conf.setInt("zookeeper.recovery.retry", 1);
098    conf.setInt("zookeeper.recovery.retry.intervalmill", 10);
099    conf.setLong(HConstants.THREAD_WAKE_FREQUENCY, 100);
100    conf.setInt("replication.stats.thread.period.seconds", 5);
101    conf.setBoolean("hbase.tests.use.shortcircuit.reads", false);
102    conf.setInt(HConstants.HBASE_CLIENT_SERVERSIDE_RETRIES_MULTIPLIER, 1);
103    // Enable hbase:meta replication.
104    conf.setBoolean(ServerRegionReplicaUtil.REGION_REPLICA_REPLICATION_CATALOG_CONF_KEY, true);
105    // Set hbase:meta replicas to be 3.
106    // conf.setInt(HConstants.META_REPLICAS_NUM, numOfMetaReplica);
107    HTU.startMiniCluster(NB_SERVERS);
108    // Enable hbase:meta replication.
109    HBaseTestingUtility.setReplicas(HTU.getAdmin(), TableName.META_TABLE_NAME, numOfMetaReplica);
110
111    HTU.waitFor(30000, () -> HTU.getMiniHBaseCluster().getRegions(TableName.META_TABLE_NAME).size()
112        >= numOfMetaReplica);
113  }
114
115  @After
116  public void after() throws Exception {
117    HTU.shutdownMiniCluster();
118  }
119
120  /**
121   * Assert that the ReplicationSource for hbase:meta gets created when hbase:meta is opened.
122   */
123  @Test
124  public void testHBaseMetaReplicationSourceCreatedOnOpen() throws Exception {
125    MiniHBaseCluster cluster = HTU.getMiniHBaseCluster();
126    HRegionServer hrs = cluster.getRegionServer(cluster.getServerHoldingMeta());
127    // Replicate a row to prove all working.
128    testHBaseMetaReplicatesOneRow(0);
129    assertTrue(isMetaRegionReplicaReplicationSource(hrs));
130    // Now move the hbase:meta and make sure the ReplicationSource is in both places.
131    HRegionServer hrsOther = null;
132    for (int i = 0; i < cluster.getNumLiveRegionServers(); i++) {
133      hrsOther = cluster.getRegionServer(i);
134      if (hrsOther.getServerName().equals(hrs.getServerName())) {
135        hrsOther = null;
136        continue;
137      }
138      break;
139    }
140    assertNotNull(hrsOther);
141    assertFalse(isMetaRegionReplicaReplicationSource(hrsOther));
142    Region meta = null;
143    for (Region region : hrs.getOnlineRegionsLocalContext()) {
144      if (region.getRegionInfo().isMetaRegion()) {
145        meta = region;
146        break;
147      }
148    }
149    assertNotNull(meta);
150    HTU.moveRegionAndWait(meta.getRegionInfo(), hrsOther.getServerName());
151    // Assert that there is a ReplicationSource in both places now.
152    assertTrue(isMetaRegionReplicaReplicationSource(hrs));
153    assertTrue(isMetaRegionReplicaReplicationSource(hrsOther));
154    // Replicate to show stuff still works.
155    testHBaseMetaReplicatesOneRow(1);
156    // Now pretend a few hours have gone by... roll the meta WAL in original location... Move the
157    // meta back and retry replication. See if it works.
158    hrs.getWAL(meta.getRegionInfo()).rollWriter(true);
159    testHBaseMetaReplicatesOneRow(2);
160    hrs.getWAL(meta.getRegionInfo()).rollWriter(true);
161    testHBaseMetaReplicatesOneRow(3);
162  }
163
164  /**
165   * Test meta region replica replication. Create some tables and see if replicas pick up the
166   * additions.
167   */
168  private void testHBaseMetaReplicatesOneRow(int i) throws Exception {
169    waitForMetaReplicasToOnline();
170    try (Table table = HTU.createTable(TableName.valueOf(this.name.getMethodName() + "_" + i),
171      HConstants.CATALOG_FAMILY)) {
172      verifyReplication(TableName.META_TABLE_NAME, numOfMetaReplica, getMetaCells(table.getName()));
173    }
174  }
175
176  /** Returns Whether the special meta region replica peer is enabled on <code>hrs</code> */
177  private boolean isMetaRegionReplicaReplicationSource(HRegionServer hrs) {
178    return hrs.getReplicationSourceService().getReplicationManager().catalogReplicationSource.get()
179        != null;
180  }
181
182  /**
183   * Test meta region replica replication. Create some tables and see if replicas pick up the
184   * additions.
185   */
186  @Test
187  public void testHBaseMetaReplicates() throws Exception {
188    try (Table table = HTU.createTable(TableName.valueOf(this.name.getMethodName() + "_0"),
189      HConstants.CATALOG_FAMILY,
190      Arrays.copyOfRange(HBaseTestingUtility.KEYS, 1, HBaseTestingUtility.KEYS.length))) {
191      verifyReplication(TableName.META_TABLE_NAME, numOfMetaReplica, getMetaCells(table.getName()));
192    }
193    try (Table table = HTU.createTable(TableName.valueOf(this.name.getMethodName() + "_1"),
194      HConstants.CATALOG_FAMILY,
195      Arrays.copyOfRange(HBaseTestingUtility.KEYS, 1, HBaseTestingUtility.KEYS.length))) {
196      verifyReplication(TableName.META_TABLE_NAME, numOfMetaReplica, getMetaCells(table.getName()));
197      // Try delete.
198      HTU.deleteTableIfAny(table.getName());
199      verifyDeletedReplication(TableName.META_TABLE_NAME, numOfMetaReplica, table.getName());
200    }
201  }
202
203  @Test
204  public void testCatalogReplicaReplicationWithFlushAndCompaction() throws Exception {
205    Connection connection = ConnectionFactory.createConnection(HTU.getConfiguration());
206    TableName tableName = TableName.valueOf("hbase:meta");
207    Table table = connection.getTable(tableName);
208    try {
209      // load the data to the table
210      for (int i = 0; i < 5; i++) {
211        LOG.info("Writing data from " + i * 1000 + " to " + (i * 1000 + 1000));
212        HTU.loadNumericRows(table, HConstants.CATALOG_FAMILY, i * 1000, i * 1000 + 1000);
213        LOG.info("flushing table");
214        HTU.flush(tableName);
215        LOG.info("compacting table");
216        if (i < 4) {
217          HTU.compact(tableName, false);
218        }
219      }
220
221      verifyReplication(tableName, numOfMetaReplica, 0, 5000, HConstants.CATALOG_FAMILY);
222    } finally {
223      table.close();
224      connection.close();
225    }
226  }
227
228  @Test
229  public void testCatalogReplicaReplicationWithReplicaMoved() throws Exception {
230    MiniHBaseCluster cluster = HTU.getMiniHBaseCluster();
231    HRegionServer hrs = cluster.getRegionServer(cluster.getServerHoldingMeta());
232
233    HRegionServer hrsMetaReplica = null;
234    HRegionServer hrsNoMetaReplica = null;
235    HRegionServer server = null;
236    Region metaReplica = null;
237    boolean hostingMeta;
238
239    for (int i = 0; i < cluster.getNumLiveRegionServers(); i++) {
240      server = cluster.getRegionServer(i);
241      hostingMeta = false;
242      if (server == hrs) {
243        continue;
244      }
245      for (Region region : server.getOnlineRegionsLocalContext()) {
246        if (region.getRegionInfo().isMetaRegion()) {
247          if (metaReplica == null) {
248            metaReplica = region;
249          }
250          hostingMeta = true;
251          break;
252        }
253      }
254      if (!hostingMeta) {
255        hrsNoMetaReplica = server;
256      }
257    }
258
259    Connection connection = ConnectionFactory.createConnection(HTU.getConfiguration());
260    TableName tableName = TableName.valueOf("hbase:meta");
261    Table table = connection.getTable(tableName);
262    try {
263      // load the data to the table
264      for (int i = 0; i < 5; i++) {
265        LOG.info("Writing data from " + i * 1000 + " to " + (i * 1000 + 1000));
266        HTU.loadNumericRows(table, HConstants.CATALOG_FAMILY, i * 1000, i * 1000 + 1000);
267        if (i == 0) {
268          HTU.moveRegionAndWait(metaReplica.getRegionInfo(), hrsNoMetaReplica.getServerName());
269        }
270      }
271
272      verifyReplication(tableName, numOfMetaReplica, 0, 5000, HConstants.CATALOG_FAMILY);
273    } finally {
274      table.close();
275      connection.close();
276    }
277  }
278
279  protected void verifyReplication(TableName tableName, int regionReplication, final int startRow,
280    final int endRow, final byte[] family) throws Exception {
281    verifyReplication(tableName, regionReplication, startRow, endRow, family, true);
282  }
283
284  private void verifyReplication(TableName tableName, int regionReplication, final int startRow,
285    final int endRow, final byte[] family, final boolean present) throws Exception {
286    // find the regions
287    final Region[] regions = new Region[regionReplication];
288
289    for (int i = 0; i < NB_SERVERS; i++) {
290      HRegionServer rs = HTU.getMiniHBaseCluster().getRegionServer(i);
291      List<HRegion> onlineRegions = rs.getRegions(tableName);
292      for (HRegion region : onlineRegions) {
293        regions[region.getRegionInfo().getReplicaId()] = region;
294      }
295    }
296
297    for (Region region : regions) {
298      assertNotNull(region);
299    }
300
301    for (int i = 1; i < regionReplication; i++) {
302      final Region region = regions[i];
303      // wait until all the data is replicated to all secondary regions
304      Waiter.waitFor(HTU.getConfiguration(), 90000, 1000, new Waiter.Predicate<Exception>() {
305        @Override
306        public boolean evaluate() throws Exception {
307          LOG.info("verifying replication for region replica:" + region.getRegionInfo());
308          try {
309            HTU.verifyNumericRows(region, family, startRow, endRow, present);
310          } catch (Throwable ex) {
311            LOG.warn("Verification from secondary region is not complete yet", ex);
312            // still wait
313            return false;
314          }
315          return true;
316        }
317      });
318    }
319  }
320
321  /**
322   * Replicas come online after primary.
323   */
324  private void waitForMetaReplicasToOnline() throws IOException {
325    final RegionLocator regionLocator =
326      HTU.getConnection().getRegionLocator(TableName.META_TABLE_NAME);
327    HTU.waitFor(10000,
328      // getRegionLocations returns an entry for each replica but if unassigned, entry is null.
329      // Pass reload to force us to skip cache else it just keeps returning default.
330      () -> regionLocator.getRegionLocations(HConstants.EMPTY_START_ROW, true).stream()
331        .filter(Objects::nonNull).count() >= numOfMetaReplica);
332    List<HRegionLocation> locations = regionLocator.getRegionLocations(HConstants.EMPTY_START_ROW);
333    LOG.info("Found locations {}", locations);
334    assertEquals(numOfMetaReplica, locations.size());
335  }
336
337  /**
338   * Scan hbase:meta for <code>tableName</code> content.
339   */
340  private List<Result> getMetaCells(TableName tableName) throws IOException {
341    final List<Result> results = new ArrayList<>();
342    MetaTableAccessor.Visitor visitor = new MetaTableAccessor.Visitor() {
343      @Override
344      public boolean visit(Result r) throws IOException {
345        results.add(r);
346        return true;
347      }
348    };
349    MetaTableAccessor.scanMetaForTableRegions(HTU.getConnection(), visitor, tableName);
350    return results;
351  }
352
353  /** Returns All Regions for tableName including Replicas. */
354  private Region[] getAllRegions(TableName tableName, int replication) {
355    final Region[] regions = new Region[replication];
356    for (int i = 0; i < NB_SERVERS; i++) {
357      HRegionServer rs = HTU.getMiniHBaseCluster().getRegionServer(i);
358      List<HRegion> onlineRegions = rs.getRegions(tableName);
359      for (HRegion region : onlineRegions) {
360        regions[region.getRegionInfo().getReplicaId()] = region;
361      }
362    }
363    for (Region region : regions) {
364      assertNotNull(region);
365    }
366    return regions;
367  }
368
369  private Region getOneRegion(TableName tableName) {
370    for (int i = 0; i < NB_SERVERS; i++) {
371      HRegionServer rs = HTU.getMiniHBaseCluster().getRegionServer(i);
372      List<HRegion> onlineRegions = rs.getRegions(tableName);
373      if (onlineRegions.size() > 1) {
374        return onlineRegions.get(0);
375      }
376    }
377    return null;
378  }
379
380  /**
381   * Verify when a Table is deleted from primary, then there are no references in replicas (because
382   * they get the delete of the table rows too).
383   */
384  private void verifyDeletedReplication(TableName tableName, int regionReplication,
385    final TableName deletedTableName) {
386    final Region[] regions = getAllRegions(tableName, regionReplication);
387
388    // Start count at '1' so we skip default, primary replica and only look at secondaries.
389    for (int i = 1; i < regionReplication; i++) {
390      final Region region = regions[i];
391      // wait until all the data is replicated to all secondary regions
392      Waiter.waitFor(HTU.getConfiguration(), 30000, 1000, new Waiter.Predicate<Exception>() {
393        @Override
394        public boolean evaluate() throws Exception {
395          LOG.info("Verifying replication for region replica {}", region.getRegionInfo());
396          try (RegionScanner rs = region.getScanner(new Scan())) {
397            List<Cell> cells = new ArrayList<>();
398            while (rs.next(cells)) {
399              continue;
400            }
401            return doesNotContain(cells, deletedTableName);
402          } catch (Throwable ex) {
403            LOG.warn("Verification from secondary region is not complete yet", ex);
404            // still wait
405            return false;
406          }
407        }
408      });
409    }
410  }
411
412  /**
413   * Cells are from hbase:meta replica so will start w/ 'tableName,'; i.e. the tablename followed by
414   * HConstants.DELIMITER. Make sure the deleted table is no longer present in passed
415   * <code>cells</code>.
416   */
417  private boolean doesNotContain(List<Cell> cells, TableName tableName) {
418    for (Cell cell : cells) {
419      String row = Bytes.toString(cell.getRowArray(), cell.getRowOffset(), cell.getRowLength());
420      if (row.startsWith(tableName.toString() + HConstants.DELIMITER)) {
421        return false;
422      }
423    }
424    return true;
425  }
426
427  /**
428   * Verify Replicas have results (exactly).
429   */
430  private void verifyReplication(TableName tableName, int regionReplication,
431    List<Result> contains) {
432    final Region[] regions = getAllRegions(tableName, regionReplication);
433
434    // Start count at '1' so we skip default, primary replica and only look at secondaries.
435    for (int i = 1; i < regionReplication; i++) {
436      final Region region = regions[i];
437      // wait until all the data is replicated to all secondary regions
438      Waiter.waitFor(HTU.getConfiguration(), 30000, 1000, new Waiter.Predicate<Exception>() {
439        @Override
440        public boolean evaluate() throws Exception {
441          LOG.info("Verifying replication for region replica {}", region.getRegionInfo());
442          try (RegionScanner rs = region.getScanner(new Scan())) {
443            List<Cell> cells = new ArrayList<>();
444            while (rs.next(cells)) {
445              continue;
446            }
447            return contains(contains, cells);
448          } catch (Throwable ex) {
449            LOG.warn("Verification from secondary region is not complete yet", ex);
450            // still wait
451            return false;
452          }
453        }
454      });
455    }
456  }
457
458  /**
459   * Presumes sorted Cells. Verify that <code>cells</code> has <code>contains</code> at least.
460   */
461  static boolean contains(List<Result> contains, List<Cell> cells) throws IOException {
462    CellScanner containsScanner = CellUtil.createCellScanner(contains);
463    CellScanner cellsScanner = CellUtil.createCellScanner(cells);
464    int matches = 0;
465    int count = 0;
466    while (containsScanner.advance()) {
467      while (cellsScanner.advance()) {
468        count++;
469        LOG.info("{} {}", containsScanner.current(), cellsScanner.current());
470        if (containsScanner.current().equals(cellsScanner.current())) {
471          matches++;
472          break;
473        }
474      }
475    }
476    return !containsScanner.advance() && matches >= 1 && count >= matches && count == cells.size();
477  }
478
479  private void doNGets(final Table table, final byte[][] keys) throws Exception {
480    for (byte[] key : keys) {
481      Result r = table.get(new Get(key));
482      assertArrayEquals(VALUE, r.getValue(HConstants.CATALOG_FAMILY, HConstants.CATALOG_FAMILY));
483    }
484  }
485
486  private void primaryNoChangeReplicaIncrease(final long[] before, final long[] after) {
487    assertEquals(before[RegionInfo.DEFAULT_REPLICA_ID], after[RegionInfo.DEFAULT_REPLICA_ID]);
488
489    for (int i = 1; i < after.length; i++) {
490      assertTrue(after[i] > before[i]);
491    }
492  }
493
494  private void primaryIncreaseReplicaNoChange(final long[] before, final long[] after) {
495    // There are read requests increase for primary meta replica.
496    assertTrue(after[RegionInfo.DEFAULT_REPLICA_ID] > before[RegionInfo.DEFAULT_REPLICA_ID]);
497
498    // No change for replica regions
499    for (int i = 1; i < after.length; i++) {
500      assertEquals(before[i], after[i]);
501    }
502  }
503
504  private void primaryMayIncreaseReplicaNoChange(final long[] before, final long[] after) {
505    // For primary meta replica, scan request may increase. No change for replica meta regions.
506    assertTrue(after[RegionInfo.DEFAULT_REPLICA_ID] >= before[RegionInfo.DEFAULT_REPLICA_ID]);
507
508    // No change for replica regions
509    for (int i = 1; i < after.length; i++) {
510      assertEquals(before[i], after[i]);
511    }
512  }
513
514  private void primaryIncreaseReplicaIncrease(final long[] before, final long[] after) {
515    // There are read requests increase for all meta replica regions,
516    for (int i = 0; i < after.length; i++) {
517      assertTrue(after[i] > before[i]);
518    }
519  }
520
521  private void getMetaReplicaReadRequests(final Region[] metaRegions, final long[] counters) {
522    int i = 0;
523    for (Region r : metaRegions) {
524      LOG.info("read request for region {} is {}", r, r.getReadRequestsCount());
525      counters[i] = r.getReadRequestsCount();
526      i++;
527    }
528  }
529
530  @Test
531  public void testHBaseMetaReplicaGets() throws Exception {
532
533    TableName tn = TableName.valueOf(this.name.getMethodName());
534    final Region[] metaRegions = getAllRegions(TableName.META_TABLE_NAME, numOfMetaReplica);
535    long[] readReqsForMetaReplicas = new long[numOfMetaReplica];
536    long[] readReqsForMetaReplicasAfterGet = new long[numOfMetaReplica];
537    long[] readReqsForMetaReplicasAfterGetAllLocations = new long[numOfMetaReplica];
538    long[] readReqsForMetaReplicasAfterMove = new long[numOfMetaReplica];
539    long[] readReqsForMetaReplicasAfterSecondMove = new long[numOfMetaReplica];
540    long[] readReqsForMetaReplicasAfterThirdGet = new long[numOfMetaReplica];
541    Region userRegion = null;
542    HRegionServer srcRs = null;
543    HRegionServer destRs = null;
544
545    try (Table table = HTU.createTable(tn, HConstants.CATALOG_FAMILY,
546      Arrays.copyOfRange(HBaseTestingUtility.KEYS, 1, HBaseTestingUtility.KEYS.length))) {
547      verifyReplication(TableName.META_TABLE_NAME, numOfMetaReplica, getMetaCells(table.getName()));
548      // load different values
549      HTU.loadTable(table, new byte[][] { HConstants.CATALOG_FAMILY }, VALUE);
550      for (int i = 0; i < NB_SERVERS; i++) {
551        HRegionServer rs = HTU.getMiniHBaseCluster().getRegionServer(i);
552        List<HRegion> onlineRegions = rs.getRegions(tn);
553        if (onlineRegions.size() > 0) {
554          userRegion = onlineRegions.get(0);
555          srcRs = rs;
556          if (i > 0) {
557            destRs = HTU.getMiniHBaseCluster().getRegionServer(0);
558          } else {
559            destRs = HTU.getMiniHBaseCluster().getRegionServer(1);
560          }
561        }
562      }
563
564      getMetaReplicaReadRequests(metaRegions, readReqsForMetaReplicas);
565
566      Configuration c = new Configuration(HTU.getConfiguration());
567      c.set(LOCATOR_META_REPLICAS_MODE, "LoadBalance");
568      Connection connection = ConnectionFactory.createConnection(c);
569      Table tableForGet = connection.getTable(tn);
570      byte[][] getRows = new byte[HBaseTestingUtility.KEYS.length][];
571
572      int i = 0;
573      for (byte[] key : HBaseTestingUtility.KEYS) {
574        getRows[i] = key;
575        i++;
576      }
577      getRows[0] = Bytes.toBytes("aaa");
578      doNGets(tableForGet, getRows);
579
580      getMetaReplicaReadRequests(metaRegions, readReqsForMetaReplicasAfterGet);
581
582      // There are more reads against all meta replica regions, including the primary region.
583      primaryIncreaseReplicaIncrease(readReqsForMetaReplicas, readReqsForMetaReplicasAfterGet);
584
585      RegionLocator locator = tableForGet.getRegionLocator();
586
587      for (int j = 0; j < numOfMetaReplica * 3; j++) {
588        locator.getAllRegionLocations();
589      }
590
591      getMetaReplicaReadRequests(metaRegions, readReqsForMetaReplicasAfterGetAllLocations);
592      primaryIncreaseReplicaIncrease(readReqsForMetaReplicasAfterGet,
593        readReqsForMetaReplicasAfterGetAllLocations);
594
595      // move one of regions so it meta cache may be invalid.
596      HTU.moveRegionAndWait(userRegion.getRegionInfo(), destRs.getServerName());
597
598      doNGets(tableForGet, getRows);
599
600      getMetaReplicaReadRequests(metaRegions, readReqsForMetaReplicasAfterMove);
601
602      // There are read requests increase for primary meta replica.
603      // For rest of meta replicas, there is no change as regionMove will tell the new location
604      primaryIncreaseReplicaNoChange(readReqsForMetaReplicasAfterGetAllLocations,
605        readReqsForMetaReplicasAfterMove);
606      // Move region again.
607      HTU.moveRegionAndWait(userRegion.getRegionInfo(), srcRs.getServerName());
608
609      // Wait until moveRegion cache timeout.
610      while (destRs.getMovedRegion(userRegion.getRegionInfo().getEncodedName()) != null) {
611        Thread.sleep(1000);
612      }
613
614      getMetaReplicaReadRequests(metaRegions, readReqsForMetaReplicasAfterSecondMove);
615
616      // There may be read requests increase for primary meta replica.
617      // For rest of meta replicas, there is no change.
618      primaryMayIncreaseReplicaNoChange(readReqsForMetaReplicasAfterMove,
619        readReqsForMetaReplicasAfterSecondMove);
620
621      doNGets(tableForGet, getRows);
622
623      getMetaReplicaReadRequests(metaRegions, readReqsForMetaReplicasAfterThirdGet);
624
625      // Since it gets RegionNotServedException, it will go to primary for the next lookup.
626      // There are read requests increase for primary meta replica.
627      // For rest of meta replicas, there is no change.
628      primaryIncreaseReplicaNoChange(readReqsForMetaReplicasAfterSecondMove,
629        readReqsForMetaReplicasAfterThirdGet);
630    }
631  }
632}