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