001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.client;
019
020import static org.apache.hadoop.hbase.HConstants.DEFAULT_USE_META_REPLICAS;
021import static org.apache.hadoop.hbase.HConstants.EMPTY_END_ROW;
022import static org.apache.hadoop.hbase.HConstants.NINES;
023import static org.apache.hadoop.hbase.HConstants.USE_META_REPLICAS;
024import static org.apache.hadoop.hbase.HConstants.ZEROES;
025import static org.apache.hadoop.hbase.TableName.META_TABLE_NAME;
026import static org.apache.hadoop.hbase.client.AsyncRegionLocatorHelper.createRegionLocations;
027import static org.apache.hadoop.hbase.client.AsyncRegionLocatorHelper.isGood;
028import static org.apache.hadoop.hbase.client.ConnectionUtils.createClosestRowAfter;
029import static org.apache.hadoop.hbase.client.ConnectionUtils.isEmptyStopRow;
030import static org.apache.hadoop.hbase.client.RegionInfo.createRegionName;
031import static org.apache.hadoop.hbase.client.RegionLocator.LOCATOR_META_REPLICAS_MODE;
032import static org.apache.hadoop.hbase.util.ConcurrentMapUtils.computeIfAbsent;
033
034import java.io.IOException;
035import java.util.ArrayList;
036import java.util.Arrays;
037import java.util.HashSet;
038import java.util.Iterator;
039import java.util.LinkedHashMap;
040import java.util.List;
041import java.util.Map;
042import java.util.Optional;
043import java.util.Set;
044import java.util.concurrent.CompletableFuture;
045import java.util.concurrent.ConcurrentHashMap;
046import java.util.concurrent.ConcurrentMap;
047import java.util.concurrent.TimeUnit;
048import org.apache.commons.lang3.ObjectUtils;
049import org.apache.hadoop.hbase.CatalogReplicaMode;
050import org.apache.hadoop.hbase.HBaseIOException;
051import org.apache.hadoop.hbase.HConstants;
052import org.apache.hadoop.hbase.HRegionLocation;
053import org.apache.hadoop.hbase.MetaTableAccessor;
054import org.apache.hadoop.hbase.RegionLocations;
055import org.apache.hadoop.hbase.ServerName;
056import org.apache.hadoop.hbase.TableName;
057import org.apache.hadoop.hbase.TableNotFoundException;
058import org.apache.hadoop.hbase.client.Scan.ReadType;
059import org.apache.hadoop.hbase.util.Bytes;
060import org.apache.yetus.audience.InterfaceAudience;
061import org.slf4j.Logger;
062import org.slf4j.LoggerFactory;
063
064/**
065 * The asynchronous locator for regions other than meta.
066 */
067@InterfaceAudience.Private
068class AsyncNonMetaRegionLocator {
069
070  private static final Logger LOG = LoggerFactory.getLogger(AsyncNonMetaRegionLocator.class);
071
072  static final String MAX_CONCURRENT_LOCATE_REQUEST_PER_TABLE =
073    "hbase.client.meta.max.concurrent.locate.per.table";
074
075  private static final int DEFAULT_MAX_CONCURRENT_LOCATE_REQUEST_PER_TABLE = 8;
076
077  static String LOCATE_PREFETCH_LIMIT = "hbase.client.locate.prefetch.limit";
078
079  private static final int DEFAULT_LOCATE_PREFETCH_LIMIT = 10;
080
081  private final AsyncConnectionImpl conn;
082
083  private final int maxConcurrentLocateRequestPerTable;
084
085  private final int locatePrefetchLimit;
086
087  // The mode tells if HedgedRead, LoadBalance mode is supported.
088  // The default mode is CatalogReplicaMode.None.
089  private CatalogReplicaMode metaReplicaMode;
090  private CatalogReplicaLoadBalanceSelector metaReplicaSelector;
091
092  private final ConcurrentMap<TableName, TableCache> cache = new ConcurrentHashMap<>();
093
094  private static final class LocateRequest {
095
096    private final byte[] row;
097
098    private final RegionLocateType locateType;
099
100    public LocateRequest(byte[] row, RegionLocateType locateType) {
101      this.row = row;
102      this.locateType = locateType;
103    }
104
105    @Override
106    public int hashCode() {
107      return Bytes.hashCode(row) ^ locateType.hashCode();
108    }
109
110    @Override
111    public boolean equals(Object obj) {
112      if (obj == null || obj.getClass() != LocateRequest.class) {
113        return false;
114      }
115      LocateRequest that = (LocateRequest) obj;
116      return locateType.equals(that.locateType) && Bytes.equals(row, that.row);
117    }
118  }
119
120  private static final class RegionLocationsFutureResult {
121    private final CompletableFuture<RegionLocations> future;
122    private final RegionLocations result;
123    private final Throwable e;
124
125    public RegionLocationsFutureResult(CompletableFuture<RegionLocations> future,
126      RegionLocations result, Throwable e) {
127      this.future = future;
128      this.result = result;
129      this.e = e;
130    }
131
132    public void complete() {
133      if (e != null) {
134        future.completeExceptionally(e);
135      }
136      future.complete(result);
137    }
138  }
139
140  private static final class TableCache {
141
142    private final Set<LocateRequest> pendingRequests = new HashSet<>();
143
144    private final Map<LocateRequest, CompletableFuture<RegionLocations>> allRequests =
145      new LinkedHashMap<>();
146    private final AsyncRegionLocationCache regionLocationCache;
147
148    public TableCache(TableName tableName) {
149      regionLocationCache = new AsyncRegionLocationCache(tableName);
150    }
151
152    public boolean hasQuota(int max) {
153      return pendingRequests.size() < max;
154    }
155
156    public boolean isPending(LocateRequest req) {
157      return pendingRequests.contains(req);
158    }
159
160    public void send(LocateRequest req) {
161      pendingRequests.add(req);
162    }
163
164    public Optional<LocateRequest> getCandidate() {
165      return allRequests.keySet().stream().filter(r -> !isPending(r)).findFirst();
166    }
167
168    public List<RegionLocationsFutureResult> clearCompletedRequests(RegionLocations locations) {
169      List<RegionLocationsFutureResult> futureResultList = new ArrayList<>();
170      for (Iterator<Map.Entry<LocateRequest, CompletableFuture<RegionLocations>>> iter =
171        allRequests.entrySet().iterator(); iter.hasNext();) {
172        Map.Entry<LocateRequest, CompletableFuture<RegionLocations>> entry = iter.next();
173        if (tryComplete(entry.getKey(), entry.getValue(), locations, futureResultList)) {
174          iter.remove();
175        }
176      }
177      return futureResultList;
178    }
179
180    private boolean tryComplete(LocateRequest req, CompletableFuture<RegionLocations> future,
181      RegionLocations locations, List<RegionLocationsFutureResult> futureResultList) {
182      if (future.isDone()) {
183        return true;
184      }
185      if (locations == null) {
186        return false;
187      }
188      HRegionLocation loc = ObjectUtils.firstNonNull(locations.getRegionLocations());
189      // we should at least have one location available, otherwise the request should fail and
190      // should not arrive here
191      assert loc != null;
192      boolean completed;
193      if (req.locateType.equals(RegionLocateType.BEFORE)) {
194        // for locating the row before current row, the common case is to find the previous region
195        // in reverse scan, so we check the endKey first. In general, the condition should be
196        // startKey < req.row and endKey >= req.row. Here we split it to endKey == req.row ||
197        // (endKey > req.row && startKey < req.row). The two conditions are equal since startKey <
198        // endKey.
199        byte[] endKey = loc.getRegion().getEndKey();
200        int c = Bytes.compareTo(endKey, req.row);
201        completed = c == 0 || ((c > 0 || Bytes.equals(EMPTY_END_ROW, endKey))
202          && Bytes.compareTo(loc.getRegion().getStartKey(), req.row) < 0);
203      } else {
204        completed = loc.getRegion().containsRow(req.row);
205      }
206      if (completed) {
207        futureResultList.add(new RegionLocationsFutureResult(future, locations, null));
208        return true;
209      } else {
210        return false;
211      }
212    }
213  }
214
215  AsyncNonMetaRegionLocator(AsyncConnectionImpl conn) {
216    this.conn = conn;
217    this.maxConcurrentLocateRequestPerTable = conn.getConfiguration().getInt(
218      MAX_CONCURRENT_LOCATE_REQUEST_PER_TABLE, DEFAULT_MAX_CONCURRENT_LOCATE_REQUEST_PER_TABLE);
219    this.locatePrefetchLimit =
220      conn.getConfiguration().getInt(LOCATE_PREFETCH_LIMIT, DEFAULT_LOCATE_PREFETCH_LIMIT);
221
222    // Get the region locator's meta replica mode.
223    this.metaReplicaMode = CatalogReplicaMode.fromString(
224      conn.getConfiguration().get(LOCATOR_META_REPLICAS_MODE, CatalogReplicaMode.NONE.toString()));
225
226    switch (this.metaReplicaMode) {
227      case LOAD_BALANCE:
228        String replicaSelectorClass =
229          conn.getConfiguration().get(RegionLocator.LOCATOR_META_REPLICAS_MODE_LOADBALANCE_SELECTOR,
230            CatalogReplicaLoadBalanceSimpleSelector.class.getName());
231
232        this.metaReplicaSelector = CatalogReplicaLoadBalanceSelectorFactory
233          .createSelector(replicaSelectorClass, META_TABLE_NAME, conn.getChoreService(), () -> {
234            int numOfReplicas = CatalogReplicaLoadBalanceSelector.UNINITIALIZED_NUM_OF_REPLICAS;
235            try {
236              RegionLocations metaLocations = conn.registry.getMetaRegionLocations()
237                .get(conn.connConf.getReadRpcTimeoutNs(), TimeUnit.NANOSECONDS);
238              numOfReplicas = metaLocations.size();
239            } catch (Exception e) {
240              LOG.error("Failed to get table {}'s region replication, ", META_TABLE_NAME, e);
241            }
242            return numOfReplicas;
243          });
244        break;
245      case NONE:
246        // If user does not configure LOCATOR_META_REPLICAS_MODE, let's check the legacy config.
247
248        boolean useMetaReplicas =
249          conn.getConfiguration().getBoolean(USE_META_REPLICAS, DEFAULT_USE_META_REPLICAS);
250        if (useMetaReplicas) {
251          this.metaReplicaMode = CatalogReplicaMode.HEDGED_READ;
252        }
253        break;
254      default:
255        // Doing nothing
256    }
257  }
258
259  private TableCache getTableCache(TableName tableName) {
260    return computeIfAbsent(cache, tableName, () -> new TableCache(tableName));
261  }
262
263  private void complete(TableName tableName, LocateRequest req, RegionLocations locs,
264    Throwable error) {
265    if (error != null) {
266      LOG.warn("Failed to locate region in '" + tableName + "', row='"
267        + Bytes.toStringBinary(req.row) + "', locateType=" + req.locateType, error);
268    }
269    Optional<LocateRequest> toSend = Optional.empty();
270    TableCache tableCache = getTableCache(tableName);
271    if (locs != null) {
272      RegionLocations addedLocs = tableCache.regionLocationCache.add(locs);
273      List<RegionLocationsFutureResult> futureResultList = new ArrayList<>();
274      synchronized (tableCache) {
275        tableCache.pendingRequests.remove(req);
276        futureResultList.addAll(tableCache.clearCompletedRequests(addedLocs));
277        // Remove a complete locate request in a synchronized block, so the table cache must have
278        // quota to send a candidate request.
279        toSend = tableCache.getCandidate();
280        toSend.ifPresent(r -> tableCache.send(r));
281      }
282      futureResultList.forEach(RegionLocationsFutureResult::complete);
283      toSend.ifPresent(r -> locateInMeta(tableName, r));
284    } else {
285      // we meet an error
286      assert error != null;
287      List<RegionLocationsFutureResult> futureResultList = new ArrayList<>();
288      synchronized (tableCache) {
289        tableCache.pendingRequests.remove(req);
290        // fail the request itself, no matter whether it is a DoNotRetryIOException, as we have
291        // already retried several times
292        CompletableFuture<RegionLocations> future = tableCache.allRequests.remove(req);
293        if (future != null) {
294          futureResultList.add(new RegionLocationsFutureResult(future, null, error));
295        }
296        futureResultList.addAll(tableCache.clearCompletedRequests(null));
297        // Remove a complete locate request in a synchronized block, so the table cache must have
298        // quota to send a candidate request.
299        toSend = tableCache.getCandidate();
300        toSend.ifPresent(r -> tableCache.send(r));
301      }
302      futureResultList.forEach(RegionLocationsFutureResult::complete);
303      toSend.ifPresent(r -> locateInMeta(tableName, r));
304    }
305  }
306
307  // return whether we should stop the scan
308  private boolean onScanNext(TableName tableName, LocateRequest req, Result result) {
309    RegionLocations locs = MetaTableAccessor.getRegionLocations(result);
310    if (LOG.isDebugEnabled()) {
311      LOG.debug("The fetched location of '{}', row='{}', locateType={} is {}", tableName,
312        Bytes.toStringBinary(req.row), req.locateType, locs);
313    }
314    // remove HRegionLocation with null location, i.e, getServerName returns null.
315    if (locs != null) {
316      locs = locs.removeElementsWithNullLocation();
317    }
318
319    // the default region location should always be presented when fetching from meta, otherwise
320    // let's fail the request.
321    if (locs == null || locs.getDefaultRegionLocation() == null) {
322      complete(tableName, req, null,
323        new HBaseIOException(String.format("No location found for '%s', row='%s', locateType=%s",
324          tableName, Bytes.toStringBinary(req.row), req.locateType)));
325      return true;
326    }
327    HRegionLocation loc = locs.getDefaultRegionLocation();
328    RegionInfo info = loc.getRegion();
329    if (info == null) {
330      complete(tableName, req, null,
331        new HBaseIOException(String.format("HRegionInfo is null for '%s', row='%s', locateType=%s",
332          tableName, Bytes.toStringBinary(req.row), req.locateType)));
333      return true;
334    }
335    if (info.isSplitParent()) {
336      return false;
337    }
338    complete(tableName, req, locs, null);
339    return true;
340  }
341
342  private void recordCacheHit() {
343    conn.getConnectionMetrics().ifPresent(MetricsConnection::incrMetaCacheHit);
344  }
345
346  private void recordCacheMiss() {
347    conn.getConnectionMetrics().ifPresent(MetricsConnection::incrMetaCacheMiss);
348  }
349
350  private RegionLocations locateRowInCache(TableCache tableCache, byte[] row, int replicaId) {
351    RegionLocations locs = tableCache.regionLocationCache.findForRow(row, replicaId);
352    if (locs == null) {
353      recordCacheMiss();
354    } else {
355      recordCacheHit();
356    }
357    return locs;
358  }
359
360  private RegionLocations locateRowBeforeInCache(TableCache tableCache, byte[] row, int replicaId) {
361    RegionLocations locs = tableCache.regionLocationCache.findForBeforeRow(row, replicaId);
362    if (locs == null) {
363      recordCacheMiss();
364    } else {
365      recordCacheHit();
366    }
367    return locs;
368  }
369
370  private void locateInMeta(TableName tableName, LocateRequest req) {
371    if (LOG.isTraceEnabled()) {
372      LOG.trace("Try locate '" + tableName + "', row='" + Bytes.toStringBinary(req.row)
373        + "', locateType=" + req.locateType + " in meta");
374    }
375    byte[] metaStartKey;
376    if (req.locateType.equals(RegionLocateType.BEFORE)) {
377      if (isEmptyStopRow(req.row)) {
378        byte[] binaryTableName = tableName.getName();
379        metaStartKey = Arrays.copyOf(binaryTableName, binaryTableName.length + 1);
380      } else {
381        metaStartKey = createRegionName(tableName, req.row, ZEROES, false);
382      }
383    } else {
384      metaStartKey = createRegionName(tableName, req.row, NINES, false);
385    }
386    byte[] metaStopKey =
387      RegionInfo.createRegionName(tableName, HConstants.EMPTY_START_ROW, "", false);
388    Scan scan = new Scan().withStartRow(metaStartKey).withStopRow(metaStopKey, true)
389      .addFamily(HConstants.CATALOG_FAMILY).setReversed(true).setCaching(locatePrefetchLimit)
390      .setReadType(ReadType.PREAD);
391
392    switch (this.metaReplicaMode) {
393      case LOAD_BALANCE:
394        int metaReplicaId = this.metaReplicaSelector.select(tableName, req.row, req.locateType);
395        if (metaReplicaId != RegionInfo.DEFAULT_REPLICA_ID) {
396          // If the selector gives a non-primary meta replica region, then go with it.
397          // Otherwise, just go to primary in non-hedgedRead mode.
398          scan.setConsistency(Consistency.TIMELINE);
399          scan.setReplicaId(metaReplicaId);
400        }
401        break;
402      case HEDGED_READ:
403        scan.setConsistency(Consistency.TIMELINE);
404        break;
405      default:
406        // do nothing
407    }
408
409    conn.getTable(META_TABLE_NAME).scan(scan, new AdvancedScanResultConsumer() {
410
411      private boolean completeNormally = false;
412
413      private boolean tableNotFound = true;
414
415      @Override
416      public void onError(Throwable error) {
417        complete(tableName, req, null, error);
418      }
419
420      @Override
421      public void onComplete() {
422        if (tableNotFound) {
423          complete(tableName, req, null, new TableNotFoundException(tableName));
424        } else if (!completeNormally) {
425          complete(tableName, req, null, new IOException(
426            "Unable to find region for '" + Bytes.toStringBinary(req.row) + "' in " + tableName));
427        }
428      }
429
430      @Override
431      public void onNext(Result[] results, ScanController controller) {
432        if (results.length == 0) {
433          return;
434        }
435        tableNotFound = false;
436        int i = 0;
437        for (; i < results.length; i++) {
438          if (onScanNext(tableName, req, results[i])) {
439            completeNormally = true;
440            controller.terminate();
441            i++;
442            break;
443          }
444        }
445        // Add the remaining results into cache
446        if (i < results.length) {
447          TableCache tableCache = getTableCache(tableName);
448          for (; i < results.length; i++) {
449            RegionLocations locs = MetaTableAccessor.getRegionLocations(results[i]);
450            if (locs == null) {
451              continue;
452            }
453            HRegionLocation loc = locs.getDefaultRegionLocation();
454            if (loc == null) {
455              continue;
456            }
457            RegionInfo info = loc.getRegion();
458            if (info == null || info.isOffline() || info.isSplitParent()) {
459              continue;
460            }
461            RegionLocations addedLocs = tableCache.regionLocationCache.add(locs);
462            List<RegionLocationsFutureResult> futureResultList = new ArrayList<>();
463            synchronized (tableCache) {
464              futureResultList.addAll(tableCache.clearCompletedRequests(addedLocs));
465            }
466            futureResultList.forEach(RegionLocationsFutureResult::complete);
467          }
468        }
469      }
470    });
471  }
472
473  private RegionLocations locateInCache(TableCache tableCache, byte[] row, int replicaId,
474    RegionLocateType locateType) {
475    return locateType.equals(RegionLocateType.BEFORE)
476      ? locateRowBeforeInCache(tableCache, row, replicaId)
477      : locateRowInCache(tableCache, row, replicaId);
478  }
479
480  // locateToPrevious is true means we will use the start key of a region to locate the region
481  // placed before it. Used for reverse scan. See the comment of
482  // AsyncRegionLocator.getPreviousRegionLocation.
483  private CompletableFuture<RegionLocations> getRegionLocationsInternal(TableName tableName,
484    byte[] row, int replicaId, RegionLocateType locateType, boolean reload) {
485    // AFTER should be convert to CURRENT before calling this method
486    assert !locateType.equals(RegionLocateType.AFTER);
487    TableCache tableCache = getTableCache(tableName);
488    if (!reload) {
489      RegionLocations locs = locateInCache(tableCache, row, replicaId, locateType);
490      if (isGood(locs, replicaId)) {
491        return CompletableFuture.completedFuture(locs);
492      }
493    }
494    CompletableFuture<RegionLocations> future;
495    LocateRequest req;
496    boolean sendRequest = false;
497    synchronized (tableCache) {
498      // check again
499      if (!reload) {
500        RegionLocations locs = locateInCache(tableCache, row, replicaId, locateType);
501        if (isGood(locs, replicaId)) {
502          return CompletableFuture.completedFuture(locs);
503        }
504      }
505      req = new LocateRequest(row, locateType);
506      future = tableCache.allRequests.get(req);
507      if (future == null) {
508        future = new CompletableFuture<>();
509        tableCache.allRequests.put(req, future);
510        if (tableCache.hasQuota(maxConcurrentLocateRequestPerTable) && !tableCache.isPending(req)) {
511          tableCache.send(req);
512          sendRequest = true;
513        }
514      }
515    }
516    if (sendRequest) {
517      locateInMeta(tableName, req);
518    }
519    return future;
520  }
521
522  CompletableFuture<RegionLocations> getRegionLocations(TableName tableName, byte[] row,
523    int replicaId, RegionLocateType locateType, boolean reload) {
524    // as we know the exact row after us, so we can just create the new row, and use the same
525    // algorithm to locate it.
526    if (locateType.equals(RegionLocateType.AFTER)) {
527      row = createClosestRowAfter(row);
528      locateType = RegionLocateType.CURRENT;
529    }
530    return getRegionLocationsInternal(tableName, row, replicaId, locateType, reload);
531  }
532
533  private void recordClearRegionCache() {
534    conn.getConnectionMetrics().ifPresent(MetricsConnection::incrMetaCacheNumClearRegion);
535  }
536
537  private void removeLocationFromCache(HRegionLocation loc) {
538    TableCache tableCache = cache.get(loc.getRegion().getTable());
539    if (tableCache != null) {
540      if (tableCache.regionLocationCache.remove(loc)) {
541        recordClearRegionCache();
542        updateMetaReplicaSelector(loc);
543      }
544    }
545  }
546
547  private void updateMetaReplicaSelector(HRegionLocation loc) {
548    // Tell metaReplicaSelector that the location is stale. It will create a stale entry
549    // with timestamp internally. Next time the client looks up the same location,
550    // it will pick a different meta replica region.
551    if (metaReplicaMode == CatalogReplicaMode.LOAD_BALANCE) {
552      metaReplicaSelector.onError(loc);
553    }
554  }
555
556  void addLocationToCache(HRegionLocation loc) {
557    getTableCache(loc.getRegion().getTable()).regionLocationCache.add(createRegionLocations(loc));
558  }
559
560  private HRegionLocation getCachedLocation(HRegionLocation loc) {
561    TableCache tableCache = cache.get(loc.getRegion().getTable());
562    if (tableCache == null) {
563      return null;
564    }
565    RegionLocations locs = tableCache.regionLocationCache.get(loc.getRegion().getStartKey());
566    return locs != null ? locs.getRegionLocation(loc.getRegion().getReplicaId()) : null;
567  }
568
569  void updateCachedLocationOnError(HRegionLocation loc, Throwable exception) {
570    Optional<MetricsConnection> connectionMetrics = conn.getConnectionMetrics();
571    AsyncRegionLocatorHelper.updateCachedLocationOnError(loc, exception, this::getCachedLocation,
572      this::addLocationToCache, this::removeLocationFromCache, connectionMetrics.orElse(null));
573  }
574
575  void clearCache(TableName tableName) {
576    TableCache tableCache = cache.remove(tableName);
577    if (tableCache == null) {
578      return;
579    }
580    List<RegionLocationsFutureResult> futureResultList = new ArrayList<>();
581    synchronized (tableCache) {
582      if (!tableCache.allRequests.isEmpty()) {
583        IOException error = new IOException("Cache cleared");
584        tableCache.allRequests.values().forEach(f -> {
585          futureResultList.add(new RegionLocationsFutureResult(f, null, error));
586        });
587      }
588    }
589    futureResultList.forEach(RegionLocationsFutureResult::complete);
590    conn.getConnectionMetrics().ifPresent(
591      metrics -> metrics.incrMetaCacheNumClearRegion(tableCache.regionLocationCache.size()));
592  }
593
594  void clearCache() {
595    cache.clear();
596  }
597
598  void clearCache(ServerName serverName) {
599    for (TableCache tableCache : cache.values()) {
600      tableCache.regionLocationCache.removeForServer(serverName);
601    }
602  }
603
604  // only used for testing whether we have cached the location for a region.
605  RegionLocations getRegionLocationInCache(TableName tableName, byte[] row) {
606    TableCache tableCache = cache.get(tableName);
607    if (tableCache == null) {
608      return null;
609    }
610    return locateRowInCache(tableCache, row, RegionReplicaUtil.DEFAULT_REPLICA_ID);
611  }
612
613}