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.master.procedure;
020
021import java.io.IOException;
022import java.util.ArrayList;
023import java.util.List;
024
025import org.apache.hadoop.hbase.Cell;
026import org.apache.hadoop.hbase.HConstants;
027import org.apache.hadoop.hbase.MetaTableAccessor;
028import org.apache.hadoop.hbase.TableName;
029import org.apache.hadoop.hbase.TableNotDisabledException;
030import org.apache.hadoop.hbase.TableNotFoundException;
031import org.apache.yetus.audience.InterfaceAudience;
032import org.slf4j.Logger;
033import org.slf4j.LoggerFactory;
034import org.apache.hadoop.hbase.client.Connection;
035import org.apache.hadoop.hbase.client.Get;
036import org.apache.hadoop.hbase.client.RegionInfo;
037import org.apache.hadoop.hbase.client.RegionReplicaUtil;
038import org.apache.hadoop.hbase.client.Result;
039import org.apache.hadoop.hbase.client.Table;
040import org.apache.hadoop.hbase.client.TableDescriptor;
041import org.apache.hadoop.hbase.client.TableState;
042import org.apache.hadoop.hbase.master.MasterCoprocessorHost;
043import org.apache.hadoop.hbase.master.TableStateManager;
044import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer;
045import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
046import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos;
047import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.EnableTableState;
048
049@InterfaceAudience.Private
050public class EnableTableProcedure
051    extends AbstractStateMachineTableProcedure<EnableTableState> {
052  private static final Logger LOG = LoggerFactory.getLogger(EnableTableProcedure.class);
053
054  private TableName tableName;
055  private boolean skipTableStateCheck;
056
057  private Boolean traceEnabled = null;
058
059  public EnableTableProcedure() {
060    super();
061  }
062
063  /**
064   * Constructor
065   * @param env MasterProcedureEnv
066   * @param tableName the table to operate on
067   * @param skipTableStateCheck whether to check table state
068   */
069  public EnableTableProcedure(final MasterProcedureEnv env, final TableName tableName,
070      final boolean skipTableStateCheck) {
071    this(env, tableName, skipTableStateCheck, null);
072  }
073
074  /**
075   * Constructor
076   * @param env MasterProcedureEnv
077   * @param tableName the table to operate on
078   * @param skipTableStateCheck whether to check table state
079   */
080  public EnableTableProcedure(final MasterProcedureEnv env, final TableName tableName,
081      final boolean skipTableStateCheck, final ProcedurePrepareLatch syncLatch) {
082    super(env, syncLatch);
083    this.tableName = tableName;
084    this.skipTableStateCheck = skipTableStateCheck;
085  }
086
087  @SuppressWarnings("deprecation")
088  @Override
089  protected Flow executeFromState(final MasterProcedureEnv env, final EnableTableState state)
090      throws InterruptedException {
091    if (isTraceEnabled()) {
092      LOG.trace(this + " execute state=" + state);
093    }
094
095    try {
096      switch (state) {
097      case ENABLE_TABLE_PREPARE:
098        if (prepareEnable(env)) {
099          setNextState(EnableTableState.ENABLE_TABLE_PRE_OPERATION);
100        } else {
101          assert isFailed() : "enable should have an exception here";
102          return Flow.NO_MORE_STATE;
103        }
104        break;
105      case ENABLE_TABLE_PRE_OPERATION:
106        preEnable(env, state);
107        setNextState(EnableTableState.ENABLE_TABLE_SET_ENABLING_TABLE_STATE);
108        break;
109      case ENABLE_TABLE_SET_ENABLING_TABLE_STATE:
110        setTableStateToEnabling(env, tableName);
111        setNextState(EnableTableState.ENABLE_TABLE_MARK_REGIONS_ONLINE);
112        break;
113      case ENABLE_TABLE_MARK_REGIONS_ONLINE:
114        Connection connection = env.getMasterServices().getConnection();
115        // we will need to get the tableDescriptor here to see if there is a change in the replica
116        // count
117        TableDescriptor hTableDescriptor =
118            env.getMasterServices().getTableDescriptors().get(tableName);
119
120        // Get the replica count
121        int regionReplicaCount = hTableDescriptor.getRegionReplication();
122
123        // Get the regions for the table from memory; get both online and offline regions ('true').
124        List<RegionInfo> regionsOfTable =
125            env.getAssignmentManager().getRegionStates().getRegionsOfTable(tableName, true);
126
127        if (regionReplicaCount > 1) {
128          int currentMaxReplica = 0;
129          // Check if the regions in memory have replica regions as marked in META table
130          for (RegionInfo regionInfo : regionsOfTable) {
131            if (regionInfo.getReplicaId() > currentMaxReplica) {
132              // Iterating through all the list to identify the highest replicaID region.
133              // We can stop after checking with the first set of regions??
134              currentMaxReplica = regionInfo.getReplicaId();
135            }
136          }
137
138          // read the META table to know the actual number of replicas for the table - if there
139          // was a table modification on region replica then this will reflect the new entries also
140          int replicasFound =
141              getNumberOfReplicasFromMeta(connection, regionReplicaCount, regionsOfTable);
142          assert regionReplicaCount - 1 == replicasFound;
143          LOG.info(replicasFound + " META entries added for the given regionReplicaCount "
144              + regionReplicaCount + " for the table " + tableName.getNameAsString());
145          if (currentMaxReplica == (regionReplicaCount - 1)) {
146            if (LOG.isDebugEnabled()) {
147              LOG.debug("There is no change to the number of region replicas."
148                  + " Assigning the available regions." + " Current and previous"
149                  + "replica count is " + regionReplicaCount);
150            }
151          } else if (currentMaxReplica > (regionReplicaCount - 1)) {
152            // we have additional regions as the replica count has been decreased. Delete
153            // those regions because already the table is in the unassigned state
154            LOG.info("The number of replicas " + (currentMaxReplica + 1)
155                + "  is more than the region replica count " + regionReplicaCount);
156            List<RegionInfo> copyOfRegions = new ArrayList<RegionInfo>(regionsOfTable);
157            for (RegionInfo regionInfo : copyOfRegions) {
158              if (regionInfo.getReplicaId() > (regionReplicaCount - 1)) {
159                // delete the region from the regionStates
160                env.getAssignmentManager().getRegionStates().deleteRegion(regionInfo);
161                // remove it from the list of regions of the table
162                LOG.info("The regioninfo being removed is " + regionInfo + " "
163                    + regionInfo.getReplicaId());
164                regionsOfTable.remove(regionInfo);
165              }
166            }
167          } else {
168            // the replicasFound is less than the regionReplication
169            LOG.info(
170              "The number of replicas has been changed(increased)."
171              + " Lets assign the new region replicas. The previous replica count was "
172                  + (currentMaxReplica + 1) + ". The current replica count is "
173                  + regionReplicaCount);
174            regionsOfTable = RegionReplicaUtil.addReplicas(hTableDescriptor, regionsOfTable,
175              currentMaxReplica + 1, regionReplicaCount);
176          }
177        }
178        // Assign all the table regions. (including region replicas if added).
179        // createAssignProcedure will try to retain old assignments if possible.
180        addChildProcedure(env.getAssignmentManager().createAssignProcedures(regionsOfTable));
181        setNextState(EnableTableState.ENABLE_TABLE_SET_ENABLED_TABLE_STATE);
182        break;
183      case ENABLE_TABLE_SET_ENABLED_TABLE_STATE:
184        setTableStateToEnabled(env, tableName);
185        setNextState(EnableTableState.ENABLE_TABLE_POST_OPERATION);
186        break;
187      case ENABLE_TABLE_POST_OPERATION:
188        postEnable(env, state);
189        return Flow.NO_MORE_STATE;
190      default:
191        throw new UnsupportedOperationException("unhandled state=" + state);
192      }
193    } catch (IOException e) {
194      if (isRollbackSupported(state)) {
195        setFailure("master-enable-table", e);
196      } else {
197        LOG.warn("Retriable error trying to enable table=" + tableName +
198          " (in state=" + state + ")", e);
199      }
200    }
201    return Flow.HAS_MORE_STATE;
202  }
203
204  private int getNumberOfReplicasFromMeta(Connection connection, int regionReplicaCount,
205      List<RegionInfo> regionsOfTable) throws IOException {
206    Result r = getRegionFromMeta(connection, regionsOfTable);
207    int replicasFound = 0;
208    for (int i = 1; i < regionReplicaCount; i++) {
209      // Since we have already added the entries to the META we will be getting only that here
210      List<Cell> columnCells =
211          r.getColumnCells(HConstants.CATALOG_FAMILY, MetaTableAccessor.getServerColumn(i));
212      if (!columnCells.isEmpty()) {
213        replicasFound++;
214      }
215    }
216    return replicasFound;
217  }
218
219  private Result getRegionFromMeta(Connection connection, List<RegionInfo> regionsOfTable)
220      throws IOException {
221    byte[] metaKeyForRegion = MetaTableAccessor.getMetaKeyForRegion(regionsOfTable.get(0));
222    Get get = new Get(metaKeyForRegion);
223    get.addFamily(HConstants.CATALOG_FAMILY);
224    Table metaTable = MetaTableAccessor.getMetaHTable(connection);
225    Result r = metaTable.get(get);
226    return r;
227  }
228
229  @Override
230  protected void rollbackState(final MasterProcedureEnv env, final EnableTableState state)
231      throws IOException {
232    // nothing to rollback, prepare-disable is just table-state checks.
233    // We can fail if the table does not exist or is not disabled.
234    switch (state) {
235      case ENABLE_TABLE_PRE_OPERATION:
236        return;
237      case ENABLE_TABLE_PREPARE:
238        releaseSyncLatch();
239        return;
240      default:
241        break;
242    }
243
244    // The delete doesn't have a rollback. The execution will succeed, at some point.
245    throw new UnsupportedOperationException("unhandled state=" + state);
246  }
247
248  @Override
249  protected boolean isRollbackSupported(final EnableTableState state) {
250    switch (state) {
251      case ENABLE_TABLE_PREPARE:
252      case ENABLE_TABLE_PRE_OPERATION:
253        return true;
254      default:
255        return false;
256    }
257  }
258
259  @Override
260  protected EnableTableState getState(final int stateId) {
261    return EnableTableState.valueOf(stateId);
262  }
263
264  @Override
265  protected int getStateId(final EnableTableState state) {
266    return state.getNumber();
267  }
268
269  @Override
270  protected EnableTableState getInitialState() {
271    return EnableTableState.ENABLE_TABLE_PREPARE;
272  }
273
274  @Override
275  protected void serializeStateData(ProcedureStateSerializer serializer)
276      throws IOException {
277    super.serializeStateData(serializer);
278
279    MasterProcedureProtos.EnableTableStateData.Builder enableTableMsg =
280        MasterProcedureProtos.EnableTableStateData.newBuilder()
281            .setUserInfo(MasterProcedureUtil.toProtoUserInfo(getUser()))
282            .setTableName(ProtobufUtil.toProtoTableName(tableName))
283            .setSkipTableStateCheck(skipTableStateCheck);
284
285    serializer.serialize(enableTableMsg.build());
286  }
287
288  @Override
289  protected void deserializeStateData(ProcedureStateSerializer serializer)
290      throws IOException {
291    super.deserializeStateData(serializer);
292
293    MasterProcedureProtos.EnableTableStateData enableTableMsg =
294        serializer.deserialize(MasterProcedureProtos.EnableTableStateData.class);
295    setUser(MasterProcedureUtil.toUserInfo(enableTableMsg.getUserInfo()));
296    tableName = ProtobufUtil.toTableName(enableTableMsg.getTableName());
297    skipTableStateCheck = enableTableMsg.getSkipTableStateCheck();
298  }
299
300  @Override
301  public TableName getTableName() {
302    return tableName;
303  }
304
305  @Override
306  public TableOperationType getTableOperationType() {
307    return TableOperationType.ENABLE;
308  }
309
310
311  /**
312   * Action before any real action of enabling table. Set the exception in the procedure instead
313   * of throwing it.  This approach is to deal with backward compatible with 1.0.
314   * @param env MasterProcedureEnv
315   * @return whether the table passes the necessary checks
316   * @throws IOException
317   */
318  private boolean prepareEnable(final MasterProcedureEnv env) throws IOException {
319    boolean canTableBeEnabled = true;
320
321    // Check whether table exists
322    if (!MetaTableAccessor.tableExists(env.getMasterServices().getConnection(), tableName)) {
323      setFailure("master-enable-table", new TableNotFoundException(tableName));
324      canTableBeEnabled = false;
325    } else if (!skipTableStateCheck) {
326      // There could be multiple client requests trying to disable or enable
327      // the table at the same time. Ensure only the first request is honored
328      // After that, no other requests can be accepted until the table reaches
329      // DISABLED or ENABLED.
330      //
331      // Note: in 1.0 release, we called TableStateManager.setTableStateIfInStates() to set
332      // the state to ENABLING from DISABLED. The implementation was done before table lock
333      // was implemented. With table lock, there is no need to set the state here (it will
334      // set the state later on). A quick state check should be enough for us to move forward.
335      TableStateManager tsm = env.getMasterServices().getTableStateManager();
336      TableState ts = tsm.getTableState(tableName);
337      if(!ts.isDisabled()){
338        LOG.info("Not DISABLED tableState={}; skipping enable; {}", ts.getState(), this);
339        setFailure("master-enable-table", new TableNotDisabledException(ts.toString()));
340        canTableBeEnabled = false;
341      }
342    }
343
344    // We are done the check. Future actions in this procedure could be done asynchronously.
345    releaseSyncLatch();
346
347    return canTableBeEnabled;
348  }
349
350  /**
351   * Action before enabling table.
352   * @param env MasterProcedureEnv
353   * @param state the procedure state
354   * @throws IOException
355   * @throws InterruptedException
356   */
357  private void preEnable(final MasterProcedureEnv env, final EnableTableState state)
358      throws IOException, InterruptedException {
359    runCoprocessorAction(env, state);
360  }
361
362  /**
363   * Mark table state to Enabling
364   * @param env MasterProcedureEnv
365   * @param tableName the target table
366   * @throws IOException
367   */
368  protected static void setTableStateToEnabling(
369      final MasterProcedureEnv env,
370      final TableName tableName) throws IOException {
371    // Set table disabling flag up in zk.
372    LOG.info("Attempting to enable the table " + tableName);
373    env.getMasterServices().getTableStateManager().setTableState(
374      tableName,
375      TableState.State.ENABLING);
376  }
377
378  /**
379   * Mark table state to Enabled
380   * @param env MasterProcedureEnv
381   * @throws IOException
382   */
383  protected static void setTableStateToEnabled(
384      final MasterProcedureEnv env,
385      final TableName tableName) throws IOException {
386    // Flip the table to Enabled
387    env.getMasterServices().getTableStateManager().setTableState(
388      tableName,
389      TableState.State.ENABLED);
390    LOG.info("Table '" + tableName + "' was successfully enabled.");
391  }
392
393  /**
394   * Action after enabling table.
395   * @param env MasterProcedureEnv
396   * @param state the procedure state
397   * @throws IOException
398   * @throws InterruptedException
399   */
400  private void postEnable(final MasterProcedureEnv env, final EnableTableState state)
401      throws IOException, InterruptedException {
402    runCoprocessorAction(env, state);
403  }
404
405  /**
406   * The procedure could be restarted from a different machine. If the variable is null, we need to
407   * retrieve it.
408   * @return traceEnabled
409   */
410  private Boolean isTraceEnabled() {
411    if (traceEnabled == null) {
412      traceEnabled = LOG.isTraceEnabled();
413    }
414    return traceEnabled;
415  }
416
417  /**
418   * Coprocessor Action.
419   * @param env MasterProcedureEnv
420   * @param state the procedure state
421   * @throws IOException
422   * @throws InterruptedException
423   */
424  private void runCoprocessorAction(final MasterProcedureEnv env, final EnableTableState state)
425      throws IOException, InterruptedException {
426    final MasterCoprocessorHost cpHost = env.getMasterCoprocessorHost();
427    if (cpHost != null) {
428      switch (state) {
429        case ENABLE_TABLE_PRE_OPERATION:
430          cpHost.preEnableTableAction(getTableName(), getUser());
431          break;
432        case ENABLE_TABLE_POST_OPERATION:
433          cpHost.postCompletedEnableTableAction(getTableName(), getUser());
434          break;
435        default:
436          throw new UnsupportedOperationException(this + " unhandled state=" + state);
437      }
438    }
439  }
440}