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.assignment;
019
020import edu.umd.cs.findbugs.annotations.Nullable;
021import java.io.IOException;
022import org.apache.hadoop.hbase.HBaseIOException;
023import org.apache.hadoop.hbase.ServerName;
024import org.apache.hadoop.hbase.TableName;
025import org.apache.hadoop.hbase.client.RegionInfo;
026import org.apache.hadoop.hbase.client.RegionReplicaUtil;
027import org.apache.hadoop.hbase.client.RetriesExhaustedException;
028import org.apache.hadoop.hbase.master.MetricsAssignmentManager;
029import org.apache.hadoop.hbase.master.RegionState.State;
030import org.apache.hadoop.hbase.master.procedure.AbstractStateMachineRegionProcedure;
031import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv;
032import org.apache.hadoop.hbase.master.procedure.ServerCrashProcedure;
033import org.apache.hadoop.hbase.procedure2.Procedure;
034import org.apache.hadoop.hbase.procedure2.ProcedureMetrics;
035import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer;
036import org.apache.hadoop.hbase.procedure2.ProcedureSuspendedException;
037import org.apache.hadoop.hbase.procedure2.ProcedureUtil;
038import org.apache.hadoop.hbase.procedure2.ProcedureYieldException;
039import org.apache.hadoop.hbase.util.RetryCounter;
040import org.apache.yetus.audience.InterfaceAudience;
041import org.slf4j.Logger;
042import org.slf4j.LoggerFactory;
043
044import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
045import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState;
046import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionStateData;
047import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionTransitionType;
048import org.apache.hadoop.hbase.shaded.protobuf.generated.ProcedureProtos;
049import org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition.TransitionCode;
050
051/**
052 * The procedure to deal with the state transition of a region. A region with a TRSP in place is
053 * called RIT, i.e, RegionInTransition.
054 * <p/>
055 * It can be used to assign/unassign/reopen/move a region, and for
056 * {@link #unassign(MasterProcedureEnv, RegionInfo)} and
057 * {@link #reopen(MasterProcedureEnv, RegionInfo)}, you do not need to specify a target server, and
058 * for {@link #assign(MasterProcedureEnv, RegionInfo, ServerName)} and
059 * {@link #move(MasterProcedureEnv, RegionInfo, ServerName)}, if you want to you can provide a
060 * target server. And for {@link #move(MasterProcedureEnv, RegionInfo, ServerName)}, if you do not
061 * specify a targetServer, we will select one randomly.
062 * <p/>
063 * <p/>
064 * The typical state transition for assigning a region is:
065 *
066 * <pre>
067 * GET_ASSIGN_CANDIDATE ------> OPEN -----> CONFIRM_OPENED
068 * </pre>
069 *
070 * Notice that, if there are failures we may go back to the {@code GET_ASSIGN_CANDIDATE} state to
071 * try again.
072 * <p/>
073 * The typical state transition for unassigning a region is:
074 *
075 * <pre>
076 * CLOSE -----> CONFIRM_CLOSED
077 * </pre>
078 *
079 * Here things go a bit different, if there are failures, especially that if there is a server
080 * crash, we will go to the {@code GET_ASSIGN_CANDIDATE} state to bring the region online first, and
081 * then go through the normal way to unassign it.
082 * <p/>
083 * The typical state transition for reopening/moving a region is:
084 *
085 * <pre>
086 * CLOSE -----> CONFIRM_CLOSED -----> GET_ASSIGN_CANDIDATE ------> OPEN -----> CONFIRM_OPENED
087 * </pre>
088 *
089 * The retry logic is the same with the above assign/unassign.
090 * <p/>
091 * Notice that, although we allow specify a target server, it just acts as a candidate, we do not
092 * guarantee that the region will finally be on the target server. If this is important for you, you
093 * should check whether the region is on the target server after the procedure is finished.
094 * <p/>
095 * When you want to schedule a TRSP, please check whether there is still one for this region, and
096 * the check should be under the RegionStateNode lock. We will remove the TRSP from a
097 * RegionStateNode when we are done, see the code in {@code reportTransition} method below. There
098 * could be at most one TRSP for a give region.
099 */
100@InterfaceAudience.Private
101public class TransitRegionStateProcedure
102    extends AbstractStateMachineRegionProcedure<RegionStateTransitionState> {
103
104  private static final Logger LOG = LoggerFactory.getLogger(TransitRegionStateProcedure.class);
105
106  private TransitionType type;
107
108  private RegionStateTransitionState initialState;
109
110  private RegionStateTransitionState lastState;
111
112  // the candidate where we want to assign the region to.
113  private ServerName assignCandidate;
114
115  private boolean forceNewPlan;
116
117  private RetryCounter retryCounter;
118
119  private RegionRemoteProcedureBase remoteProc;
120
121  public TransitRegionStateProcedure() {
122  }
123
124  private void setInitialAndLastState() {
125    switch (type) {
126      case ASSIGN:
127        initialState = RegionStateTransitionState.REGION_STATE_TRANSITION_GET_ASSIGN_CANDIDATE;
128        lastState = RegionStateTransitionState.REGION_STATE_TRANSITION_CONFIRM_OPENED;
129        break;
130      case UNASSIGN:
131        initialState = RegionStateTransitionState.REGION_STATE_TRANSITION_CLOSE;
132        lastState = RegionStateTransitionState.REGION_STATE_TRANSITION_CONFIRM_CLOSED;
133        break;
134      case MOVE:
135      case REOPEN:
136        initialState = RegionStateTransitionState.REGION_STATE_TRANSITION_CLOSE;
137        lastState = RegionStateTransitionState.REGION_STATE_TRANSITION_CONFIRM_OPENED;
138        break;
139      default:
140        throw new IllegalArgumentException("Unknown TransitionType: " + type);
141    }
142  }
143
144  protected TransitRegionStateProcedure(MasterProcedureEnv env, RegionInfo hri,
145      ServerName assignCandidate, boolean forceNewPlan, TransitionType type) {
146    super(env, hri);
147    this.assignCandidate = assignCandidate;
148    this.forceNewPlan = forceNewPlan;
149    this.type = type;
150    setInitialAndLastState();
151
152    // when do reopen TRSP, let the rs know the targetServer so it can keep some info on close
153    if (type == TransitionType.REOPEN) {
154      this.assignCandidate = getRegionStateNode(env).getRegionLocation();
155    }
156  }
157
158  @Override
159  public TableOperationType getTableOperationType() {
160    // TODO: maybe we should make another type here, REGION_TRANSITION?
161    return TableOperationType.REGION_EDIT;
162  }
163
164  @Override
165  protected boolean waitInitialized(MasterProcedureEnv env) {
166    if (TableName.isMetaTableName(getTableName())) {
167      return false;
168    }
169    // First we need meta to be loaded, and second, if meta is not online then we will likely to
170    // fail when updating meta so we wait until it is assigned.
171    AssignmentManager am = env.getAssignmentManager();
172    return am.waitMetaLoaded(this) || am.waitMetaAssigned(this, getRegion());
173  }
174
175  private void queueAssign(MasterProcedureEnv env, RegionStateNode regionNode)
176      throws ProcedureSuspendedException {
177    boolean retain = false;
178    if (forceNewPlan) {
179      // set the region location to null if forceNewPlan is true
180      regionNode.setRegionLocation(null);
181    } else {
182      if (assignCandidate != null) {
183        retain = assignCandidate.equals(regionNode.getLastHost());
184        regionNode.setRegionLocation(assignCandidate);
185      } else if (regionNode.getLastHost() != null) {
186        retain = true;
187        LOG.info("Setting lastHost as the region location {}", regionNode.getLastHost());
188        regionNode.setRegionLocation(regionNode.getLastHost());
189      }
190    }
191    LOG.info("Starting {}; {}; forceNewPlan={}, retain={}", this, regionNode.toShortString(),
192      forceNewPlan, retain);
193    env.getAssignmentManager().queueAssign(regionNode);
194    setNextState(RegionStateTransitionState.REGION_STATE_TRANSITION_OPEN);
195    if (regionNode.getProcedureEvent().suspendIfNotReady(this)) {
196      throw new ProcedureSuspendedException();
197    }
198  }
199
200  private void openRegion(MasterProcedureEnv env, RegionStateNode regionNode) throws IOException {
201    ServerName loc = regionNode.getRegionLocation();
202    if (loc == null) {
203      LOG.warn("No location specified for {}, jump back to state {} to get one", getRegion(),
204        RegionStateTransitionState.REGION_STATE_TRANSITION_GET_ASSIGN_CANDIDATE);
205      setNextState(RegionStateTransitionState.REGION_STATE_TRANSITION_GET_ASSIGN_CANDIDATE);
206      return;
207    }
208    env.getAssignmentManager().regionOpening(regionNode);
209    addChildProcedure(new OpenRegionProcedure(this, getRegion(), loc));
210    setNextState(RegionStateTransitionState.REGION_STATE_TRANSITION_CONFIRM_OPENED);
211  }
212
213  private Flow confirmOpened(MasterProcedureEnv env, RegionStateNode regionNode)
214      throws IOException {
215    if (regionNode.isInState(State.OPEN)) {
216      retryCounter = null;
217      if (lastState == RegionStateTransitionState.REGION_STATE_TRANSITION_CONFIRM_OPENED) {
218        // we are the last state, finish
219        regionNode.unsetProcedure(this);
220        ServerCrashProcedure.updateProgress(env, getParentProcId());
221        return Flow.NO_MORE_STATE;
222      }
223      // It is possible that we arrive here but confirm opened is not the last state, for example,
224      // when merging or splitting a region, we unassign the region from a RS and the RS is crashed,
225      // then there will be recovered edits for this region, we'd better make the region online
226      // again and then unassign it, otherwise we have to fail the merge/split procedure as we may
227      // loss data.
228      setNextState(RegionStateTransitionState.REGION_STATE_TRANSITION_CLOSE);
229      return Flow.HAS_MORE_STATE;
230    }
231
232    int retries = env.getAssignmentManager().getRegionStates().addToFailedOpen(regionNode)
233        .incrementAndGetRetries();
234    int maxAttempts = env.getAssignmentManager().getAssignMaxAttempts();
235    LOG.info("Retry={} of max={}; {}; {}", retries, maxAttempts, this, regionNode.toShortString());
236
237    if (retries >= maxAttempts) {
238      env.getAssignmentManager().regionFailedOpen(regionNode, true);
239      setFailure(getClass().getSimpleName(), new RetriesExhaustedException(
240        "Max attempts " + env.getAssignmentManager().getAssignMaxAttempts() + " exceeded"));
241      regionNode.unsetProcedure(this);
242      return Flow.NO_MORE_STATE;
243    }
244
245    env.getAssignmentManager().regionFailedOpen(regionNode, false);
246    // we failed to assign the region, force a new plan
247    forceNewPlan = true;
248    regionNode.setRegionLocation(null);
249    setNextState(RegionStateTransitionState.REGION_STATE_TRANSITION_GET_ASSIGN_CANDIDATE);
250
251    if (retries > env.getAssignmentManager().getAssignRetryImmediatelyMaxAttempts()) {
252      // Throw exception to backoff and retry when failed open too many times
253      throw new HBaseIOException("Failed confirm OPEN of " + regionNode +
254          " (remote log may yield more detail on why).");
255    } else {
256      // Here we do not throw exception because we want to the region to be online ASAP
257      return Flow.HAS_MORE_STATE;
258    }
259  }
260
261  private void closeRegion(MasterProcedureEnv env, RegionStateNode regionNode) throws IOException {
262    if (regionNode.isInState(State.OPEN, State.CLOSING, State.MERGING, State.SPLITTING)) {
263      // this is the normal case
264      env.getAssignmentManager().regionClosing(regionNode);
265      addChildProcedure(new CloseRegionProcedure(this, getRegion(), regionNode.getRegionLocation(),
266        assignCandidate));
267      setNextState(RegionStateTransitionState.REGION_STATE_TRANSITION_CONFIRM_CLOSED);
268    } else {
269      forceNewPlan = true;
270      regionNode.setRegionLocation(null);
271      setNextState(RegionStateTransitionState.REGION_STATE_TRANSITION_GET_ASSIGN_CANDIDATE);
272    }
273  }
274
275  private Flow confirmClosed(MasterProcedureEnv env, RegionStateNode regionNode)
276      throws IOException {
277    if (regionNode.isInState(State.CLOSED)) {
278      retryCounter = null;
279      if (lastState == RegionStateTransitionState.REGION_STATE_TRANSITION_CONFIRM_CLOSED) {
280        // we are the last state, finish
281        regionNode.unsetProcedure(this);
282        return Flow.NO_MORE_STATE;
283      }
284      // This means we need to open the region again, should be a move or reopen
285      setNextState(RegionStateTransitionState.REGION_STATE_TRANSITION_GET_ASSIGN_CANDIDATE);
286      return Flow.HAS_MORE_STATE;
287    }
288    if (regionNode.isInState(State.CLOSING)) {
289      // This is possible, think the target RS crashes and restarts immediately, the close region
290      // operation will return a NotServingRegionException soon, we can only recover after SCP takes
291      // care of this RS. So here we throw an IOException to let upper layer to retry with backoff.
292      setNextState(RegionStateTransitionState.REGION_STATE_TRANSITION_CLOSE);
293      throw new HBaseIOException("Failed to close region");
294    }
295    // abnormally closed, need to reopen it, no matter what is the last state, see the comment in
296    // confirmOpened for more details that why we need to reopen the region first even if we just
297    // want to close it.
298    // The only exception is for non-default replica, where we do not need to deal with recovered
299    // edits. Notice that the region will remain in ABNORMALLY_CLOSED state, the upper layer need to
300    // deal with this state. For non-default replica, this is usually the same with CLOSED.
301    assert regionNode.isInState(State.ABNORMALLY_CLOSED);
302    if (!RegionReplicaUtil.isDefaultReplica(getRegion()) &&
303      lastState == RegionStateTransitionState.REGION_STATE_TRANSITION_CONFIRM_CLOSED) {
304      regionNode.unsetProcedure(this);
305      return Flow.NO_MORE_STATE;
306    }
307    retryCounter = null;
308    setNextState(RegionStateTransitionState.REGION_STATE_TRANSITION_GET_ASSIGN_CANDIDATE);
309    return Flow.HAS_MORE_STATE;
310  }
311
312  // Override to lock RegionStateNode
313  @SuppressWarnings("rawtypes")
314  @Override
315  protected Procedure[] execute(MasterProcedureEnv env)
316      throws ProcedureSuspendedException, ProcedureYieldException, InterruptedException {
317    RegionStateNode regionNode =
318      env.getAssignmentManager().getRegionStates().getOrCreateRegionStateNode(getRegion());
319    regionNode.lock();
320    try {
321      return super.execute(env);
322    } finally {
323      regionNode.unlock();
324    }
325  }
326
327  private RegionStateNode getRegionStateNode(MasterProcedureEnv env) {
328    return env.getAssignmentManager().getRegionStates().getOrCreateRegionStateNode(getRegion());
329  }
330
331  @Override
332  protected Flow executeFromState(MasterProcedureEnv env, RegionStateTransitionState state)
333      throws ProcedureSuspendedException, ProcedureYieldException, InterruptedException {
334    RegionStateNode regionNode = getRegionStateNode(env);
335    try {
336      switch (state) {
337        case REGION_STATE_TRANSITION_GET_ASSIGN_CANDIDATE:
338          // Need to do some sanity check for replica region, if the region does not exist at
339          // master, do not try to assign the replica region, log error and return.
340          if (!RegionReplicaUtil.isDefaultReplica(regionNode.getRegionInfo())) {
341            RegionInfo defaultRI =
342              RegionReplicaUtil.getRegionInfoForDefaultReplica(regionNode.getRegionInfo());
343            if (env.getMasterServices().getAssignmentManager().getRegionStates().
344              getRegionStateNode(defaultRI) == null) {
345              LOG.error(
346                "Cannot assign replica region {} because its primary region {} does not exist.",
347                regionNode.getRegionInfo(), defaultRI);
348              regionNode.unsetProcedure(this);
349              return Flow.NO_MORE_STATE;
350            }
351          }
352          queueAssign(env, regionNode);
353          return Flow.HAS_MORE_STATE;
354        case REGION_STATE_TRANSITION_OPEN:
355          openRegion(env, regionNode);
356          return Flow.HAS_MORE_STATE;
357        case REGION_STATE_TRANSITION_CONFIRM_OPENED:
358          return confirmOpened(env, regionNode);
359        case REGION_STATE_TRANSITION_CLOSE:
360          closeRegion(env, regionNode);
361          return Flow.HAS_MORE_STATE;
362        case REGION_STATE_TRANSITION_CONFIRM_CLOSED:
363          return confirmClosed(env, regionNode);
364        default:
365          throw new UnsupportedOperationException("unhandled state=" + state);
366      }
367    } catch (IOException e) {
368      if (retryCounter == null) {
369        retryCounter = ProcedureUtil.createRetryCounter(env.getMasterConfiguration());
370      }
371      long backoff = retryCounter.getBackoffTimeAndIncrementAttempts();
372      LOG.warn(
373        "Failed transition, suspend {}secs {}; {}; waiting on rectified condition fixed " +
374          "by other Procedure or operator intervention",
375        backoff / 1000, this, regionNode.toShortString(), e);
376      setTimeout(Math.toIntExact(backoff));
377      setState(ProcedureProtos.ProcedureState.WAITING_TIMEOUT);
378      skipPersistence();
379      throw new ProcedureSuspendedException();
380    }
381  }
382
383  /**
384   * At end of timeout, wake ourselves up so we run again.
385   */
386  @Override
387  protected synchronized boolean setTimeoutFailure(MasterProcedureEnv env) {
388    setState(ProcedureProtos.ProcedureState.RUNNABLE);
389    env.getProcedureScheduler().addFront(this);
390    return false; // 'false' means that this procedure handled the timeout
391  }
392
393  // Should be called with RegionStateNode locked
394  public void reportTransition(MasterProcedureEnv env, RegionStateNode regionNode,
395      ServerName serverName, TransitionCode code, long seqId, long procId) throws IOException {
396    if (remoteProc == null) {
397      LOG.warn(
398        "There is no outstanding remote region procedure for {}, serverName={}, code={}," +
399          " seqId={}, proc={}, should be a retry, ignore",
400        regionNode, serverName, code, seqId, this);
401      return;
402    }
403    // The procId could be -1 if it is from an old region server, we need to deal with it so that we
404    // can do rolling upgraing.
405    if (procId >= 0 && remoteProc.getProcId() != procId) {
406      LOG.warn(
407        "The pid of remote region procedure for {} is {}, the reported pid={}, serverName={}," +
408          " code={}, seqId={}, proc={}, should be a retry, ignore",
409        regionNode, remoteProc.getProcId(), procId, serverName, code, seqId, this);
410      return;
411    }
412    remoteProc.reportTransition(env, regionNode, serverName, code, seqId);
413  }
414
415  // Should be called with RegionStateNode locked
416  public void serverCrashed(MasterProcedureEnv env, RegionStateNode regionNode,
417      ServerName serverName) throws IOException {
418    // force to assign to a new candidate server
419    // AssignmentManager#regionClosedAbnormally will set region location to null
420    // TODO: the forceNewPlan flag not be persistent so if master crash then the flag will be lost.
421    // But assign to old server is not big deal because it not effect correctness.
422    // See HBASE-23035 for more details.
423    forceNewPlan = true;
424    if (remoteProc != null) {
425      // this means we are waiting for the sub procedure, so wake it up
426      remoteProc.serverCrashed(env, regionNode, serverName);
427    } else {
428      // we are in RUNNING state, just update the region state, and we will process it later.
429      env.getAssignmentManager().regionClosedAbnormally(regionNode);
430    }
431  }
432
433  void attachRemoteProc(RegionRemoteProcedureBase proc) {
434    this.remoteProc = proc;
435  }
436
437  void unattachRemoteProc(RegionRemoteProcedureBase proc) {
438    assert this.remoteProc == proc;
439    this.remoteProc = null;
440  }
441
442  // will be called after we finish loading the meta entry for this region.
443  // used to change the state of the region node if we have a sub procedure, as we may not persist
444  // the state to meta yet. See the code in RegionRemoteProcedureBase.execute for more details.
445  void stateLoaded(AssignmentManager am, RegionStateNode regionNode) {
446    if (remoteProc != null) {
447      remoteProc.stateLoaded(am, regionNode);
448    }
449  }
450
451  @Override
452  protected void rollbackState(MasterProcedureEnv env, RegionStateTransitionState state)
453      throws IOException, InterruptedException {
454    // no rollback
455    throw new UnsupportedOperationException();
456  }
457
458  @Override
459  protected RegionStateTransitionState getState(int stateId) {
460    return RegionStateTransitionState.forNumber(stateId);
461  }
462
463  @Override
464  protected int getStateId(RegionStateTransitionState state) {
465    return state.getNumber();
466  }
467
468  @Override
469  protected RegionStateTransitionState getInitialState() {
470    return initialState;
471  }
472
473  private static TransitionType convert(RegionTransitionType type) {
474    switch (type) {
475      case ASSIGN:
476        return TransitionType.ASSIGN;
477      case UNASSIGN:
478        return TransitionType.UNASSIGN;
479      case MOVE:
480        return TransitionType.MOVE;
481      case REOPEN:
482        return TransitionType.REOPEN;
483      default:
484        throw new IllegalArgumentException("Unknown RegionTransitionType: " + type);
485    }
486  }
487
488  private static RegionTransitionType convert(TransitionType type) {
489    switch (type) {
490      case ASSIGN:
491        return RegionTransitionType.ASSIGN;
492      case UNASSIGN:
493        return RegionTransitionType.UNASSIGN;
494      case MOVE:
495        return RegionTransitionType.MOVE;
496      case REOPEN:
497        return RegionTransitionType.REOPEN;
498      default:
499        throw new IllegalArgumentException("Unknown TransitionType: " + type);
500    }
501  }
502
503  @Override
504  protected void serializeStateData(ProcedureStateSerializer serializer) throws IOException {
505    super.serializeStateData(serializer);
506    RegionStateTransitionStateData.Builder builder = RegionStateTransitionStateData.newBuilder()
507      .setType(convert(type)).setForceNewPlan(forceNewPlan);
508    if (assignCandidate != null) {
509      builder.setAssignCandidate(ProtobufUtil.toServerName(assignCandidate));
510    }
511    serializer.serialize(builder.build());
512  }
513
514  @Override
515  protected void deserializeStateData(ProcedureStateSerializer serializer) throws IOException {
516    super.deserializeStateData(serializer);
517    RegionStateTransitionStateData data =
518      serializer.deserialize(RegionStateTransitionStateData.class);
519    type = convert(data.getType());
520    setInitialAndLastState();
521    forceNewPlan = data.getForceNewPlan();
522    if (data.hasAssignCandidate()) {
523      assignCandidate = ProtobufUtil.toServerName(data.getAssignCandidate());
524    }
525  }
526
527  @Override
528  protected ProcedureMetrics getProcedureMetrics(MasterProcedureEnv env) {
529    MetricsAssignmentManager metrics = env.getAssignmentManager().getAssignmentManagerMetrics();
530    switch (type) {
531      case ASSIGN:
532        return metrics.getAssignProcMetrics();
533      case UNASSIGN:
534        return metrics.getUnassignProcMetrics();
535      case MOVE:
536        return metrics.getMoveProcMetrics();
537      case REOPEN:
538        return metrics.getReopenProcMetrics();
539      default:
540        throw new IllegalArgumentException("Unknown transition type: " + type);
541    }
542  }
543
544  @Override
545  public void toStringClassDetails(StringBuilder sb) {
546    super.toStringClassDetails(sb);
547    if (initialState == RegionStateTransitionState.REGION_STATE_TRANSITION_GET_ASSIGN_CANDIDATE) {
548      sb.append(", ASSIGN");
549    } else if (lastState == RegionStateTransitionState.REGION_STATE_TRANSITION_CONFIRM_CLOSED) {
550      sb.append(", UNASSIGN");
551    } else {
552      sb.append(", REOPEN/MOVE");
553    }
554  }
555
556  private static TransitRegionStateProcedure setOwner(MasterProcedureEnv env,
557      TransitRegionStateProcedure proc) {
558    proc.setOwner(env.getRequestUser().getShortName());
559    return proc;
560  }
561
562  public enum TransitionType {
563    ASSIGN, UNASSIGN, MOVE, REOPEN
564  }
565
566  // Be careful that, when you call these 4 methods below, you need to manually attach the returned
567  // procedure with the RegionStateNode, otherwise the procedure will quit immediately without doing
568  // anything. See the comment in executeFromState to find out why we need this assumption.
569  public static TransitRegionStateProcedure assign(MasterProcedureEnv env, RegionInfo region,
570      @Nullable ServerName targetServer) {
571    return assign(env, region, false, targetServer);
572  }
573
574  public static TransitRegionStateProcedure assign(MasterProcedureEnv env, RegionInfo region,
575      boolean forceNewPlan, @Nullable ServerName targetServer) {
576    return setOwner(env, new TransitRegionStateProcedure(env, region, targetServer, forceNewPlan,
577        TransitionType.ASSIGN));
578  }
579
580  public static TransitRegionStateProcedure unassign(MasterProcedureEnv env, RegionInfo region) {
581    return setOwner(env,
582      new TransitRegionStateProcedure(env, region, null, false, TransitionType.UNASSIGN));
583  }
584
585  public static TransitRegionStateProcedure reopen(MasterProcedureEnv env, RegionInfo region) {
586    return setOwner(env,
587      new TransitRegionStateProcedure(env, region, null, false, TransitionType.REOPEN));
588  }
589
590  public static TransitRegionStateProcedure move(MasterProcedureEnv env, RegionInfo region,
591      @Nullable ServerName targetServer) {
592    return setOwner(env, new TransitRegionStateProcedure(env, region, targetServer,
593      targetServer == null, TransitionType.MOVE));
594  }
595}