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.apache.hadoop.hbase.client.RegionLocator.LOCATOR_META_REPLICAS_MODE;
021import static org.apache.hadoop.hbase.util.FutureUtils.addListener;
022
023import java.io.IOException;
024import java.util.ArrayList;
025import java.util.Arrays;
026import java.util.Collections;
027import java.util.List;
028import java.util.Map;
029import java.util.NavigableMap;
030import java.util.Optional;
031import java.util.SortedMap;
032import java.util.concurrent.CompletableFuture;
033import java.util.concurrent.ThreadLocalRandom;
034import java.util.regex.Matcher;
035import java.util.regex.Pattern;
036import java.util.stream.Collectors;
037import org.apache.hadoop.hbase.MetaTableAccessor.CollectingVisitor;
038import org.apache.hadoop.hbase.MetaTableAccessor.QueryType;
039import org.apache.hadoop.hbase.MetaTableAccessor.Visitor;
040import org.apache.hadoop.hbase.client.AdvancedScanResultConsumer;
041import org.apache.hadoop.hbase.client.AsyncTable;
042import org.apache.hadoop.hbase.client.Consistency;
043import org.apache.hadoop.hbase.client.Get;
044import org.apache.hadoop.hbase.client.RegionInfo;
045import org.apache.hadoop.hbase.client.RegionReplicaUtil;
046import org.apache.hadoop.hbase.client.Result;
047import org.apache.hadoop.hbase.client.Scan;
048import org.apache.hadoop.hbase.client.Scan.ReadType;
049import org.apache.hadoop.hbase.client.TableState;
050import org.apache.hadoop.hbase.exceptions.DeserializationException;
051import org.apache.hadoop.hbase.util.Bytes;
052import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
053import org.apache.hadoop.hbase.util.Pair;
054import org.apache.yetus.audience.InterfaceAudience;
055import org.slf4j.Logger;
056import org.slf4j.LoggerFactory;
057
058/**
059 * The asynchronous meta table accessor. Used to read/write region and assignment information store
060 * in <code>hbase:meta</code>.
061 * @since 2.0.0
062 */
063@InterfaceAudience.Private
064public class AsyncMetaTableAccessor {
065
066  private static final Logger LOG = LoggerFactory.getLogger(AsyncMetaTableAccessor.class);
067
068  /** The delimiter for meta columns for replicaIds &gt; 0 */
069  private static final char META_REPLICA_ID_DELIMITER = '_';
070
071  /** A regex for parsing server columns from meta. See above javadoc for meta layout */
072  private static final Pattern SERVER_COLUMN_PATTERN =
073    Pattern.compile("^server(_[0-9a-fA-F]{4})?$");
074
075  public static CompletableFuture<Boolean> tableExists(AsyncTable<?> metaTable,
076    TableName tableName) {
077    return getTableState(metaTable, tableName).thenApply(Optional::isPresent);
078  }
079
080  public static CompletableFuture<Optional<TableState>> getTableState(AsyncTable<?> metaTable,
081    TableName tableName) {
082    CompletableFuture<Optional<TableState>> future = new CompletableFuture<>();
083    Get get = new Get(tableName.getName()).addColumn(getTableFamily(), getStateColumn());
084    long time = EnvironmentEdgeManager.currentTime();
085    try {
086      get.setTimeRange(0, time);
087      addListener(metaTable.get(get), (result, error) -> {
088        if (error != null) {
089          future.completeExceptionally(error);
090          return;
091        }
092        try {
093          future.complete(getTableState(result));
094        } catch (IOException e) {
095          future.completeExceptionally(e);
096        }
097      });
098    } catch (IOException ioe) {
099      future.completeExceptionally(ioe);
100    }
101    return future;
102  }
103
104  /** Returns the HRegionLocation from meta for the given region */
105  public static CompletableFuture<Optional<HRegionLocation>>
106    getRegionLocation(AsyncTable<?> metaTable, byte[] regionName) {
107    CompletableFuture<Optional<HRegionLocation>> future = new CompletableFuture<>();
108    try {
109      RegionInfo parsedRegionInfo = MetaTableAccessor.parseRegionInfoFromRegionName(regionName);
110      addListener(metaTable.get(new Get(MetaTableAccessor.getMetaKeyForRegion(parsedRegionInfo))
111        .addFamily(HConstants.CATALOG_FAMILY)), (r, err) -> {
112          if (err != null) {
113            future.completeExceptionally(err);
114            return;
115          }
116          future.complete(getRegionLocations(r)
117            .map(locations -> locations.getRegionLocation(parsedRegionInfo.getReplicaId())));
118        });
119    } catch (IOException parseEx) {
120      LOG.warn("Failed to parse the passed region name: " + Bytes.toStringBinary(regionName));
121      future.completeExceptionally(parseEx);
122    }
123    return future;
124  }
125
126  /** Returns the HRegionLocation from meta for the given encoded region name */
127  public static CompletableFuture<Optional<HRegionLocation>>
128    getRegionLocationWithEncodedName(AsyncTable<?> metaTable, byte[] encodedRegionName) {
129    CompletableFuture<Optional<HRegionLocation>> future = new CompletableFuture<>();
130    addListener(
131      metaTable
132        .scanAll(new Scan().setReadType(ReadType.PREAD).addFamily(HConstants.CATALOG_FAMILY)),
133      (results, err) -> {
134        if (err != null) {
135          future.completeExceptionally(err);
136          return;
137        }
138        String encodedRegionNameStr = Bytes.toString(encodedRegionName);
139        results.stream().filter(result -> !result.isEmpty())
140          .filter(result -> MetaTableAccessor.getRegionInfo(result) != null).forEach(result -> {
141            getRegionLocations(result).ifPresent(locations -> {
142              for (HRegionLocation location : locations.getRegionLocations()) {
143                if (
144                  location != null
145                    && encodedRegionNameStr.equals(location.getRegion().getEncodedName())
146                ) {
147                  future.complete(Optional.of(location));
148                  return;
149                }
150              }
151            });
152          });
153        future.complete(Optional.empty());
154      });
155    return future;
156  }
157
158  private static Optional<TableState> getTableState(Result r) throws IOException {
159    Cell cell = r.getColumnLatestCell(getTableFamily(), getStateColumn());
160    if (cell == null) return Optional.empty();
161    try {
162      return Optional.of(
163        TableState.parseFrom(TableName.valueOf(r.getRow()), Arrays.copyOfRange(cell.getValueArray(),
164          cell.getValueOffset(), cell.getValueOffset() + cell.getValueLength())));
165    } catch (DeserializationException e) {
166      throw new IOException("Failed to parse table state from result: " + r, e);
167    }
168  }
169
170  /**
171   * Used to get all region locations for the specific table
172   * @param metaTable scanner over meta table
173   * @param tableName table we're looking for, can be null for getting all regions
174   * @return the list of region locations. The return value will be wrapped by a
175   *         {@link CompletableFuture}.
176   */
177  public static CompletableFuture<List<HRegionLocation>> getTableHRegionLocations(
178    AsyncTable<AdvancedScanResultConsumer> metaTable, TableName tableName) {
179    CompletableFuture<List<HRegionLocation>> future = new CompletableFuture<>();
180    addListener(getTableRegionsAndLocations(metaTable, tableName, true), (locations, err) -> {
181      if (err != null) {
182        future.completeExceptionally(err);
183      } else if (locations == null || locations.isEmpty()) {
184        future.complete(Collections.emptyList());
185      } else {
186        List<HRegionLocation> regionLocations =
187          locations.stream().map(loc -> new HRegionLocation(loc.getFirst(), loc.getSecond()))
188            .collect(Collectors.toList());
189        future.complete(regionLocations);
190      }
191    });
192    return future;
193  }
194
195  /**
196   * Used to get table regions' info and server.
197   * @param metaTable                   scanner over meta table
198   * @param tableName                   table we're looking for, can be null for getting all regions
199   * @param excludeOfflinedSplitParents don't return split parents
200   * @return the list of regioninfos and server. The return value will be wrapped by a
201   *         {@link CompletableFuture}.
202   */
203  private static CompletableFuture<List<Pair<RegionInfo, ServerName>>> getTableRegionsAndLocations(
204    final AsyncTable<AdvancedScanResultConsumer> metaTable, final TableName tableName,
205    final boolean excludeOfflinedSplitParents) {
206    CompletableFuture<List<Pair<RegionInfo, ServerName>>> future = new CompletableFuture<>();
207    if (TableName.META_TABLE_NAME.equals(tableName)) {
208      future.completeExceptionally(new IOException(
209        "This method can't be used to locate meta regions;" + " use MetaTableLocator instead"));
210    }
211
212    // Make a version of CollectingVisitor that collects RegionInfo and ServerAddress
213    CollectingVisitor<Pair<RegionInfo, ServerName>> visitor =
214      new CollectingVisitor<Pair<RegionInfo, ServerName>>() {
215        private RegionLocations current = null;
216
217        @Override
218        public boolean visit(Result r) throws IOException {
219          Optional<RegionLocations> currentRegionLocations = getRegionLocations(r);
220          current = currentRegionLocations.orElse(null);
221          if (current == null || current.getRegionLocation().getRegion() == null) {
222            LOG.warn("No serialized RegionInfo in " + r);
223            return true;
224          }
225          RegionInfo hri = current.getRegionLocation().getRegion();
226          if (excludeOfflinedSplitParents && hri.isSplitParent()) return true;
227          // Else call super and add this Result to the collection.
228          return super.visit(r);
229        }
230
231        @Override
232        void add(Result r) {
233          if (current == null) {
234            return;
235          }
236          for (HRegionLocation loc : current.getRegionLocations()) {
237            if (loc != null) {
238              this.results
239                .add(new Pair<RegionInfo, ServerName>(loc.getRegion(), loc.getServerName()));
240            }
241          }
242        }
243      };
244
245    addListener(scanMeta(metaTable, tableName, QueryType.REGION, visitor), (v, error) -> {
246      if (error != null) {
247        future.completeExceptionally(error);
248        return;
249      }
250      future.complete(visitor.getResults());
251    });
252    return future;
253  }
254
255  /**
256   * Performs a scan of META table for given table.
257   * @param metaTable scanner over meta table
258   * @param tableName table within we scan
259   * @param type      scanned part of meta
260   * @param visitor   Visitor invoked against each row
261   */
262  private static CompletableFuture<Void> scanMeta(AsyncTable<AdvancedScanResultConsumer> metaTable,
263    TableName tableName, QueryType type, final Visitor visitor) {
264    return scanMeta(metaTable, getTableStartRowForMeta(tableName, type),
265      getTableStopRowForMeta(tableName, type), type, Integer.MAX_VALUE, visitor);
266  }
267
268  /**
269   * Performs a scan of META table for given table.
270   * @param metaTable scanner over meta table
271   * @param startRow  Where to start the scan
272   * @param stopRow   Where to stop the scan
273   * @param type      scanned part of meta
274   * @param maxRows   maximum rows to return
275   * @param visitor   Visitor invoked against each row
276   */
277  private static CompletableFuture<Void> scanMeta(AsyncTable<AdvancedScanResultConsumer> metaTable,
278    byte[] startRow, byte[] stopRow, QueryType type, int maxRows, final Visitor visitor) {
279    int rowUpperLimit = maxRows > 0 ? maxRows : Integer.MAX_VALUE;
280    Scan scan = getMetaScan(metaTable, rowUpperLimit);
281    for (byte[] family : type.getFamilies()) {
282      scan.addFamily(family);
283    }
284    if (startRow != null) {
285      scan.withStartRow(startRow);
286    }
287    if (stopRow != null) {
288      scan.withStopRow(stopRow);
289    }
290
291    if (LOG.isDebugEnabled()) {
292      LOG.debug("Scanning META" + " starting at row=" + Bytes.toStringBinary(scan.getStartRow())
293        + " stopping at row=" + Bytes.toStringBinary(scan.getStopRow()) + " for max="
294        + rowUpperLimit + " with caching=" + scan.getCaching());
295    }
296
297    CompletableFuture<Void> future = new CompletableFuture<Void>();
298    // Get the region locator's meta replica mode.
299    CatalogReplicaMode metaReplicaMode = CatalogReplicaMode.fromString(metaTable.getConfiguration()
300      .get(LOCATOR_META_REPLICAS_MODE, CatalogReplicaMode.NONE.toString()));
301
302    if (metaReplicaMode == CatalogReplicaMode.LOAD_BALANCE) {
303      addListener(metaTable.getDescriptor(), (desc, error) -> {
304        if (error != null) {
305          LOG.error("Failed to get meta table descriptor, error: ", error);
306          future.completeExceptionally(error);
307          return;
308        }
309
310        int numOfReplicas = desc.getRegionReplication();
311        if (numOfReplicas > 1) {
312          int replicaId = ThreadLocalRandom.current().nextInt(numOfReplicas);
313
314          // When the replicaId is 0, do not set to Consistency.TIMELINE
315          if (replicaId > 0) {
316            scan.setReplicaId(replicaId);
317            scan.setConsistency(Consistency.TIMELINE);
318          }
319        }
320        metaTable.scan(scan, new MetaTableScanResultConsumer(rowUpperLimit, visitor, future));
321      });
322    } else {
323      if (metaReplicaMode == CatalogReplicaMode.HEDGED_READ) {
324        scan.setConsistency(Consistency.TIMELINE);
325      }
326      metaTable.scan(scan, new MetaTableScanResultConsumer(rowUpperLimit, visitor, future));
327    }
328
329    return future;
330  }
331
332  private static final class MetaTableScanResultConsumer implements AdvancedScanResultConsumer {
333
334    private int currentRowCount;
335
336    private final int rowUpperLimit;
337
338    private final Visitor visitor;
339
340    private final CompletableFuture<Void> future;
341
342    MetaTableScanResultConsumer(int rowUpperLimit, Visitor visitor,
343      CompletableFuture<Void> future) {
344      this.rowUpperLimit = rowUpperLimit;
345      this.visitor = visitor;
346      this.future = future;
347      this.currentRowCount = 0;
348    }
349
350    @Override
351    public void onError(Throwable error) {
352      future.completeExceptionally(error);
353    }
354
355    @Override
356    @edu.umd.cs.findbugs.annotations.SuppressWarnings(value = "NP_NONNULL_PARAM_VIOLATION",
357        justification = "https://github.com/findbugsproject/findbugs/issues/79")
358    public void onComplete() {
359      future.complete(null);
360    }
361
362    @Override
363    public void onNext(Result[] results, ScanController controller) {
364      boolean terminateScan = false;
365      for (Result result : results) {
366        try {
367          if (!visitor.visit(result)) {
368            terminateScan = true;
369            break;
370          }
371        } catch (Exception e) {
372          future.completeExceptionally(e);
373          terminateScan = true;
374          break;
375        }
376        if (++currentRowCount >= rowUpperLimit) {
377          terminateScan = true;
378          break;
379        }
380      }
381      if (terminateScan) {
382        controller.terminate();
383      }
384    }
385  }
386
387  private static Scan getMetaScan(AsyncTable<?> metaTable, int rowUpperLimit) {
388    Scan scan = new Scan();
389    int scannerCaching = metaTable.getConfiguration().getInt(HConstants.HBASE_META_SCANNER_CACHING,
390      HConstants.DEFAULT_HBASE_META_SCANNER_CACHING);
391    if (
392      metaTable.getConfiguration().getBoolean(HConstants.USE_META_REPLICAS,
393        HConstants.DEFAULT_USE_META_REPLICAS)
394    ) {
395      scan.setConsistency(Consistency.TIMELINE);
396    }
397    if (rowUpperLimit <= scannerCaching) {
398      scan.setLimit(rowUpperLimit);
399    }
400    int rows = Math.min(rowUpperLimit, scannerCaching);
401    scan.setCaching(rows);
402    return scan;
403  }
404
405  /**
406   * Returns an HRegionLocationList extracted from the result.
407   * @return an HRegionLocationList containing all locations for the region range or null if we
408   *         can't deserialize the result.
409   */
410  private static Optional<RegionLocations> getRegionLocations(final Result r) {
411    if (r == null) {
412      return Optional.empty();
413    }
414    Optional<RegionInfo> regionInfo = getHRegionInfo(r, getRegionInfoColumn());
415    if (!regionInfo.isPresent()) {
416      return Optional.empty();
417    }
418
419    List<HRegionLocation> locations = new ArrayList<HRegionLocation>(1);
420    NavigableMap<byte[], NavigableMap<byte[], byte[]>> familyMap = r.getNoVersionMap();
421
422    locations.add(getRegionLocation(r, regionInfo.get(), 0));
423
424    NavigableMap<byte[], byte[]> infoMap = familyMap.get(getCatalogFamily());
425    if (infoMap == null) {
426      return Optional.of(new RegionLocations(locations));
427    }
428
429    // iterate until all serverName columns are seen
430    int replicaId = 0;
431    byte[] serverColumn = getServerColumn(replicaId);
432    SortedMap<byte[], byte[]> serverMap = infoMap.tailMap(serverColumn, false);
433
434    if (serverMap.isEmpty()) {
435      return Optional.of(new RegionLocations(locations));
436    }
437
438    for (Map.Entry<byte[], byte[]> entry : serverMap.entrySet()) {
439      replicaId = parseReplicaIdFromServerColumn(entry.getKey());
440      if (replicaId < 0) {
441        break;
442      }
443      HRegionLocation location = getRegionLocation(r, regionInfo.get(), replicaId);
444      // In case the region replica is newly created, it's location might be null. We usually do not
445      // have HRL's in RegionLocations object with null ServerName. They are handled as null HRLs.
446      if (location == null || location.getServerName() == null) {
447        locations.add(null);
448      } else {
449        locations.add(location);
450      }
451    }
452
453    return Optional.of(new RegionLocations(locations));
454  }
455
456  /**
457   * Returns the HRegionLocation parsed from the given meta row Result for the given regionInfo and
458   * replicaId. The regionInfo can be the default region info for the replica.
459   * @param r          the meta row result
460   * @param regionInfo RegionInfo for default replica
461   * @param replicaId  the replicaId for the HRegionLocation
462   * @return HRegionLocation parsed from the given meta row Result for the given replicaId
463   */
464  private static HRegionLocation getRegionLocation(final Result r, final RegionInfo regionInfo,
465    final int replicaId) {
466    Optional<ServerName> serverName = getServerName(r, replicaId);
467    long seqNum = getSeqNumDuringOpen(r, replicaId);
468    RegionInfo replicaInfo = RegionReplicaUtil.getRegionInfoForReplica(regionInfo, replicaId);
469    return new HRegionLocation(replicaInfo, serverName.orElse(null), seqNum);
470  }
471
472  /**
473   * Returns a {@link ServerName} from catalog table {@link Result}.
474   * @param r Result to pull from
475   * @return A ServerName instance.
476   */
477  private static Optional<ServerName> getServerName(final Result r, final int replicaId) {
478    byte[] serverColumn = getServerColumn(replicaId);
479    Cell cell = r.getColumnLatestCell(getCatalogFamily(), serverColumn);
480    if (cell == null || cell.getValueLength() == 0) return Optional.empty();
481    String hostAndPort =
482      Bytes.toString(cell.getValueArray(), cell.getValueOffset(), cell.getValueLength());
483    byte[] startcodeColumn = getStartCodeColumn(replicaId);
484    cell = r.getColumnLatestCell(getCatalogFamily(), startcodeColumn);
485    if (cell == null || cell.getValueLength() == 0) return Optional.empty();
486    try {
487      return Optional.of(ServerName.valueOf(hostAndPort,
488        Bytes.toLong(cell.getValueArray(), cell.getValueOffset(), cell.getValueLength())));
489    } catch (IllegalArgumentException e) {
490      LOG.error("Ignoring invalid region for server " + hostAndPort + "; cell=" + cell, e);
491      return Optional.empty();
492    }
493  }
494
495  /**
496   * The latest seqnum that the server writing to meta observed when opening the region. E.g. the
497   * seqNum when the result of {@link #getServerName(Result, int)} was written.
498   * @param r Result to pull the seqNum from
499   * @return SeqNum, or HConstants.NO_SEQNUM if there's no value written.
500   */
501  private static long getSeqNumDuringOpen(final Result r, final int replicaId) {
502    Cell cell = r.getColumnLatestCell(getCatalogFamily(), getSeqNumColumn(replicaId));
503    if (cell == null || cell.getValueLength() == 0) return HConstants.NO_SEQNUM;
504    return Bytes.toLong(cell.getValueArray(), cell.getValueOffset(), cell.getValueLength());
505  }
506
507  /**
508   * @param tableName table we're working with
509   * @return start row for scanning META according to query type
510   */
511  private static byte[] getTableStartRowForMeta(TableName tableName, QueryType type) {
512    if (tableName == null) {
513      return null;
514    }
515    switch (type) {
516      case REGION:
517      case REPLICATION: {
518        byte[] startRow = new byte[tableName.getName().length + 2];
519        System.arraycopy(tableName.getName(), 0, startRow, 0, tableName.getName().length);
520        startRow[startRow.length - 2] = HConstants.DELIMITER;
521        startRow[startRow.length - 1] = HConstants.DELIMITER;
522        return startRow;
523      }
524      case ALL:
525      case TABLE:
526      default: {
527        return tableName.getName();
528      }
529    }
530  }
531
532  /**
533   * @param tableName table we're working with
534   * @return stop row for scanning META according to query type
535   */
536  private static byte[] getTableStopRowForMeta(TableName tableName, QueryType type) {
537    if (tableName == null) {
538      return null;
539    }
540    final byte[] stopRow;
541    switch (type) {
542      case REGION:
543      case REPLICATION: {
544        stopRow = new byte[tableName.getName().length + 3];
545        System.arraycopy(tableName.getName(), 0, stopRow, 0, tableName.getName().length);
546        stopRow[stopRow.length - 3] = ' ';
547        stopRow[stopRow.length - 2] = HConstants.DELIMITER;
548        stopRow[stopRow.length - 1] = HConstants.DELIMITER;
549        break;
550      }
551      case ALL:
552      case TABLE:
553      default: {
554        stopRow = new byte[tableName.getName().length + 1];
555        System.arraycopy(tableName.getName(), 0, stopRow, 0, tableName.getName().length);
556        stopRow[stopRow.length - 1] = ' ';
557        break;
558      }
559    }
560    return stopRow;
561  }
562
563  /**
564   * Returns the RegionInfo object from the column {@link HConstants#CATALOG_FAMILY} and
565   * <code>qualifier</code> of the catalog table result.
566   * @param r         a Result object from the catalog table scan
567   * @param qualifier Column family qualifier
568   * @return An RegionInfo instance.
569   */
570  private static Optional<RegionInfo> getHRegionInfo(final Result r, byte[] qualifier) {
571    Cell cell = r.getColumnLatestCell(getCatalogFamily(), qualifier);
572    if (cell == null) return Optional.empty();
573    return Optional.ofNullable(RegionInfo.parseFromOrNull(cell.getValueArray(),
574      cell.getValueOffset(), cell.getValueLength()));
575  }
576
577  /**
578   * Returns the column family used for meta columns.
579   * @return HConstants.CATALOG_FAMILY.
580   */
581  private static byte[] getCatalogFamily() {
582    return HConstants.CATALOG_FAMILY;
583  }
584
585  /**
586   * Returns the column family used for table columns.
587   * @return HConstants.TABLE_FAMILY.
588   */
589  private static byte[] getTableFamily() {
590    return HConstants.TABLE_FAMILY;
591  }
592
593  /**
594   * Returns the column qualifier for serialized region info
595   * @return HConstants.REGIONINFO_QUALIFIER
596   */
597  private static byte[] getRegionInfoColumn() {
598    return HConstants.REGIONINFO_QUALIFIER;
599  }
600
601  /**
602   * Returns the column qualifier for serialized table state
603   * @return HConstants.TABLE_STATE_QUALIFIER
604   */
605  private static byte[] getStateColumn() {
606    return HConstants.TABLE_STATE_QUALIFIER;
607  }
608
609  /**
610   * Returns the column qualifier for server column for replicaId
611   * @param replicaId the replicaId of the region
612   * @return a byte[] for server column qualifier
613   */
614  private static byte[] getServerColumn(int replicaId) {
615    return replicaId == 0
616      ? HConstants.SERVER_QUALIFIER
617      : Bytes.toBytes(HConstants.SERVER_QUALIFIER_STR + META_REPLICA_ID_DELIMITER
618        + String.format(RegionInfo.REPLICA_ID_FORMAT, replicaId));
619  }
620
621  /**
622   * Returns the column qualifier for server start code column for replicaId
623   * @param replicaId the replicaId of the region
624   * @return a byte[] for server start code column qualifier
625   */
626  private static byte[] getStartCodeColumn(int replicaId) {
627    return replicaId == 0
628      ? HConstants.STARTCODE_QUALIFIER
629      : Bytes.toBytes(HConstants.STARTCODE_QUALIFIER_STR + META_REPLICA_ID_DELIMITER
630        + String.format(RegionInfo.REPLICA_ID_FORMAT, replicaId));
631  }
632
633  /**
634   * Returns the column qualifier for seqNum column for replicaId
635   * @param replicaId the replicaId of the region
636   * @return a byte[] for seqNum column qualifier
637   */
638  private static byte[] getSeqNumColumn(int replicaId) {
639    return replicaId == 0
640      ? HConstants.SEQNUM_QUALIFIER
641      : Bytes.toBytes(HConstants.SEQNUM_QUALIFIER_STR + META_REPLICA_ID_DELIMITER
642        + String.format(RegionInfo.REPLICA_ID_FORMAT, replicaId));
643  }
644
645  /**
646   * Parses the replicaId from the server column qualifier. See top of the class javadoc for the
647   * actual meta layout
648   * @param serverColumn the column qualifier
649   * @return an int for the replicaId
650   */
651  private static int parseReplicaIdFromServerColumn(byte[] serverColumn) {
652    String serverStr = Bytes.toString(serverColumn);
653
654    Matcher matcher = SERVER_COLUMN_PATTERN.matcher(serverStr);
655    if (matcher.matches() && matcher.groupCount() > 0) {
656      String group = matcher.group(1);
657      if (group != null && group.length() > 0) {
658        return Integer.parseInt(group.substring(1), 16);
659      } else {
660        return 0;
661      }
662    }
663    return -1;
664  }
665}