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