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.master;
019
020import java.util.Date;
021import org.apache.hadoop.hbase.ServerName;
022import org.apache.hadoop.hbase.client.RegionInfo;
023import org.apache.yetus.audience.InterfaceAudience;
024import org.apache.yetus.audience.InterfaceStability;
025
026import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
027import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
028import org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos;
029
030/**
031 * State of a Region while undergoing transitions.
032 * This class is immutable.
033 */
034@InterfaceAudience.Private
035public class RegionState {
036
037  @InterfaceAudience.Private
038  @InterfaceStability.Evolving
039  public enum State {
040    OFFLINE,        // region is in an offline state
041    OPENING,        // server has begun to open but not yet done
042    OPEN,           // server opened region and updated meta
043    CLOSING,        // server has begun to close but not yet done
044    CLOSED,         // server closed region and updated meta
045    SPLITTING,      // server started split of a region
046    SPLIT,          // server completed split of a region
047    FAILED_OPEN,    // failed to open, and won't retry any more
048    FAILED_CLOSE,   // failed to close, and won't retry any more
049    MERGING,        // server started merge a region
050    MERGED,         // server completed merge a region
051    SPLITTING_NEW,  // new region to be created when RS splits a parent
052                    // region but hasn't be created yet, or master doesn't
053                    // know it's already created
054    MERGING_NEW;    // new region to be created when RS merges two
055                    // daughter regions but hasn't be created yet, or
056                    // master doesn't know it's already created
057
058    /**
059     * Convert to protobuf ClusterStatusProtos.RegionState.State
060     */
061    public ClusterStatusProtos.RegionState.State convert() {
062      ClusterStatusProtos.RegionState.State rs;
063      switch (this) {
064      case OFFLINE:
065        rs = ClusterStatusProtos.RegionState.State.OFFLINE;
066        break;
067      case OPENING:
068        rs = ClusterStatusProtos.RegionState.State.OPENING;
069        break;
070      case OPEN:
071        rs = ClusterStatusProtos.RegionState.State.OPEN;
072        break;
073      case CLOSING:
074        rs = ClusterStatusProtos.RegionState.State.CLOSING;
075        break;
076      case CLOSED:
077        rs = ClusterStatusProtos.RegionState.State.CLOSED;
078        break;
079      case SPLITTING:
080        rs = ClusterStatusProtos.RegionState.State.SPLITTING;
081        break;
082      case SPLIT:
083        rs = ClusterStatusProtos.RegionState.State.SPLIT;
084        break;
085      case FAILED_OPEN:
086        rs = ClusterStatusProtos.RegionState.State.FAILED_OPEN;
087        break;
088      case FAILED_CLOSE:
089        rs = ClusterStatusProtos.RegionState.State.FAILED_CLOSE;
090        break;
091      case MERGING:
092        rs = ClusterStatusProtos.RegionState.State.MERGING;
093        break;
094      case MERGED:
095        rs = ClusterStatusProtos.RegionState.State.MERGED;
096        break;
097      case SPLITTING_NEW:
098        rs = ClusterStatusProtos.RegionState.State.SPLITTING_NEW;
099        break;
100      case MERGING_NEW:
101        rs = ClusterStatusProtos.RegionState.State.MERGING_NEW;
102        break;
103      default:
104        throw new IllegalStateException("");
105      }
106      return rs;
107    }
108
109    /**
110     * Convert a protobuf HBaseProtos.RegionState.State to a RegionState.State
111     *
112     * @return the RegionState.State
113     */
114    public static State convert(ClusterStatusProtos.RegionState.State protoState) {
115      State state;
116      switch (protoState) {
117      case OFFLINE:
118        state = OFFLINE;
119        break;
120      case PENDING_OPEN:
121      case OPENING:
122        state = OPENING;
123        break;
124      case OPEN:
125        state = OPEN;
126        break;
127      case PENDING_CLOSE:
128      case CLOSING:
129        state = CLOSING;
130        break;
131      case CLOSED:
132        state = CLOSED;
133        break;
134      case SPLITTING:
135        state = SPLITTING;
136        break;
137      case SPLIT:
138        state = SPLIT;
139        break;
140      case FAILED_OPEN:
141        state = FAILED_OPEN;
142        break;
143      case FAILED_CLOSE:
144        state = FAILED_CLOSE;
145        break;
146      case MERGING:
147        state = MERGING;
148        break;
149      case MERGED:
150        state = MERGED;
151        break;
152      case SPLITTING_NEW:
153        state = SPLITTING_NEW;
154        break;
155      case MERGING_NEW:
156        state = MERGING_NEW;
157        break;
158      default:
159        throw new IllegalStateException("Unhandled state " + protoState);
160      }
161      return state;
162    }
163  }
164
165  private final long stamp;
166  private final RegionInfo hri;
167  private final ServerName serverName;
168  private final State state;
169  // The duration of region in transition
170  private long ritDuration;
171
172  @VisibleForTesting
173  public static RegionState createForTesting(RegionInfo region, State state) {
174    return new RegionState(region, state, System.currentTimeMillis(), null);
175  }
176
177  public RegionState(RegionInfo region, State state, ServerName serverName) {
178    this(region, state, System.currentTimeMillis(), serverName);
179  }
180
181  public RegionState(RegionInfo region,
182      State state, long stamp, ServerName serverName) {
183    this(region, state, stamp, serverName, 0);
184  }
185
186  public RegionState(RegionInfo region, State state, long stamp, ServerName serverName,
187      long ritDuration) {
188    this.hri = region;
189    this.state = state;
190    this.stamp = stamp;
191    this.serverName = serverName;
192    this.ritDuration = ritDuration;
193  }
194
195  public State getState() {
196    return state;
197  }
198
199  public long getStamp() {
200    return stamp;
201  }
202
203  public RegionInfo getRegion() {
204    return hri;
205  }
206
207  public ServerName getServerName() {
208    return serverName;
209  }
210
211  public long getRitDuration() {
212    return ritDuration;
213  }
214
215  /**
216   * Update the duration of region in transition
217   * @param previousStamp previous RegionState's timestamp
218   */
219  @InterfaceAudience.Private
220  void updateRitDuration(long previousStamp) {
221    this.ritDuration += (this.stamp - previousStamp);
222  }
223
224  public boolean isClosing() {
225    return state == State.CLOSING;
226  }
227
228  public boolean isClosed() {
229    return state == State.CLOSED;
230  }
231
232  public boolean isOpening() {
233    return state == State.OPENING;
234  }
235
236  public boolean isOpened() {
237    return state == State.OPEN;
238  }
239
240  public boolean isOffline() {
241    return state == State.OFFLINE;
242  }
243
244  public boolean isSplitting() {
245    return state == State.SPLITTING;
246  }
247
248  public boolean isSplit() {
249    return state == State.SPLIT;
250  }
251
252  public boolean isSplittingNew() {
253    return state == State.SPLITTING_NEW;
254  }
255
256  public boolean isFailedOpen() {
257    return state == State.FAILED_OPEN;
258  }
259
260  public boolean isFailedClose() {
261    return state == State.FAILED_CLOSE;
262  }
263
264  public boolean isMerging() {
265    return state == State.MERGING;
266  }
267
268  public boolean isMerged() {
269    return state == State.MERGED;
270  }
271
272  public boolean isMergingNew() {
273    return state == State.MERGING_NEW;
274  }
275
276  public boolean isOnServer(final ServerName sn) {
277    return serverName != null && serverName.equals(sn);
278  }
279
280  public boolean isMergingOnServer(final ServerName sn) {
281    return isOnServer(sn) && isMerging();
282  }
283
284  public boolean isMergingNewOnServer(final ServerName sn) {
285    return isOnServer(sn) && isMergingNew();
286  }
287
288  public boolean isMergingNewOrOpenedOnServer(final ServerName sn) {
289    return isOnServer(sn) && (isMergingNew() || isOpened());
290  }
291
292  public boolean isMergingNewOrOfflineOnServer(final ServerName sn) {
293    return isOnServer(sn) && (isMergingNew() || isOffline());
294  }
295
296  public boolean isSplittingOnServer(final ServerName sn) {
297    return isOnServer(sn) && isSplitting();
298  }
299
300  public boolean isSplittingNewOnServer(final ServerName sn) {
301    return isOnServer(sn) && isSplittingNew();
302  }
303
304  public boolean isSplittingOrOpenedOnServer(final ServerName sn) {
305    return isOnServer(sn) && (isSplitting() || isOpened());
306  }
307
308  public boolean isSplittingOrSplitOnServer(final ServerName sn) {
309    return isOnServer(sn) && (isSplitting() || isSplit());
310  }
311
312  public boolean isClosingOrClosedOnServer(final ServerName sn) {
313    return isOnServer(sn) && (isClosing() || isClosed());
314  }
315
316  public boolean isOpeningOrFailedOpenOnServer(final ServerName sn) {
317    return isOnServer(sn) && (isOpening() || isFailedOpen());
318  }
319
320  public boolean isOpeningOrOpenedOnServer(final ServerName sn) {
321    return isOnServer(sn) && (isOpening() || isOpened());
322  }
323
324  public boolean isOpenedOnServer(final ServerName sn) {
325    return isOnServer(sn) && isOpened();
326  }
327
328  /**
329   * Check if a region state can transition to offline
330   */
331  public boolean isReadyToOffline() {
332    return isMerged() || isSplit() || isOffline()
333      || isSplittingNew() || isMergingNew();
334  }
335
336  /**
337   * Check if a region state can transition to online
338   */
339  public boolean isReadyToOnline() {
340    return isOpened() || isSplittingNew() || isMergingNew();
341  }
342
343  /**
344   * Check if a region state is one of offline states that
345   * can't transition to pending_close/closing (unassign/offline)
346   */
347  public boolean isUnassignable() {
348    return isUnassignable(state);
349  }
350
351  /**
352   * Check if a region state is one of offline states that
353   * can't transition to pending_close/closing (unassign/offline)
354   */
355  public static boolean isUnassignable(State state) {
356    return state == State.MERGED || state == State.SPLIT || state == State.OFFLINE
357      || state == State.SPLITTING_NEW || state == State.MERGING_NEW;
358  }
359
360  @Override
361  public String toString() {
362    return "{" + hri.getShortNameToLog()
363      + " state=" + state
364      + ", ts=" + stamp
365      + ", server=" + serverName + "}";
366  }
367
368  /**
369   * A slower (but more easy-to-read) stringification
370   */
371  public String toDescriptiveString() {
372    long relTime = System.currentTimeMillis() - stamp;
373    return hri.getRegionNameAsString()
374      + " state=" + state + ", ts=" + new Date(stamp) + " (" +
375        java.time.Duration.ofMillis(relTime).toString() +
376        " ago), server=" + serverName;
377  }
378
379  /**
380   * Convert a RegionState to an HBaseProtos.RegionState
381   *
382   * @return the converted HBaseProtos.RegionState
383   */
384  public ClusterStatusProtos.RegionState convert() {
385    ClusterStatusProtos.RegionState.Builder regionState = ClusterStatusProtos.RegionState.newBuilder();
386    regionState.setRegionInfo(ProtobufUtil.toRegionInfo(hri));
387    regionState.setState(state.convert());
388    regionState.setStamp(getStamp());
389    return regionState.build();
390  }
391
392  /**
393   * Convert a protobuf HBaseProtos.RegionState to a RegionState
394   *
395   * @return the RegionState
396   */
397  public static RegionState convert(ClusterStatusProtos.RegionState proto) {
398    return new RegionState(ProtobufUtil.toRegionInfo(proto.getRegionInfo()),
399      State.convert(proto.getState()), proto.getStamp(), null);
400  }
401
402  /**
403   * Check if two states are the same, except timestamp
404   */
405  @Override
406  public boolean equals(Object obj) {
407    if (this == obj) return true;
408    if (obj == null || getClass() != obj.getClass()) {
409      return false;
410    }
411    RegionState tmp = (RegionState)obj;
412
413    return RegionInfo.COMPARATOR.compare(tmp.hri, hri) == 0 && tmp.state == state
414      && ((serverName != null && serverName.equals(tmp.serverName))
415        || (tmp.serverName == null && serverName == null));
416  }
417
418  /**
419   * Don't count timestamp in hash code calculation
420   */
421  @Override
422  public int hashCode() {
423    return (serverName != null ? serverName.hashCode() * 11 : 0)
424      + hri.hashCode() + 5 * state.ordinal();
425  }
426}