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