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.procedure;
019
020import java.io.IOException;
021import org.apache.hadoop.hbase.HBaseIOException;
022import org.apache.hadoop.hbase.HConstants;
023import org.apache.hadoop.hbase.TableName;
024import org.apache.hadoop.hbase.TableNotEnabledException;
025import org.apache.hadoop.hbase.TableNotFoundException;
026import org.apache.hadoop.hbase.client.BufferedMutator;
027import org.apache.hadoop.hbase.client.RegionInfo;
028import org.apache.hadoop.hbase.client.TableState;
029import org.apache.hadoop.hbase.constraint.ConstraintException;
030import org.apache.hadoop.hbase.master.MasterCoprocessorHost;
031import org.apache.hadoop.hbase.master.MasterFileSystem;
032import org.apache.hadoop.hbase.master.TableStateManager;
033import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer;
034import org.apache.hadoop.hbase.replication.ReplicationBarrierFamilyFormat;
035import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
036import org.apache.hadoop.hbase.wal.WALSplitUtil;
037import org.apache.yetus.audience.InterfaceAudience;
038import org.slf4j.Logger;
039import org.slf4j.LoggerFactory;
040
041import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
042import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos;
043import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.DisableTableState;
044
045@InterfaceAudience.Private
046public class DisableTableProcedure extends AbstractStateMachineTableProcedure<DisableTableState> {
047  private static final Logger LOG = LoggerFactory.getLogger(DisableTableProcedure.class);
048
049  private TableName tableName;
050  private boolean skipTableStateCheck;
051
052  public DisableTableProcedure() {
053    super();
054  }
055
056  /**
057   * Constructor
058   * @param env                 MasterProcedureEnv
059   * @param tableName           the table to operate on
060   * @param skipTableStateCheck whether to check table state
061   */
062  public DisableTableProcedure(final MasterProcedureEnv env, final TableName tableName,
063    final boolean skipTableStateCheck) throws HBaseIOException {
064    this(env, tableName, skipTableStateCheck, null);
065  }
066
067  /**
068   * Constructor
069   * @param env                 MasterProcedureEnv
070   * @param tableName           the table to operate on
071   * @param skipTableStateCheck whether to check table state
072   */
073  public DisableTableProcedure(final MasterProcedureEnv env, final TableName tableName,
074    final boolean skipTableStateCheck, final ProcedurePrepareLatch syncLatch)
075    throws HBaseIOException {
076    super(env, syncLatch);
077    this.tableName = tableName;
078    preflightChecks(env, true);
079    this.skipTableStateCheck = skipTableStateCheck;
080  }
081
082  @Override
083  protected Flow executeFromState(final MasterProcedureEnv env, final DisableTableState state)
084    throws InterruptedException {
085    LOG.trace("{} execute state={}", this, state);
086    try {
087      switch (state) {
088        case DISABLE_TABLE_PREPARE:
089          if (prepareDisable(env)) {
090            setNextState(DisableTableState.DISABLE_TABLE_PRE_OPERATION);
091          } else {
092            assert isFailed() : "disable should have an exception here";
093            return Flow.NO_MORE_STATE;
094          }
095          break;
096        case DISABLE_TABLE_PRE_OPERATION:
097          preDisable(env, state);
098          setNextState(DisableTableState.DISABLE_TABLE_SET_DISABLING_TABLE_STATE);
099          break;
100        case DISABLE_TABLE_SET_DISABLING_TABLE_STATE:
101          setTableStateToDisabling(env, tableName);
102          setNextState(DisableTableState.DISABLE_TABLE_MARK_REGIONS_OFFLINE);
103          break;
104        case DISABLE_TABLE_MARK_REGIONS_OFFLINE:
105          addChildProcedure(
106            env.getAssignmentManager().createUnassignProceduresForDisabling(tableName));
107          setNextState(DisableTableState.DISABLE_TABLE_ADD_REPLICATION_BARRIER);
108          break;
109        case DISABLE_TABLE_ADD_REPLICATION_BARRIER:
110          if (
111            env.getMasterServices().getTableDescriptors().get(tableName).hasGlobalReplicationScope()
112          ) {
113            MasterFileSystem fs = env.getMasterFileSystem();
114            try (BufferedMutator mutator = env.getMasterServices().getConnection()
115              .getBufferedMutator(TableName.META_TABLE_NAME)) {
116              for (RegionInfo region : env.getAssignmentManager().getRegionStates()
117                .getRegionsOfTable(tableName)) {
118                long maxSequenceId = WALSplitUtil.getMaxRegionSequenceId(
119                  env.getMasterConfiguration(), region, fs::getFileSystem, fs::getWALFileSystem);
120                long openSeqNum = maxSequenceId > 0 ? maxSequenceId + 1 : HConstants.NO_SEQNUM;
121                mutator.mutate(ReplicationBarrierFamilyFormat.makePutForReplicationBarrier(region,
122                  openSeqNum, EnvironmentEdgeManager.currentTime()));
123              }
124            }
125          }
126          setNextState(DisableTableState.DISABLE_TABLE_SET_DISABLED_TABLE_STATE);
127          break;
128        case DISABLE_TABLE_SET_DISABLED_TABLE_STATE:
129          setTableStateToDisabled(env, tableName);
130          setNextState(DisableTableState.DISABLE_TABLE_POST_OPERATION);
131          break;
132        case DISABLE_TABLE_POST_OPERATION:
133          postDisable(env, state);
134          return Flow.NO_MORE_STATE;
135        default:
136          throw new UnsupportedOperationException("Unhandled state=" + state);
137      }
138    } catch (IOException e) {
139      if (isRollbackSupported(state)) {
140        setFailure("master-disable-table", e);
141      } else {
142        LOG.warn("Retryable error in {}", this, e);
143      }
144    }
145    return Flow.HAS_MORE_STATE;
146  }
147
148  @Override
149  protected void rollbackState(final MasterProcedureEnv env, final DisableTableState state)
150    throws IOException {
151    // nothing to rollback, prepare-disable is just table-state checks.
152    // We can fail if the table does not exist or is not disabled.
153    switch (state) {
154      case DISABLE_TABLE_PRE_OPERATION:
155        return;
156      case DISABLE_TABLE_PREPARE:
157        releaseSyncLatch();
158        return;
159      default:
160        break;
161    }
162
163    // The delete doesn't have a rollback. The execution will succeed, at some point.
164    throw new UnsupportedOperationException("Unhandled state=" + state);
165  }
166
167  @Override
168  protected boolean isRollbackSupported(final DisableTableState state) {
169    switch (state) {
170      case DISABLE_TABLE_PREPARE:
171      case DISABLE_TABLE_PRE_OPERATION:
172        return true;
173      default:
174        return false;
175    }
176  }
177
178  @Override
179  protected DisableTableState getState(final int stateId) {
180    return DisableTableState.forNumber(stateId);
181  }
182
183  @Override
184  protected int getStateId(final DisableTableState state) {
185    return state.getNumber();
186  }
187
188  @Override
189  protected DisableTableState getInitialState() {
190    return DisableTableState.DISABLE_TABLE_PREPARE;
191  }
192
193  @Override
194  protected void serializeStateData(ProcedureStateSerializer serializer) throws IOException {
195    super.serializeStateData(serializer);
196
197    MasterProcedureProtos.DisableTableStateData.Builder disableTableMsg =
198      MasterProcedureProtos.DisableTableStateData.newBuilder()
199        .setUserInfo(MasterProcedureUtil.toProtoUserInfo(getUser()))
200        .setTableName(ProtobufUtil.toProtoTableName(tableName))
201        .setSkipTableStateCheck(skipTableStateCheck);
202
203    serializer.serialize(disableTableMsg.build());
204  }
205
206  @Override
207  protected void deserializeStateData(ProcedureStateSerializer serializer) throws IOException {
208    super.deserializeStateData(serializer);
209
210    MasterProcedureProtos.DisableTableStateData disableTableMsg =
211      serializer.deserialize(MasterProcedureProtos.DisableTableStateData.class);
212    setUser(MasterProcedureUtil.toUserInfo(disableTableMsg.getUserInfo()));
213    tableName = ProtobufUtil.toTableName(disableTableMsg.getTableName());
214    skipTableStateCheck = disableTableMsg.getSkipTableStateCheck();
215  }
216
217  // For disabling a table, we does not care whether a region can be online so hold the table xlock
218  // for ever. This will simplify the logic as we will not be conflict with procedures other than
219  // SCP.
220  @Override
221  protected boolean holdLock(MasterProcedureEnv env) {
222    return true;
223  }
224
225  @Override
226  public TableName getTableName() {
227    return tableName;
228  }
229
230  @Override
231  public TableOperationType getTableOperationType() {
232    return TableOperationType.DISABLE;
233  }
234
235  /**
236   * Action before any real action of disabling table. Set the exception in the procedure instead of
237   * throwing it. This approach is to deal with backward compatible with 1.0.
238   * @param env MasterProcedureEnv
239   */
240  private boolean prepareDisable(final MasterProcedureEnv env) throws IOException {
241    boolean canTableBeDisabled = true;
242    if (tableName.equals(TableName.META_TABLE_NAME)) {
243      setFailure("master-disable-table",
244        new ConstraintException("Cannot disable " + this.tableName));
245      canTableBeDisabled = false;
246    } else if (!env.getMasterServices().getTableDescriptors().exists(tableName)) {
247      setFailure("master-disable-table", new TableNotFoundException(tableName));
248      canTableBeDisabled = false;
249    } else if (!skipTableStateCheck) {
250      // There could be multiple client requests trying to disable or enable
251      // the table at the same time. Ensure only the first request is honored
252      // After that, no other requests can be accepted until the table reaches
253      // DISABLED or ENABLED.
254      //
255      // Note: in 1.0 release, we called TableStateManager.setTableStateIfInStates() to set
256      // the state to DISABLING from ENABLED. The implementation was done before table lock
257      // was implemented. With table lock, there is no need to set the state here (it will
258      // set the state later on). A quick state check should be enough for us to move forward.
259      TableStateManager tsm = env.getMasterServices().getTableStateManager();
260      TableState ts = tsm.getTableState(tableName);
261      if (!ts.isEnabled()) {
262        LOG.info("Not ENABLED, state={}, skipping disable; {}", ts.getState(), this);
263        setFailure("master-disable-table", new TableNotEnabledException(ts.toString()));
264        canTableBeDisabled = false;
265      }
266    }
267
268    // We are done the check. Future actions in this procedure could be done asynchronously.
269    releaseSyncLatch();
270
271    return canTableBeDisabled;
272  }
273
274  /**
275   * Action before disabling table.
276   * @param env   MasterProcedureEnv
277   * @param state the procedure state
278   */
279  protected void preDisable(final MasterProcedureEnv env, final DisableTableState state)
280    throws IOException, InterruptedException {
281    runCoprocessorAction(env, state);
282  }
283
284  /**
285   * Mark table state to Disabling
286   * @param env MasterProcedureEnv
287   */
288  private static void setTableStateToDisabling(final MasterProcedureEnv env,
289    final TableName tableName) throws IOException {
290    // Set table disabling flag up in zk.
291    env.getMasterServices().getTableStateManager().setTableState(tableName,
292      TableState.State.DISABLING);
293    LOG.info("Set {} to state={}", tableName, TableState.State.DISABLING);
294  }
295
296  /**
297   * Mark table state to Disabled
298   * @param env MasterProcedureEnv
299   */
300  protected static void setTableStateToDisabled(final MasterProcedureEnv env,
301    final TableName tableName) throws IOException {
302    // Flip the table to disabled
303    env.getMasterServices().getTableStateManager().setTableState(tableName,
304      TableState.State.DISABLED);
305    LOG.info("Set {} to state={}", tableName, TableState.State.DISABLED);
306  }
307
308  /**
309   * Action after disabling table.
310   * @param env   MasterProcedureEnv
311   * @param state the procedure state
312   */
313  protected void postDisable(final MasterProcedureEnv env, final DisableTableState state)
314    throws IOException, InterruptedException {
315    runCoprocessorAction(env, state);
316  }
317
318  /**
319   * Coprocessor Action.
320   * @param env   MasterProcedureEnv
321   * @param state the procedure state
322   */
323  private void runCoprocessorAction(final MasterProcedureEnv env, final DisableTableState state)
324    throws IOException, InterruptedException {
325    final MasterCoprocessorHost cpHost = env.getMasterCoprocessorHost();
326    if (cpHost != null) {
327      switch (state) {
328        case DISABLE_TABLE_PRE_OPERATION:
329          cpHost.preDisableTableAction(tableName, getUser());
330          break;
331        case DISABLE_TABLE_POST_OPERATION:
332          cpHost.postCompletedDisableTableAction(tableName, getUser());
333          break;
334        default:
335          throw new UnsupportedOperationException(this + " unhandled state=" + state);
336      }
337    }
338  }
339}