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 edu.umd.cs.findbugs.annotations.CheckForNull;
021import java.io.DataInputStream;
022import java.io.IOException;
023import java.util.ArrayList;
024import java.util.Arrays;
025import java.util.Comparator;
026import java.util.List;
027import java.util.stream.Collectors;
028import org.apache.hadoop.hbase.HConstants;
029import org.apache.hadoop.hbase.TableName;
030import org.apache.hadoop.hbase.exceptions.DeserializationException;
031import org.apache.hadoop.hbase.util.ByteArrayHashKey;
032import org.apache.hadoop.hbase.util.Bytes;
033import org.apache.hadoop.hbase.util.HashKey;
034import org.apache.hadoop.hbase.util.JenkinsHash;
035import org.apache.hadoop.hbase.util.MD5Hash;
036import org.apache.hadoop.io.DataInputBuffer;
037import org.apache.hadoop.io.IOUtils;
038import org.apache.yetus.audience.InterfaceAudience;
039
040import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
041import org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos;
042
043/**
044 * Information about a region. A region is a range of keys in the whole keyspace of a table, an
045 * identifier (a timestamp) for differentiating between subset ranges (after region split) and a
046 * replicaId for differentiating the instance for the same range and some status information about
047 * the region. The region has a unique name which consists of the following fields:
048 * <ul>
049 * <li>tableName : The name of the table</li>
050 * <li>startKey : The startKey for the region.</li>
051 * <li>regionId : A timestamp when the region is created.</li>
052 * <li>replicaId : An id starting from 0 to differentiate replicas of the same region range but
053 * hosted in separated servers. The same region range can be hosted in multiple locations.</li>
054 * <li>encodedName : An MD5 encoded string for the region name.</li>
055 * </ul>
056 * <br>
057 * Other than the fields in the region name, region info contains:
058 * <ul>
059 * <li>endKey : the endKey for the region (exclusive)</li>
060 * <li>split : Whether the region is split</li>
061 * <li>offline : Whether the region is offline</li>
062 * </ul>
063 */
064@InterfaceAudience.Public
065public interface RegionInfo extends Comparable<RegionInfo> {
066
067  /**
068   * Separator used to demarcate the encodedName in a region name in the new format. See description
069   * on new format above.
070   */
071  @InterfaceAudience.Private
072  int ENC_SEPARATOR = '.';
073
074  @InterfaceAudience.Private
075  int MD5_HEX_LENGTH = 32;
076
077  @InterfaceAudience.Private
078  int DEFAULT_REPLICA_ID = 0;
079
080  /**
081   * to keep appended int's sorted in string format. Only allows 2 bytes to be sorted for replicaId.
082   */
083  @InterfaceAudience.Private
084  String REPLICA_ID_FORMAT = "%04X";
085
086  @InterfaceAudience.Private
087  byte REPLICA_ID_DELIMITER = (byte) '_';
088
089  @InterfaceAudience.Private
090  String INVALID_REGION_NAME_FORMAT_MESSAGE = "Invalid regionName format";
091
092  @InterfaceAudience.Private
093  Comparator<RegionInfo> COMPARATOR = (RegionInfo lhs, RegionInfo rhs) -> {
094    if (rhs == null) {
095      return 1;
096    }
097
098    // Are regions of same table?
099    int result = lhs.getTable().compareTo(rhs.getTable());
100    if (result != 0) {
101      return result;
102    }
103
104    // Compare start keys.
105    result = Bytes.compareTo(lhs.getStartKey(), rhs.getStartKey());
106    if (result != 0) {
107      return result;
108    }
109
110    // Compare end keys.
111    result = Bytes.compareTo(lhs.getEndKey(), rhs.getEndKey());
112
113    if (result != 0) {
114      if (lhs.getStartKey().length != 0 && lhs.getEndKey().length == 0) {
115        return 1; // this is last region
116      }
117      if (rhs.getStartKey().length != 0 && rhs.getEndKey().length == 0) {
118        return -1; // o is the last region
119      }
120      return result;
121    }
122
123    // regionId is usually milli timestamp -- this defines older stamps
124    // to be "smaller" than newer stamps in sort order.
125    if (lhs.getRegionId() > rhs.getRegionId()) {
126      return 1;
127    } else if (lhs.getRegionId() < rhs.getRegionId()) {
128      return -1;
129    }
130
131    int replicaDiff = lhs.getReplicaId() - rhs.getReplicaId();
132    if (replicaDiff != 0) {
133      return replicaDiff;
134    }
135
136    if (lhs.isOffline() == rhs.isOffline()) {
137      return 0;
138    }
139    if (lhs.isOffline()) {
140      return -1;
141    }
142
143    return 1;
144  };
145
146  /**
147   * Returns Return a short, printable name for this region (usually encoded name) for us logging.
148   */
149  String getShortNameToLog();
150
151  /** Returns the regionId. */
152  long getRegionId();
153
154  /**
155   * Returns the regionName as an array of bytes.
156   * @see #getRegionNameAsString()
157   */
158  byte[] getRegionName();
159
160  /** Returns Region name as a String for use in logging, etc. */
161  String getRegionNameAsString();
162
163  /** Returns the encoded region name. */
164  String getEncodedName();
165
166  /** Returns the encoded region name as an array of bytes. */
167  byte[] getEncodedNameAsBytes();
168
169  /** Returns the startKey. */
170  byte[] getStartKey();
171
172  /** Returns the endKey. */
173  byte[] getEndKey();
174
175  /** Returns current table name of the region */
176  TableName getTable();
177
178  /** Returns returns region replica id */
179  int getReplicaId();
180
181  /** Returns True if has been split and has daughters. */
182  boolean isSplit();
183
184  /**
185   * Returns True if this region is offline.
186   * @deprecated since 3.0.0 and will be removed in 4.0.0
187   * @see <a href="https://issues.apache.org/jira/browse/HBASE-25210">HBASE-25210</a>
188   */
189  @Deprecated
190  boolean isOffline();
191
192  /**
193   * Returns True if this is a split parent region.
194   * @deprecated since 3.0.0 and will be removed in 4.0.0, Use {@link #isSplit()} instead.
195   * @see <a href="https://issues.apache.org/jira/browse/HBASE-25210">HBASE-25210</a>
196   */
197  @Deprecated
198  boolean isSplitParent();
199
200  /** Returns true if this region is a meta region. */
201  boolean isMetaRegion();
202
203  /**
204   * Returns true if the given inclusive range of rows is fully contained by this region. For
205   * example, if the region is foo,a,g and this is passed ["b","c"] or ["a","c"] it will return
206   * true, but if this is passed ["b","z"] it will return false.
207   * @throws IllegalArgumentException if the range passed is invalid (ie. end &lt; start)
208   */
209  boolean containsRange(byte[] rangeStartKey, byte[] rangeEndKey);
210
211  /** Returns true if the given row falls in this region. */
212  boolean containsRow(byte[] row);
213
214  /**
215   * Does region name contain its encoded name?
216   * @param regionName region name
217   * @return boolean indicating if this a new format region name which contains its encoded name.
218   */
219  @InterfaceAudience.Private
220  static boolean hasEncodedName(final byte[] regionName) {
221    // check if region name ends in ENC_SEPARATOR
222    return (regionName.length >= 1)
223      && (regionName[regionName.length - 1] == RegionInfo.ENC_SEPARATOR);
224  }
225
226  /** Returns the encodedName */
227  @InterfaceAudience.Private
228  static String encodeRegionName(final byte[] regionName) {
229    String encodedName;
230    if (hasEncodedName(regionName)) {
231      // region is in new format:
232      // <tableName>,<startKey>,<regionIdTimeStamp>/encodedName/
233      encodedName =
234        Bytes.toString(regionName, regionName.length - MD5_HEX_LENGTH - 1, MD5_HEX_LENGTH);
235    } else {
236      // old format region name. First hbase:meta region also
237      // use this format.EncodedName is the JenkinsHash value.
238      HashKey<byte[]> key = new ByteArrayHashKey(regionName, 0, regionName.length);
239      int hashVal = Math.abs(JenkinsHash.getInstance().hash(key, 0));
240      encodedName = String.valueOf(hashVal);
241    }
242    return encodedName;
243  }
244
245  @InterfaceAudience.Private
246  static String getRegionNameAsString(byte[] regionName) {
247    return getRegionNameAsString(null, regionName);
248  }
249
250  @InterfaceAudience.Private
251  static String getRegionNameAsString(@CheckForNull RegionInfo ri, byte[] regionName) {
252    if (RegionInfo.hasEncodedName(regionName)) {
253      // new format region names already have their encoded name.
254      return Bytes.toStringBinary(regionName);
255    }
256
257    // old format. regionNameStr doesn't have the region name.
258    if (ri == null) {
259      return Bytes.toStringBinary(regionName) + "." + RegionInfo.encodeRegionName(regionName);
260    } else {
261      return Bytes.toStringBinary(regionName) + "." + ri.getEncodedName();
262    }
263  }
264
265  /**
266   * Returns a String of short, printable names for <code>hris</code> (usually encoded name) for us
267   * logging.
268   */
269  static String getShortNameToLog(RegionInfo... hris) {
270    return getShortNameToLog(Arrays.asList(hris));
271  }
272
273  /**
274   * Returns a String of short, printable names for <code>hris</code> (usually encoded name) for us
275   * logging.
276   */
277  static String getShortNameToLog(final List<RegionInfo> ris) {
278    return ris.stream().map(RegionInfo::getEncodedName).collect(Collectors.toList()).toString();
279  }
280
281  /**
282   * Gets the table name from the specified region name.
283   * @param regionName to extract the table name from
284   * @return Table name
285   */
286  @InterfaceAudience.Private
287  // This method should never be used. Its awful doing parse from bytes.
288  // It is fallback in case we can't get the tablename any other way. Could try removing it.
289  // Keeping it Audience Private so can remove at later date.
290  static TableName getTable(final byte[] regionName) {
291    int offset = -1;
292    for (int i = 0; i < regionName.length; i++) {
293      if (regionName[i] == HConstants.DELIMITER) {
294        offset = i;
295        break;
296      }
297    }
298    if (offset <= 0) {
299      throw new IllegalArgumentException("offset=" + offset);
300    }
301    byte[] buff = new byte[offset];
302    System.arraycopy(regionName, 0, buff, 0, offset);
303    return TableName.valueOf(buff);
304  }
305
306  /**
307   * Gets the start key from the specified region name.
308   * @return Start key.
309   */
310  static byte[] getStartKey(final byte[] regionName) throws IOException {
311    return parseRegionName(regionName)[1];
312  }
313
314  /**
315   * Figure if the passed bytes represent an encoded region name or not.
316   * @param regionName A Region name either encoded or not.
317   * @return True if <code>regionName</code> represents an encoded name.
318   */
319  @InterfaceAudience.Private // For use by internals only.
320  public static boolean isEncodedRegionName(byte[] regionName) {
321    // If not parseable as region name, presume encoded. TODO: add stringency; e.g. if hex.
322    if (parseRegionNameOrReturnNull(regionName) == null) {
323      if (regionName.length > MD5_HEX_LENGTH) {
324        return false;
325      } else if (regionName.length == MD5_HEX_LENGTH) {
326        return true;
327      } else {
328        String encodedName = Bytes.toString(regionName);
329        try {
330          Integer.parseInt(encodedName);
331          // If this is a valid integer, it could be hbase:meta's encoded region name.
332          return true;
333        } catch (NumberFormatException er) {
334          return false;
335        }
336      }
337    }
338    return false;
339  }
340
341  /**
342   * Returns A deserialized {@link RegionInfo} or null if we failed deserialize or passed bytes null
343   */
344  @InterfaceAudience.Private
345  static RegionInfo parseFromOrNull(final byte[] bytes) {
346    if (bytes == null) return null;
347    return parseFromOrNull(bytes, 0, bytes.length);
348  }
349
350  /**
351   * Returns A deserialized {@link RegionInfo} or null if we failed deserialize or passed bytes null
352   */
353  @InterfaceAudience.Private
354  static RegionInfo parseFromOrNull(final byte[] bytes, int offset, int len) {
355    if (bytes == null || len <= 0) return null;
356    try {
357      return parseFrom(bytes, offset, len);
358    } catch (DeserializationException e) {
359      return null;
360    }
361  }
362
363  /**
364   * Returns A deserialized {@link RegionInfo}
365   */
366  @InterfaceAudience.Private
367  static RegionInfo parseFrom(final byte[] bytes) throws DeserializationException {
368    if (bytes == null) return null;
369    return parseFrom(bytes, 0, bytes.length);
370  }
371
372  /**
373   * Parse a serialized representation of {@link RegionInfo}
374   * @param bytes  A pb RegionInfo serialized with a pb magic prefix.
375   * @param offset starting point in the byte array
376   * @param len    length to read on the byte array
377   * @return A deserialized {@link RegionInfo}
378   */
379  @InterfaceAudience.Private
380  static RegionInfo parseFrom(final byte[] bytes, int offset, int len)
381    throws DeserializationException {
382    if (ProtobufUtil.isPBMagicPrefix(bytes, offset, len)) {
383      int pblen = ProtobufUtil.lengthOfPBMagic();
384      try {
385        HBaseProtos.RegionInfo.Builder builder = HBaseProtos.RegionInfo.newBuilder();
386        ProtobufUtil.mergeFrom(builder, bytes, pblen + offset, len - pblen);
387        HBaseProtos.RegionInfo ri = builder.build();
388        return ProtobufUtil.toRegionInfo(ri);
389      } catch (IOException e) {
390        throw new DeserializationException(e);
391      }
392    } else {
393      throw new DeserializationException("PB encoded RegionInfo expected");
394    }
395  }
396
397  /**
398   * Check whether two regions are adjacent; i.e. lies just before or just after in a table.
399   * @return true if two regions are adjacent
400   */
401  static boolean areAdjacent(RegionInfo regionA, RegionInfo regionB) {
402    if (regionA == null || regionB == null) {
403      throw new IllegalArgumentException("Can't check whether adjacent for null region");
404    }
405    if (!regionA.getTable().equals(regionB.getTable())) {
406      return false;
407    }
408    RegionInfo a = regionA;
409    RegionInfo b = regionB;
410    if (Bytes.compareTo(a.getStartKey(), b.getStartKey()) > 0) {
411      a = regionB;
412      b = regionA;
413    }
414    return Bytes.equals(a.getEndKey(), b.getStartKey());
415  }
416
417  /**
418   * Returns This instance serialized as protobuf w/ a magic pb prefix.
419   * @see #parseFrom(byte[])
420   */
421  static byte[] toByteArray(RegionInfo ri) {
422    byte[] bytes = ProtobufUtil.toRegionInfo(ri).toByteArray();
423    return ProtobufUtil.prependPBMagic(bytes);
424  }
425
426  /**
427   * Use logging.
428   * @param encodedRegionName The encoded regionname.
429   * @return <code>hbase:meta</code> if passed <code>1028785192</code> else returns
430   *         <code>encodedRegionName</code>
431   */
432  static String prettyPrint(final String encodedRegionName) {
433    if (encodedRegionName.equals("1028785192")) {
434      return encodedRegionName + "/hbase:meta";
435    }
436    return encodedRegionName;
437  }
438
439  /**
440   * Make a region name of passed parameters.
441   * @param startKey  Can be null
442   * @param regionid  Region id (Usually timestamp from when region was created).
443   * @param newFormat should we create the region name in the new format (such that it contains its
444   *                  encoded name?).
445   * @return Region name made of passed tableName, startKey and id
446   */
447  static byte[] createRegionName(final TableName tableName, final byte[] startKey,
448    final long regionid, boolean newFormat) {
449    return createRegionName(tableName, startKey, Long.toString(regionid), newFormat);
450  }
451
452  /**
453   * Make a region name of passed parameters.
454   * @param startKey  Can be null
455   * @param id        Region id (Usually timestamp from when region was created).
456   * @param newFormat should we create the region name in the new format (such that it contains its
457   *                  encoded name?).
458   * @return Region name made of passed tableName, startKey and id
459   */
460  static byte[] createRegionName(final TableName tableName, final byte[] startKey, final String id,
461    boolean newFormat) {
462    return createRegionName(tableName, startKey, Bytes.toBytes(id), newFormat);
463  }
464
465  /**
466   * Make a region name of passed parameters.
467   * @param startKey  Can be null
468   * @param regionid  Region id (Usually timestamp from when region was created).
469   * @param newFormat should we create the region name in the new format (such that it contains its
470   *                  encoded name?).
471   * @return Region name made of passed tableName, startKey, id and replicaId
472   */
473  static byte[] createRegionName(final TableName tableName, final byte[] startKey,
474    final long regionid, int replicaId, boolean newFormat) {
475    return createRegionName(tableName, startKey, Bytes.toBytes(Long.toString(regionid)), replicaId,
476      newFormat);
477  }
478
479  /**
480   * Make a region name of passed parameters.
481   * @param startKey  Can be null
482   * @param id        Region id (Usually timestamp from when region was created).
483   * @param newFormat should we create the region name in the new format (such that it contains its
484   *                  encoded name?).
485   * @return Region name made of passed tableName, startKey and id
486   */
487  static byte[] createRegionName(final TableName tableName, final byte[] startKey, final byte[] id,
488    boolean newFormat) {
489    return createRegionName(tableName, startKey, id, DEFAULT_REPLICA_ID, newFormat);
490  }
491
492  /**
493   * Make a region name of passed parameters.
494   * @param startKey  Can be null
495   * @param id        Region id (Usually timestamp from when region was created).
496   * @param newFormat should we create the region name in the new format
497   * @return Region name made of passed tableName, startKey, id and replicaId
498   */
499  static byte[] createRegionName(final TableName tableName, final byte[] startKey, final byte[] id,
500    final int replicaId, boolean newFormat) {
501    int len = tableName.getName().length + 2 + id.length + (startKey == null ? 0 : startKey.length);
502    if (newFormat) {
503      len += MD5_HEX_LENGTH + 2;
504    }
505    byte[] replicaIdBytes = null;
506    // Special casing: replicaId is only appended if replicaId is greater than
507    // 0. This is because all regions in meta would have to be migrated to the new
508    // name otherwise
509    if (replicaId > 0) {
510      // use string representation for replica id
511      replicaIdBytes = Bytes.toBytes(String.format(REPLICA_ID_FORMAT, replicaId));
512      len += 1 + replicaIdBytes.length;
513    }
514
515    byte[] b = new byte[len];
516
517    int offset = tableName.getName().length;
518    System.arraycopy(tableName.getName(), 0, b, 0, offset);
519    b[offset++] = HConstants.DELIMITER;
520    if (startKey != null && startKey.length > 0) {
521      System.arraycopy(startKey, 0, b, offset, startKey.length);
522      offset += startKey.length;
523    }
524    b[offset++] = HConstants.DELIMITER;
525    System.arraycopy(id, 0, b, offset, id.length);
526    offset += id.length;
527
528    if (replicaIdBytes != null) {
529      b[offset++] = REPLICA_ID_DELIMITER;
530      System.arraycopy(replicaIdBytes, 0, b, offset, replicaIdBytes.length);
531      offset += replicaIdBytes.length;
532    }
533
534    if (newFormat) {
535      //
536      // Encoded name should be built into the region name.
537      //
538      // Use the region name thus far (namely, <tablename>,<startKey>,<id>_<replicaId>)
539      // to compute a MD5 hash to be used as the encoded name, and append
540      // it to the byte buffer.
541      //
542      String md5Hash = MD5Hash.getMD5AsHex(b, 0, offset);
543      byte[] md5HashBytes = Bytes.toBytes(md5Hash);
544
545      if (md5HashBytes.length != MD5_HEX_LENGTH) {
546        System.out.println(
547          "MD5-hash length mismatch: Expected=" + MD5_HEX_LENGTH + "; Got=" + md5HashBytes.length);
548      }
549
550      // now append the bytes '.<encodedName>.' to the end
551      b[offset++] = ENC_SEPARATOR;
552      System.arraycopy(md5HashBytes, 0, b, offset, MD5_HEX_LENGTH);
553      offset += MD5_HEX_LENGTH;
554      b[offset] = ENC_SEPARATOR;
555    }
556
557    return b;
558  }
559
560  /**
561   * Creates a RegionInfo object for MOB data.
562   * @param tableName the name of the table
563   * @return the MOB {@link RegionInfo}.
564   */
565  static RegionInfo createMobRegionInfo(TableName tableName) {
566    // Skipping reference to RegionInfoBuilder in this class.
567    return new MutableRegionInfo(tableName, Bytes.toBytes(".mob"), HConstants.EMPTY_END_ROW, false,
568      0, DEFAULT_REPLICA_ID, false);
569  }
570
571  /**
572   * Separate elements of a regionName.
573   * @return Array of byte[] containing tableName, startKey and id OR null if not parseable as a
574   *         region name.
575   * @throws IOException if not parseable as regionName.
576   */
577  static byte[][] parseRegionName(final byte[] regionName) throws IOException {
578    byte[][] result = parseRegionNameOrReturnNull(regionName);
579    if (result == null) {
580      throw new IOException(
581        INVALID_REGION_NAME_FORMAT_MESSAGE + ": " + Bytes.toStringBinary(regionName));
582    }
583    return result;
584  }
585
586  /**
587   * Separate elements of a regionName. Region name is of the format:
588   * <code>tablename,startkey,regionIdTimestamp[_replicaId][.encodedName.]</code>. Startkey can
589   * contain the delimiter (',') so we parse from the start and then parse from the end.
590   * @return Array of byte[] containing tableName, startKey and id OR null if not parseable as a
591   *         region name.
592   */
593  static byte[][] parseRegionNameOrReturnNull(final byte[] regionName) {
594    int offset = -1;
595    for (int i = 0; i < regionName.length; i++) {
596      if (regionName[i] == HConstants.DELIMITER) {
597        offset = i;
598        break;
599      }
600    }
601    if (offset == -1) {
602      return null;
603    }
604    byte[] tableName = new byte[offset];
605    System.arraycopy(regionName, 0, tableName, 0, offset);
606    offset = -1;
607
608    int endOffset = regionName.length;
609    // check whether regionName contains encodedName
610    if (
611      regionName.length > MD5_HEX_LENGTH + 2 && regionName[regionName.length - 1] == ENC_SEPARATOR
612        && regionName[regionName.length - MD5_HEX_LENGTH - 2] == ENC_SEPARATOR
613    ) {
614      endOffset = endOffset - MD5_HEX_LENGTH - 2;
615    }
616
617    // parse from end
618    byte[] replicaId = null;
619    int idEndOffset = endOffset;
620    for (int i = endOffset - 1; i > 0; i--) {
621      if (regionName[i] == REPLICA_ID_DELIMITER) { // replicaId may or may not be present
622        replicaId = new byte[endOffset - i - 1];
623        System.arraycopy(regionName, i + 1, replicaId, 0, endOffset - i - 1);
624        idEndOffset = i;
625        // do not break, continue to search for id
626      }
627      if (regionName[i] == HConstants.DELIMITER) {
628        offset = i;
629        break;
630      }
631    }
632    if (offset == -1) {
633      return null;
634    }
635    byte[] startKey = HConstants.EMPTY_BYTE_ARRAY;
636    if (offset != tableName.length + 1) {
637      startKey = new byte[offset - tableName.length - 1];
638      System.arraycopy(regionName, tableName.length + 1, startKey, 0,
639        offset - tableName.length - 1);
640    }
641    byte[] id = new byte[idEndOffset - offset - 1];
642    System.arraycopy(regionName, offset + 1, id, 0, idEndOffset - offset - 1);
643    byte[][] elements = new byte[replicaId == null ? 3 : 4][];
644    elements[0] = tableName;
645    elements[1] = startKey;
646    elements[2] = id;
647    if (replicaId != null) {
648      elements[3] = replicaId;
649    }
650    return elements;
651  }
652
653  /**
654   * Serializes given RegionInfo's as a byte array. Use this instead of
655   * {@link RegionInfo#toByteArray(RegionInfo)} when writing to a stream and you want to use the pb
656   * mergeDelimitedFrom (w/o the delimiter, pb reads to EOF which may not be what you want).
657   * {@link #parseDelimitedFrom(byte[], int, int)} can be used to read back the instances.
658   * @param infos RegionInfo objects to serialize
659   * @return This instance serialized as a delimited protobuf w/ a magic pb prefix.
660   */
661  static byte[] toDelimitedByteArray(RegionInfo... infos) throws IOException {
662    byte[][] bytes = new byte[infos.length][];
663    int size = 0;
664    for (int i = 0; i < infos.length; i++) {
665      bytes[i] = toDelimitedByteArray(infos[i]);
666      size += bytes[i].length;
667    }
668
669    byte[] result = new byte[size];
670    int offset = 0;
671    for (byte[] b : bytes) {
672      System.arraycopy(b, 0, result, offset, b.length);
673      offset += b.length;
674    }
675    return result;
676  }
677
678  /**
679   * Use this instead of {@link RegionInfo#toByteArray(RegionInfo)} when writing to a stream and you
680   * want to use the pb mergeDelimitedFrom (w/o the delimiter, pb reads to EOF which may not be what
681   * you want).
682   * @return This instance serialized as a delimied protobuf w/ a magic pb prefix.
683   */
684  static byte[] toDelimitedByteArray(RegionInfo ri) throws IOException {
685    return ProtobufUtil.toDelimitedByteArray(ProtobufUtil.toRegionInfo(ri));
686  }
687
688  /**
689   * Parses an RegionInfo instance from the passed in stream. Presumes the RegionInfo was serialized
690   * to the stream with {@link #toDelimitedByteArray(RegionInfo)}.
691   * @return An instance of RegionInfo.
692   */
693  static RegionInfo parseFrom(final DataInputStream in) throws IOException {
694    // I need to be able to move back in the stream if this is not a pb
695    // serialization so I can do the Writable decoding instead.
696    int pblen = ProtobufUtil.lengthOfPBMagic();
697    byte[] pbuf = new byte[pblen];
698    if (in.markSupported()) { // read it with mark()
699      in.mark(pblen);
700    }
701
702    // assumption: if Writable serialization, it should be longer than pblen.
703    IOUtils.readFully(in, pbuf, 0, pblen);
704    if (ProtobufUtil.isPBMagicPrefix(pbuf)) {
705      return ProtobufUtil.toRegionInfo(HBaseProtos.RegionInfo.parseDelimitedFrom(in));
706    } else {
707      throw new IOException("PB encoded RegionInfo expected");
708    }
709  }
710
711  /**
712   * Parses all the RegionInfo instances from the passed in stream until EOF. Presumes the
713   * RegionInfo's were serialized to the stream with oDelimitedByteArray()
714   * @param bytes  serialized bytes
715   * @param offset the start offset into the byte[] buffer
716   * @param length how far we should read into the byte[] buffer
717   * @return All the RegionInfos that are in the byte array. Keeps reading till we hit the end.
718   */
719  static List<RegionInfo> parseDelimitedFrom(final byte[] bytes, final int offset, final int length)
720    throws IOException {
721    if (bytes == null) {
722      throw new IllegalArgumentException("Can't build an object with empty bytes array");
723    }
724    List<RegionInfo> ris = new ArrayList<>();
725    try (DataInputBuffer in = new DataInputBuffer()) {
726      in.reset(bytes, offset, length);
727      while (in.available() > 0) {
728        RegionInfo ri = parseFrom(in);
729        ris.add(ri);
730      }
731    }
732    return ris;
733  }
734
735  /** Returns True if this is first Region in Table */
736  default boolean isFirst() {
737    return Bytes.equals(getStartKey(), HConstants.EMPTY_START_ROW);
738  }
739
740  /** Returns True if this is last Region in Table */
741  default boolean isLast() {
742    return Bytes.equals(getEndKey(), HConstants.EMPTY_END_ROW);
743  }
744
745  /**
746   * Returns True if region is next, adjacent but 'after' this one.
747   * @see #isAdjacent(RegionInfo)
748   * @see #areAdjacent(RegionInfo, RegionInfo)
749   */
750  default boolean isNext(RegionInfo after) {
751    return getTable().equals(after.getTable()) && Bytes.equals(getEndKey(), after.getStartKey());
752  }
753
754  /**
755   * Returns True if region is adjacent, either just before or just after this one.
756   * @see #isNext(RegionInfo)
757   */
758  default boolean isAdjacent(RegionInfo other) {
759    return getTable().equals(other.getTable()) && areAdjacent(this, other);
760  }
761
762  /** Returns True if RegionInfo is degenerate... if startKey > endKey. */
763  default boolean isDegenerate() {
764    return !isLast() && Bytes.compareTo(getStartKey(), getEndKey()) > 0;
765  }
766
767  /**
768   * Returns True if an overlap in region range.
769   * @see #isDegenerate()
770   */
771  default boolean isOverlap(RegionInfo other) {
772    if (other == null) {
773      return false;
774    }
775    if (!getTable().equals(other.getTable())) {
776      return false;
777    }
778    int startKeyCompare = Bytes.compareTo(getStartKey(), other.getStartKey());
779    if (startKeyCompare == 0) {
780      return true;
781    }
782    if (startKeyCompare < 0) {
783      if (isLast()) {
784        return true;
785      }
786      return Bytes.compareTo(getEndKey(), other.getStartKey()) > 0;
787    }
788    if (other.isLast()) {
789      return true;
790    }
791    return Bytes.compareTo(getStartKey(), other.getEndKey()) < 0;
792  }
793
794  @Override
795  default int compareTo(RegionInfo other) {
796    return RegionInfo.COMPARATOR.compare(this, other);
797  }
798}