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.Arrays; 023import java.util.Collections; 024import java.util.HashSet; 025import java.util.List; 026import java.util.Set; 027 028import org.apache.hadoop.hbase.ConcurrentTableModificationException; 029import org.apache.hadoop.hbase.DoNotRetryIOException; 030import org.apache.hadoop.hbase.HBaseIOException; 031import org.apache.hadoop.hbase.HConstants; 032import org.apache.hadoop.hbase.MetaTableAccessor; 033import org.apache.hadoop.hbase.TableName; 034import org.apache.hadoop.hbase.TableNotDisabledException; 035import org.apache.hadoop.hbase.TableNotFoundException; 036import org.apache.hadoop.hbase.client.Connection; 037import org.apache.hadoop.hbase.client.RegionInfo; 038import org.apache.hadoop.hbase.client.Result; 039import org.apache.hadoop.hbase.client.ResultScanner; 040import org.apache.hadoop.hbase.client.Scan; 041import org.apache.hadoop.hbase.client.Table; 042import org.apache.hadoop.hbase.client.TableDescriptor; 043import org.apache.hadoop.hbase.client.TableState; 044import org.apache.hadoop.hbase.master.MasterCoprocessorHost; 045import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer; 046import org.apache.hadoop.hbase.util.Bytes; 047import org.apache.hadoop.hbase.util.ServerRegionReplicaUtil; 048import org.apache.yetus.audience.InterfaceAudience; 049import org.slf4j.Logger; 050import org.slf4j.LoggerFactory; 051 052import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil; 053import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos; 054import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.ModifyTableState; 055 056@InterfaceAudience.Private 057public class ModifyTableProcedure 058 extends AbstractStateMachineTableProcedure<ModifyTableState> { 059 private static final Logger LOG = LoggerFactory.getLogger(ModifyTableProcedure.class); 060 061 private TableDescriptor unmodifiedTableDescriptor = null; 062 private TableDescriptor modifiedTableDescriptor; 063 private boolean deleteColumnFamilyInModify; 064 private boolean shouldCheckDescriptor; 065 /** 066 * List of column families that cannot be deleted from the hbase:meta table. 067 * They are critical to cluster operation. This is a bit of an odd place to 068 * keep this list but then this is the tooling that does add/remove. Keeping 069 * it local! 070 */ 071 private static final List<byte []> UNDELETABLE_META_COLUMNFAMILIES = 072 Collections.unmodifiableList(Arrays.asList( 073 HConstants.CATALOG_FAMILY, HConstants.TABLE_FAMILY, HConstants.REPLICATION_BARRIER_FAMILY)); 074 075 public ModifyTableProcedure() { 076 super(); 077 initialize(null, false); 078 } 079 080 public ModifyTableProcedure(final MasterProcedureEnv env, final TableDescriptor htd) 081 throws HBaseIOException { 082 this(env, htd, null); 083 } 084 085 public ModifyTableProcedure(final MasterProcedureEnv env, final TableDescriptor htd, 086 final ProcedurePrepareLatch latch) 087 throws HBaseIOException { 088 this(env, htd, latch, null, false); 089 } 090 091 public ModifyTableProcedure(final MasterProcedureEnv env, 092 final TableDescriptor newTableDescriptor, final ProcedurePrepareLatch latch, 093 final TableDescriptor oldTableDescriptor, final boolean shouldCheckDescriptor) 094 throws HBaseIOException { 095 super(env, latch); 096 initialize(oldTableDescriptor, shouldCheckDescriptor); 097 this.modifiedTableDescriptor = newTableDescriptor; 098 preflightChecks(env, null/*No table checks; if changing peers, table can be online*/); 099 } 100 101 @Override 102 protected void preflightChecks(MasterProcedureEnv env, Boolean enabled) throws HBaseIOException { 103 super.preflightChecks(env, enabled); 104 if (this.modifiedTableDescriptor.isMetaTable()) { 105 // If we are modifying the hbase:meta table, make sure we are not deleting critical 106 // column families else we'll damage the cluster. 107 Set<byte []> cfs = this.modifiedTableDescriptor.getColumnFamilyNames(); 108 for (byte[] family : UNDELETABLE_META_COLUMNFAMILIES) { 109 if (!cfs.contains(family)) { 110 throw new HBaseIOException("Delete of hbase:meta column family " + 111 Bytes.toString(family)); 112 } 113 } 114 } 115 } 116 117 private void initialize(final TableDescriptor unmodifiedTableDescriptor, 118 final boolean shouldCheckDescriptor) { 119 this.unmodifiedTableDescriptor = unmodifiedTableDescriptor; 120 this.shouldCheckDescriptor = shouldCheckDescriptor; 121 this.deleteColumnFamilyInModify = false; 122 } 123 124 @Override 125 protected Flow executeFromState(final MasterProcedureEnv env, final ModifyTableState state) 126 throws InterruptedException { 127 LOG.trace("{} execute state={}", this, state); 128 try { 129 switch (state) { 130 case MODIFY_TABLE_PREPARE: 131 prepareModify(env); 132 setNextState(ModifyTableState.MODIFY_TABLE_PRE_OPERATION); 133 break; 134 case MODIFY_TABLE_PRE_OPERATION: 135 preModify(env, state); 136 setNextState(ModifyTableState.MODIFY_TABLE_UPDATE_TABLE_DESCRIPTOR); 137 break; 138 case MODIFY_TABLE_UPDATE_TABLE_DESCRIPTOR: 139 updateTableDescriptor(env); 140 setNextState(ModifyTableState.MODIFY_TABLE_REMOVE_REPLICA_COLUMN); 141 break; 142 case MODIFY_TABLE_REMOVE_REPLICA_COLUMN: 143 updateReplicaColumnsIfNeeded(env, unmodifiedTableDescriptor, modifiedTableDescriptor); 144 setNextState(ModifyTableState.MODIFY_TABLE_POST_OPERATION); 145 break; 146 case MODIFY_TABLE_POST_OPERATION: 147 postModify(env, state); 148 setNextState(ModifyTableState.MODIFY_TABLE_REOPEN_ALL_REGIONS); 149 break; 150 case MODIFY_TABLE_REOPEN_ALL_REGIONS: 151 if (env.getAssignmentManager().isTableEnabled(getTableName())) { 152 addChildProcedure(new ReopenTableRegionsProcedure(getTableName())); 153 } 154 if (deleteColumnFamilyInModify) { 155 setNextState(ModifyTableState.MODIFY_TABLE_DELETE_FS_LAYOUT); 156 } else { 157 return Flow.NO_MORE_STATE; 158 } 159 break; 160 case MODIFY_TABLE_DELETE_FS_LAYOUT: 161 deleteFromFs(env, unmodifiedTableDescriptor, modifiedTableDescriptor); 162 return Flow.NO_MORE_STATE; 163 default: 164 throw new UnsupportedOperationException("unhandled state=" + state); 165 } 166 } catch (IOException e) { 167 if (isRollbackSupported(state)) { 168 setFailure("master-modify-table", e); 169 } else { 170 LOG.warn("Retriable error trying to modify table={} (in state={})", getTableName(), state, 171 e); 172 } 173 } 174 return Flow.HAS_MORE_STATE; 175 } 176 177 @Override 178 protected void rollbackState(final MasterProcedureEnv env, final ModifyTableState state) 179 throws IOException { 180 if (state == ModifyTableState.MODIFY_TABLE_PREPARE || 181 state == ModifyTableState.MODIFY_TABLE_PRE_OPERATION) { 182 // nothing to rollback, pre-modify is just checks. 183 // TODO: coprocessor rollback semantic is still undefined. 184 return; 185 } 186 187 // The delete doesn't have a rollback. The execution will succeed, at some point. 188 throw new UnsupportedOperationException("unhandled state=" + state); 189 } 190 191 @Override 192 protected boolean isRollbackSupported(final ModifyTableState state) { 193 switch (state) { 194 case MODIFY_TABLE_PRE_OPERATION: 195 case MODIFY_TABLE_PREPARE: 196 return true; 197 default: 198 return false; 199 } 200 } 201 202 @Override 203 protected void completionCleanup(final MasterProcedureEnv env) { 204 releaseSyncLatch(); 205 } 206 207 @Override 208 protected ModifyTableState getState(final int stateId) { 209 return ModifyTableState.forNumber(stateId); 210 } 211 212 @Override 213 protected int getStateId(final ModifyTableState state) { 214 return state.getNumber(); 215 } 216 217 @Override 218 protected ModifyTableState getInitialState() { 219 return ModifyTableState.MODIFY_TABLE_PREPARE; 220 } 221 222 @Override 223 protected void serializeStateData(ProcedureStateSerializer serializer) 224 throws IOException { 225 super.serializeStateData(serializer); 226 227 MasterProcedureProtos.ModifyTableStateData.Builder modifyTableMsg = 228 MasterProcedureProtos.ModifyTableStateData.newBuilder() 229 .setUserInfo(MasterProcedureUtil.toProtoUserInfo(getUser())) 230 .setModifiedTableSchema(ProtobufUtil.toTableSchema(modifiedTableDescriptor)) 231 .setDeleteColumnFamilyInModify(deleteColumnFamilyInModify) 232 .setShouldCheckDescriptor(shouldCheckDescriptor); 233 234 if (unmodifiedTableDescriptor != null) { 235 modifyTableMsg 236 .setUnmodifiedTableSchema(ProtobufUtil.toTableSchema(unmodifiedTableDescriptor)); 237 } 238 239 serializer.serialize(modifyTableMsg.build()); 240 } 241 242 @Override 243 protected void deserializeStateData(ProcedureStateSerializer serializer) 244 throws IOException { 245 super.deserializeStateData(serializer); 246 247 MasterProcedureProtos.ModifyTableStateData modifyTableMsg = 248 serializer.deserialize(MasterProcedureProtos.ModifyTableStateData.class); 249 setUser(MasterProcedureUtil.toUserInfo(modifyTableMsg.getUserInfo())); 250 modifiedTableDescriptor = ProtobufUtil.toTableDescriptor(modifyTableMsg.getModifiedTableSchema()); 251 deleteColumnFamilyInModify = modifyTableMsg.getDeleteColumnFamilyInModify(); 252 shouldCheckDescriptor = modifyTableMsg.hasShouldCheckDescriptor() 253 ? modifyTableMsg.getShouldCheckDescriptor() : false; 254 255 if (modifyTableMsg.hasUnmodifiedTableSchema()) { 256 unmodifiedTableDescriptor = 257 ProtobufUtil.toTableDescriptor(modifyTableMsg.getUnmodifiedTableSchema()); 258 } 259 } 260 261 @Override 262 public TableName getTableName() { 263 return modifiedTableDescriptor.getTableName(); 264 } 265 266 @Override 267 public TableOperationType getTableOperationType() { 268 return TableOperationType.EDIT; 269 } 270 271 /** 272 * Check conditions before any real action of modifying a table. 273 * @param env MasterProcedureEnv 274 * @throws IOException 275 */ 276 private void prepareModify(final MasterProcedureEnv env) throws IOException { 277 // Checks whether the table exists 278 if (!MetaTableAccessor.tableExists(env.getMasterServices().getConnection(), getTableName())) { 279 throw new TableNotFoundException(getTableName()); 280 } 281 282 // check that we have at least 1 CF 283 if (modifiedTableDescriptor.getColumnFamilyCount() == 0) { 284 throw new DoNotRetryIOException("Table " + getTableName().toString() + 285 " should have at least one column family."); 286 } 287 288 // If descriptor check is enabled, check whether the table descriptor when procedure was 289 // submitted matches with the current 290 // table descriptor of the table, else retrieve the old descriptor 291 // for comparison in order to update the descriptor. 292 if (shouldCheckDescriptor) { 293 if (TableDescriptor.COMPARATOR.compare(unmodifiedTableDescriptor, 294 env.getMasterServices().getTableDescriptors().get(getTableName())) != 0) { 295 LOG.error("Error while modifying table '" + getTableName().toString() 296 + "' Skipping procedure : " + this); 297 throw new ConcurrentTableModificationException( 298 "Skipping modify table operation on table '" + getTableName().toString() 299 + "' as it has already been modified by some other concurrent operation, " 300 + "Please retry."); 301 } 302 } else { 303 this.unmodifiedTableDescriptor = 304 env.getMasterServices().getTableDescriptors().get(getTableName()); 305 } 306 307 if (env.getMasterServices().getTableStateManager() 308 .isTableState(getTableName(), TableState.State.ENABLED)) { 309 if (modifiedTableDescriptor.getRegionReplication() != unmodifiedTableDescriptor 310 .getRegionReplication()) { 311 throw new TableNotDisabledException( 312 "REGION_REPLICATION change is not supported for enabled tables"); 313 } 314 } 315 this.deleteColumnFamilyInModify = isDeleteColumnFamily(unmodifiedTableDescriptor, 316 modifiedTableDescriptor); 317 } 318 319 /** 320 * Find out whether all column families in unmodifiedTableDescriptor also exists in 321 * the modifiedTableDescriptor. 322 * @return True if we are deleting a column family. 323 */ 324 private static boolean isDeleteColumnFamily(TableDescriptor originalDescriptor, 325 TableDescriptor newDescriptor) { 326 boolean result = false; 327 final Set<byte[]> originalFamilies = originalDescriptor.getColumnFamilyNames(); 328 final Set<byte[]> newFamilies = newDescriptor.getColumnFamilyNames(); 329 for (byte[] familyName : originalFamilies) { 330 if (!newFamilies.contains(familyName)) { 331 result = true; 332 break; 333 } 334 } 335 return result; 336 } 337 338 /** 339 * Action before modifying table. 340 * @param env MasterProcedureEnv 341 * @param state the procedure state 342 * @throws IOException 343 * @throws InterruptedException 344 */ 345 private void preModify(final MasterProcedureEnv env, final ModifyTableState state) 346 throws IOException, InterruptedException { 347 runCoprocessorAction(env, state); 348 } 349 350 /** 351 * Update descriptor 352 * @param env MasterProcedureEnv 353 * @throws IOException 354 **/ 355 private void updateTableDescriptor(final MasterProcedureEnv env) throws IOException { 356 env.getMasterServices().getTableDescriptors().update(modifiedTableDescriptor); 357 } 358 359 /** 360 * Removes from hdfs the families that are not longer present in the new table descriptor. 361 * @param env MasterProcedureEnv 362 * @throws IOException 363 */ 364 private void deleteFromFs(final MasterProcedureEnv env, 365 final TableDescriptor oldTableDescriptor, final TableDescriptor newTableDescriptor) 366 throws IOException { 367 final Set<byte[]> oldFamilies = oldTableDescriptor.getColumnFamilyNames(); 368 final Set<byte[]> newFamilies = newTableDescriptor.getColumnFamilyNames(); 369 for (byte[] familyName : oldFamilies) { 370 if (!newFamilies.contains(familyName)) { 371 MasterDDLOperationHelper.deleteColumnFamilyFromFileSystem( 372 env, 373 getTableName(), 374 getRegionInfoList(env), 375 familyName, oldTableDescriptor.getColumnFamily(familyName).isMobEnabled()); 376 } 377 } 378 } 379 380 /** 381 * update replica column families if necessary. 382 * @param env MasterProcedureEnv 383 * @throws IOException 384 */ 385 private void updateReplicaColumnsIfNeeded( 386 final MasterProcedureEnv env, 387 final TableDescriptor oldTableDescriptor, 388 final TableDescriptor newTableDescriptor) throws IOException { 389 final int oldReplicaCount = oldTableDescriptor.getRegionReplication(); 390 final int newReplicaCount = newTableDescriptor.getRegionReplication(); 391 392 if (newReplicaCount < oldReplicaCount) { 393 Set<byte[]> tableRows = new HashSet<>(); 394 Connection connection = env.getMasterServices().getConnection(); 395 Scan scan = MetaTableAccessor.getScanForTableName(connection, getTableName()); 396 scan.addColumn(HConstants.CATALOG_FAMILY, HConstants.REGIONINFO_QUALIFIER); 397 398 try (Table metaTable = connection.getTable(TableName.META_TABLE_NAME)) { 399 ResultScanner resScanner = metaTable.getScanner(scan); 400 for (Result result : resScanner) { 401 tableRows.add(result.getRow()); 402 } 403 MetaTableAccessor.removeRegionReplicasFromMeta( 404 tableRows, 405 newReplicaCount, 406 oldReplicaCount - newReplicaCount, 407 connection); 408 } 409 } 410 if (newReplicaCount > oldReplicaCount) { 411 Connection connection = env.getMasterServices().getConnection(); 412 // Get the existing table regions 413 List<RegionInfo> existingTableRegions = 414 MetaTableAccessor.getTableRegions(connection, getTableName()); 415 // add all the new entries to the meta table 416 addRegionsToMeta(env, newTableDescriptor, existingTableRegions); 417 if (oldReplicaCount <= 1) { 418 // The table has been newly enabled for replica. So check if we need to setup 419 // region replication 420 ServerRegionReplicaUtil.setupRegionReplicaReplication(env.getMasterConfiguration()); 421 } 422 } 423 } 424 425 private static void addRegionsToMeta(final MasterProcedureEnv env, 426 final TableDescriptor tableDescriptor, final List<RegionInfo> regionInfos) 427 throws IOException { 428 MetaTableAccessor.addRegionsToMeta(env.getMasterServices().getConnection(), regionInfos, 429 tableDescriptor.getRegionReplication()); 430 } 431 /** 432 * Action after modifying table. 433 * @param env MasterProcedureEnv 434 * @param state the procedure state 435 * @throws IOException 436 * @throws InterruptedException 437 */ 438 private void postModify(final MasterProcedureEnv env, final ModifyTableState state) 439 throws IOException, InterruptedException { 440 runCoprocessorAction(env, state); 441 } 442 443 /** 444 * Coprocessor Action. 445 * @param env MasterProcedureEnv 446 * @param state the procedure state 447 * @throws IOException 448 * @throws InterruptedException 449 */ 450 private void runCoprocessorAction(final MasterProcedureEnv env, final ModifyTableState state) 451 throws IOException, InterruptedException { 452 final MasterCoprocessorHost cpHost = env.getMasterCoprocessorHost(); 453 if (cpHost != null) { 454 switch (state) { 455 case MODIFY_TABLE_PRE_OPERATION: 456 cpHost.preModifyTableAction(getTableName(), unmodifiedTableDescriptor, 457 modifiedTableDescriptor, getUser()); 458 break; 459 case MODIFY_TABLE_POST_OPERATION: 460 cpHost.postCompletedModifyTableAction(getTableName(), unmodifiedTableDescriptor, 461 modifiedTableDescriptor,getUser()); 462 break; 463 default: 464 throw new UnsupportedOperationException(this + " unhandled state=" + state); 465 } 466 } 467 } 468 469 /** 470 * Fetches all Regions for a table. Cache the result of this method if you need to use it multiple 471 * times. Be aware that it may change over in between calls to this procedure. 472 */ 473 private List<RegionInfo> getRegionInfoList(final MasterProcedureEnv env) throws IOException { 474 return env.getAssignmentManager().getRegionStates().getRegionsOfTable(getTableName()); 475 } 476}