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;
020
021import java.util.Collection;
022
023import org.apache.hadoop.hbase.client.RegionInfo;
024import org.apache.hadoop.hbase.client.RegionReplicaUtil;
025import org.apache.hadoop.hbase.util.Bytes;
026import org.apache.yetus.audience.InterfaceAudience;
027
028/**
029 * Container for holding a list of {@link HRegionLocation}'s that correspond to the
030 * same range. The list is indexed by the replicaId. This is an immutable list,
031 * however mutation operations are provided which returns a new List via copy-on-write
032 * (assuming small number of locations)
033 */
034@InterfaceAudience.Private
035public class RegionLocations {
036
037  private final int numNonNullElements;
038
039  // locations array contains the HRL objects for known region replicas indexes by the replicaId.
040  // elements can be null if the region replica is not known at all. A null value indicates
041  // that there is a region replica with the index as replicaId, but the location is not known
042  // in the cache.
043  private final HRegionLocation[] locations; // replicaId -> HRegionLocation.
044
045  /**
046   * Constructs the region location list. The locations array should
047   * contain all the locations for known replicas for the region, and should be
048   * sorted in replicaId ascending order, although it can contain nulls indicating replicaIds
049   * that the locations of which are not known.
050   * @param locations an array of HRegionLocations for the same region range
051   */
052  public RegionLocations(HRegionLocation... locations) {
053    int numNonNullElements = 0;
054    int maxReplicaId = -1;
055    int maxReplicaIdIndex = -1;
056    int index = 0;
057    for (HRegionLocation loc : locations) {
058      if (loc != null) {
059        if (loc.getRegion().getReplicaId() >= maxReplicaId) {
060          maxReplicaId = loc.getRegion().getReplicaId();
061          maxReplicaIdIndex = index;
062        }
063      }
064      index++;
065    }
066    // account for the null elements in the array after maxReplicaIdIndex
067    maxReplicaId = maxReplicaId + (locations.length - (maxReplicaIdIndex + 1) );
068
069    if (maxReplicaId + 1 == locations.length) {
070      this.locations = locations;
071    } else {
072      this.locations = new HRegionLocation[maxReplicaId + 1];
073      for (HRegionLocation loc : locations) {
074        if (loc != null) {
075          this.locations[loc.getRegion().getReplicaId()] = loc;
076        }
077      }
078    }
079    for (HRegionLocation loc : this.locations) {
080      if (loc != null && loc.getServerName() != null){
081        numNonNullElements++;
082      }
083    }
084    this.numNonNullElements = numNonNullElements;
085  }
086
087  public RegionLocations(Collection<HRegionLocation> locations) {
088    this(locations.toArray(new HRegionLocation[locations.size()]));
089  }
090
091  /**
092   * Returns the size of the list even if some of the elements
093   * might be null.
094   * @return the size of the list (corresponding to the max replicaId)
095   */
096  public int size() {
097    return locations.length;
098  }
099
100  /**
101   * Returns the size of not-null locations
102   * @return the size of not-null locations
103   */
104  public int numNonNullElements() {
105    return numNonNullElements;
106  }
107
108  /**
109   * Returns whether there are non-null elements in the list
110   * @return whether there are non-null elements in the list
111   */
112  public boolean isEmpty() {
113    return numNonNullElements == 0;
114  }
115
116  /**
117   * Returns a new RegionLocations with the locations removed (set to null)
118   * which have the destination server as given.
119   * @param serverName the serverName to remove locations of
120   * @return an RegionLocations object with removed locations or the same object
121   * if nothing is removed
122   */
123  public RegionLocations removeByServer(ServerName serverName) {
124    HRegionLocation[] newLocations = null;
125    for (int i = 0; i < locations.length; i++) {
126      // check whether something to remove
127      if (locations[i] != null && serverName.equals(locations[i].getServerName())) {
128        if (newLocations == null) { //first time
129          newLocations = new HRegionLocation[locations.length];
130          System.arraycopy(locations, 0, newLocations, 0, i);
131        }
132        newLocations[i] = null;
133      } else if (newLocations != null) {
134        newLocations[i] = locations[i];
135      }
136    }
137    return newLocations == null ? this : new RegionLocations(newLocations);
138  }
139
140  /**
141   * Removes the given location from the list
142   * @param location the location to remove
143   * @return an RegionLocations object with removed locations or the same object
144   * if nothing is removed
145   */
146  public RegionLocations remove(HRegionLocation location) {
147    if (location == null) return this;
148    if (location.getRegion() == null) return this;
149    int replicaId = location.getRegion().getReplicaId();
150    if (replicaId >= locations.length) return this;
151
152    // check whether something to remove. HRL.compareTo() compares ONLY the
153    // serverName. We want to compare the HRI's as well.
154    if (locations[replicaId] == null
155        || RegionInfo.COMPARATOR.compare(location.getRegion(), locations[replicaId].getRegion()) != 0
156        || !location.equals(locations[replicaId])) {
157      return this;
158    }
159
160    HRegionLocation[] newLocations = new HRegionLocation[locations.length];
161    System.arraycopy(locations, 0, newLocations, 0, locations.length);
162    newLocations[replicaId] = null;
163
164    return new RegionLocations(newLocations);
165  }
166
167  /**
168   * Removes location of the given replicaId from the list
169   * @param replicaId the replicaId of the location to remove
170   * @return an RegionLocations object with removed locations or the same object
171   * if nothing is removed
172   */
173  public RegionLocations remove(int replicaId) {
174    if (getRegionLocation(replicaId) == null) {
175      return this;
176    }
177
178    HRegionLocation[] newLocations = new HRegionLocation[locations.length];
179
180    System.arraycopy(locations, 0, newLocations, 0, locations.length);
181    if (replicaId < newLocations.length) {
182      newLocations[replicaId] = null;
183    }
184
185    return new RegionLocations(newLocations);
186  }
187
188  /**
189   * Merges this RegionLocations list with the given list assuming
190   * same range, and keeping the most up to date version of the
191   * HRegionLocation entries from either list according to seqNum. If seqNums
192   * are equal, the location from the argument (other) is taken.
193   * @param other the locations to merge with
194   * @return an RegionLocations object with merged locations or the same object
195   * if nothing is merged
196   */
197  public RegionLocations mergeLocations(RegionLocations other) {
198    assert other != null;
199
200    HRegionLocation[] newLocations = null;
201
202    // Use the length from other, since it is coming from meta. Otherwise,
203    // in case of region replication going down, we might have a leak here.
204    int max = other.locations.length;
205
206    RegionInfo regionInfo = null;
207    for (int i = 0; i < max; i++) {
208      HRegionLocation thisLoc = this.getRegionLocation(i);
209      HRegionLocation otherLoc = other.getRegionLocation(i);
210      if (regionInfo == null && otherLoc != null && otherLoc.getRegion() != null) {
211        // regionInfo is the first non-null HRI from other RegionLocations. We use it to ensure that
212        // all replica region infos belong to the same region with same region id.
213        regionInfo = otherLoc.getRegion();
214      }
215
216      HRegionLocation selectedLoc = selectRegionLocation(thisLoc,
217        otherLoc, true, false);
218
219      if (selectedLoc != thisLoc) {
220        if (newLocations == null) {
221          newLocations = new HRegionLocation[max];
222          System.arraycopy(locations, 0, newLocations, 0, i);
223        }
224      }
225      if (newLocations != null) {
226        newLocations[i] = selectedLoc;
227      }
228    }
229
230    // ensure that all replicas share the same start code. Otherwise delete them
231    if (newLocations != null && regionInfo != null) {
232      for (int i=0; i < newLocations.length; i++) {
233        if (newLocations[i] != null) {
234          if (!RegionReplicaUtil.isReplicasForSameRegion(regionInfo,
235            newLocations[i].getRegion())) {
236            newLocations[i] = null;
237          }
238        }
239      }
240    }
241
242    return newLocations == null ? this : new RegionLocations(newLocations);
243  }
244
245  private HRegionLocation selectRegionLocation(HRegionLocation oldLocation,
246      HRegionLocation location, boolean checkForEquals, boolean force) {
247    if (location == null) {
248      return oldLocation == null ? null : oldLocation;
249    }
250
251    if (oldLocation == null) {
252      return location;
253    }
254
255    if (force
256        || isGreaterThan(location.getSeqNum(), oldLocation.getSeqNum(), checkForEquals)) {
257      return location;
258    }
259    return oldLocation;
260  }
261
262  /**
263   * Updates the location with new only if the new location has a higher
264   * seqNum than the old one or force is true.
265   * @param location the location to add or update
266   * @param checkForEquals whether to update the location if seqNums for the
267   * HRegionLocations for the old and new location are the same
268   * @param force whether to force update
269   * @return an RegionLocations object with updated locations or the same object
270   * if nothing is updated
271   */
272  public RegionLocations updateLocation(HRegionLocation location,
273      boolean checkForEquals, boolean force) {
274    assert location != null;
275
276    int replicaId = location.getRegion().getReplicaId();
277
278    HRegionLocation oldLoc = getRegionLocation(location.getRegion().getReplicaId());
279    HRegionLocation selectedLoc = selectRegionLocation(oldLoc, location,
280      checkForEquals, force);
281
282    if (selectedLoc == oldLoc) {
283      return this;
284    }
285    HRegionLocation[] newLocations = new HRegionLocation[Math.max(locations.length, replicaId +1)];
286    System.arraycopy(locations, 0, newLocations, 0, locations.length);
287    newLocations[replicaId] = location;
288    // ensure that all replicas share the same start code. Otherwise delete them
289    for (int i=0; i < newLocations.length; i++) {
290      if (newLocations[i] != null) {
291        if (!RegionReplicaUtil.isReplicasForSameRegion(location.getRegion(),
292          newLocations[i].getRegion())) {
293          newLocations[i] = null;
294        }
295      }
296    }
297    return new RegionLocations(newLocations);
298  }
299
300  private boolean isGreaterThan(long a, long b, boolean checkForEquals) {
301    return a > b || (checkForEquals && (a == b));
302  }
303
304  public HRegionLocation getRegionLocation(int replicaId) {
305    if (replicaId >= locations.length) {
306      return null;
307    }
308    return locations[replicaId];
309  }
310
311  /**
312   * Returns the region location from the list for matching regionName, which can
313   * be regionName or encodedRegionName
314   * @param regionName regionName or encodedRegionName
315   * @return HRegionLocation found or null
316   */
317  public HRegionLocation getRegionLocationByRegionName(byte[] regionName) {
318    for (HRegionLocation loc : locations) {
319      if (loc != null) {
320        if (Bytes.equals(loc.getRegion().getRegionName(), regionName)
321            || Bytes.equals(loc.getRegion().getEncodedNameAsBytes(), regionName)) {
322          return loc;
323        }
324      }
325    }
326    return null;
327  }
328
329  public HRegionLocation[] getRegionLocations() {
330    return locations;
331  }
332
333  public HRegionLocation getDefaultRegionLocation() {
334    return locations[RegionInfo.DEFAULT_REPLICA_ID];
335  }
336
337  /**
338   * Returns the first not-null region location in the list
339   */
340  public HRegionLocation getRegionLocation() {
341    for (HRegionLocation loc : locations) {
342      if (loc != null) {
343        return loc;
344      }
345    }
346    return null;
347  }
348
349  @Override
350  public String toString() {
351    StringBuilder builder = new StringBuilder("[");
352    for (HRegionLocation loc : locations) {
353      if (builder.length() > 1) {
354        builder.append(", ");
355      }
356      builder.append(loc == null ? "null" : loc);
357    }
358    builder.append("]");
359    return builder.toString();
360  }
361
362}