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 */
018
019package org.apache.hadoop.hbase.client;
020
021import java.util.ArrayList;
022import java.util.Collection;
023import java.util.Iterator;
024import java.util.List;
025
026import org.apache.hadoop.hbase.util.Bytes;
027import org.apache.yetus.audience.InterfaceAudience;
028
029/**
030 * Utility methods which contain the logic for regions and replicas.
031 */
032@InterfaceAudience.Private
033public class RegionReplicaUtil {
034
035  /**
036   * Whether or not the secondary region will wait for observing a flush / region open event
037   * from the primary region via async wal replication before enabling read requests. Since replayed
038   * edits from async wal replication from primary is not persisted in WAL, the memstore of the
039   * secondary region might be non-empty at the time of close or crash. For ensuring seqId's not
040   * "going back in time" in the secondary region replica, this should be enabled. However, in some
041   * cases the above semantics might be ok for some application classes.
042   * See HBASE-11580 for more context.
043   */
044  public static final String REGION_REPLICA_WAIT_FOR_PRIMARY_FLUSH_CONF_KEY
045    = "hbase.region.replica.wait.for.primary.flush";
046  protected static final boolean DEFAULT_REGION_REPLICA_WAIT_FOR_PRIMARY_FLUSH = true;
047
048  /**
049   * The default replicaId for the region
050   */
051  static final int DEFAULT_REPLICA_ID = 0;
052
053  /**
054   * Returns the RegionInfo for the given replicaId.
055   * RegionInfo's correspond to a range of a table, but more than one
056   * "instance" of the same range can be deployed which are differentiated by
057   * the replicaId.
058   * @param regionInfo
059   * @param replicaId the replicaId to use
060   * @return an RegionInfo object corresponding to the same range (table, start and
061   * end key), but for the given replicaId.
062   */
063  public static RegionInfo getRegionInfoForReplica(RegionInfo regionInfo, int replicaId) {
064    if (regionInfo.getReplicaId() == replicaId) {
065      return regionInfo;
066    }
067
068    if (regionInfo.isMetaRegion()) {
069      return RegionInfoBuilder.newBuilder(regionInfo.getTable())
070          .setRegionId(regionInfo.getRegionId())
071          .setReplicaId(replicaId)
072          .setOffline(regionInfo.isOffline())
073          .build();
074    } else {
075      return RegionInfoBuilder.newBuilder(regionInfo.getTable())
076              .setStartKey(regionInfo.getStartKey())
077              .setEndKey(regionInfo.getEndKey())
078              .setSplit(regionInfo.isSplit())
079              .setRegionId(regionInfo.getRegionId())
080              .setReplicaId(replicaId)
081              .setOffline(regionInfo.isOffline())
082              .build();
083    }
084  }
085
086  /**
087   * Returns the RegionInfo for the default replicaId (0). RegionInfo's correspond to
088   * a range of a table, but more than one "instance" of the same range can be
089   * deployed which are differentiated by the replicaId.
090   * @return an RegionInfo object corresponding to the same range (table, start and
091   * end key), but for the default replicaId.
092   */
093  public static RegionInfo getRegionInfoForDefaultReplica(RegionInfo regionInfo) {
094    return getRegionInfoForReplica(regionInfo, DEFAULT_REPLICA_ID);
095  }
096
097  /** @return true if this replicaId corresponds to default replica for the region */
098  public static boolean isDefaultReplica(int replicaId) {
099    return DEFAULT_REPLICA_ID == replicaId;
100  }
101
102  /** @return true if this region is a default replica for the region */
103  public static boolean isDefaultReplica(RegionInfo hri) {
104    return  hri.getReplicaId() == DEFAULT_REPLICA_ID;
105  }
106
107  /**
108   * Removes the non-default replicas from the passed regions collection
109   * @param regions
110   */
111  public static void removeNonDefaultRegions(Collection<RegionInfo> regions) {
112    Iterator<RegionInfo> iterator = regions.iterator();
113    while (iterator.hasNext()) {
114      RegionInfo hri = iterator.next();
115      if (!RegionReplicaUtil.isDefaultReplica(hri)) {
116        iterator.remove();
117      }
118    }
119  }
120
121  public static boolean isReplicasForSameRegion(RegionInfo regionInfoA, RegionInfo regionInfoB) {
122    return compareRegionInfosWithoutReplicaId(regionInfoA, regionInfoB) == 0;
123  }
124
125  private static int compareRegionInfosWithoutReplicaId(RegionInfo regionInfoA,
126      RegionInfo regionInfoB) {
127    int result = regionInfoA.getTable().compareTo(regionInfoB.getTable());
128    if (result != 0) {
129      return result;
130    }
131
132    // Compare start keys.
133    result = Bytes.compareTo(regionInfoA.getStartKey(), regionInfoB.getStartKey());
134    if (result != 0) {
135      return result;
136    }
137
138    // Compare end keys.
139    result = Bytes.compareTo(regionInfoA.getEndKey(), regionInfoB.getEndKey());
140
141    if (result != 0) {
142      if (regionInfoA.getStartKey().length != 0
143              && regionInfoA.getEndKey().length == 0) {
144          return 1; // this is last region
145      }
146      if (regionInfoB.getStartKey().length != 0
147              && regionInfoB.getEndKey().length == 0) {
148          return -1; // o is the last region
149      }
150      return result;
151    }
152
153    // regionId is usually milli timestamp -- this defines older stamps
154    // to be "smaller" than newer stamps in sort order.
155    if (regionInfoA.getRegionId() > regionInfoB.getRegionId()) {
156      return 1;
157    } else if (regionInfoA.getRegionId() < regionInfoB.getRegionId()) {
158      return -1;
159    }
160    return 0;
161  }
162
163  /**
164   * Create any replicas for the regions (the default replicas that was already created is passed to
165   * the method)
166   * @param tableDescriptor descriptor to use
167   * @param regions existing regions
168   * @param oldReplicaCount existing replica count
169   * @param newReplicaCount updated replica count due to modify table
170   * @return the combined list of default and non-default replicas
171   */
172  public static List<RegionInfo> addReplicas(final TableDescriptor tableDescriptor,
173      final List<RegionInfo> regions, int oldReplicaCount, int newReplicaCount) {
174    if ((newReplicaCount - 1) <= 0) {
175      return regions;
176    }
177    List<RegionInfo> hRegionInfos = new ArrayList<>((newReplicaCount) * regions.size());
178    for (RegionInfo ri : regions) {
179      if (RegionReplicaUtil.isDefaultReplica(ri) &&
180        (!ri.isOffline() || (!ri.isSplit() && !ri.isSplitParent()))) {
181        // region level replica index starts from 0. So if oldReplicaCount was 2 then the max replicaId for
182        // the existing regions would be 1
183        for (int j = oldReplicaCount; j < newReplicaCount; j++) {
184          hRegionInfos.add(RegionReplicaUtil.getRegionInfoForReplica(ri, j));
185        }
186      }
187    }
188    hRegionInfos.addAll(regions);
189    return hRegionInfos;
190  }
191}