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