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