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.Arrays; 024import java.util.List; 025 026import org.apache.hadoop.hbase.HBaseIOException; 027import org.apache.hadoop.hbase.TableName; 028import org.apache.hadoop.hbase.TableNotDisabledException; 029import org.apache.hadoop.hbase.TableNotFoundException; 030import org.apache.hadoop.hbase.client.RegionInfo; 031import org.apache.hadoop.hbase.client.RegionInfoBuilder; 032import org.apache.hadoop.hbase.client.RegionReplicaUtil; 033import org.apache.hadoop.hbase.client.TableDescriptor; 034import org.apache.hadoop.hbase.master.MasterCoprocessorHost; 035import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer; 036import org.apache.hadoop.hbase.util.ModifyRegionUtils; 037import org.apache.yetus.audience.InterfaceAudience; 038import org.slf4j.Logger; 039import org.slf4j.LoggerFactory; 040import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting; 041import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil; 042import org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos; 043import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos; 044import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.TruncateTableState; 045 046@InterfaceAudience.Private 047public class TruncateTableProcedure 048 extends AbstractStateMachineTableProcedure<TruncateTableState> { 049 private static final Logger LOG = LoggerFactory.getLogger(TruncateTableProcedure.class); 050 051 private boolean preserveSplits; 052 private List<RegionInfo> regions; 053 private TableDescriptor tableDescriptor; 054 private TableName tableName; 055 056 public TruncateTableProcedure() { 057 // Required by the Procedure framework to create the procedure on replay 058 super(); 059 } 060 061 public TruncateTableProcedure(final MasterProcedureEnv env, final TableName tableName, 062 boolean preserveSplits) 063 throws HBaseIOException { 064 this(env, tableName, preserveSplits, null); 065 } 066 067 public TruncateTableProcedure(final MasterProcedureEnv env, final TableName tableName, 068 boolean preserveSplits, ProcedurePrepareLatch latch) 069 throws HBaseIOException { 070 super(env, latch); 071 this.tableName = tableName; 072 preflightChecks(env, false); 073 this.preserveSplits = preserveSplits; 074 } 075 076 @Override 077 protected Flow executeFromState(final MasterProcedureEnv env, TruncateTableState state) 078 throws InterruptedException { 079 if (LOG.isTraceEnabled()) { 080 LOG.trace(this + " execute state=" + state); 081 } 082 try { 083 switch (state) { 084 case TRUNCATE_TABLE_PRE_OPERATION: 085 // Verify if we can truncate the table 086 if (!prepareTruncate(env)) { 087 assert isFailed() : "the truncate should have an exception here"; 088 return Flow.NO_MORE_STATE; 089 } 090 091 // TODO: Move out... in the acquireLock() 092 LOG.debug("waiting for '" + getTableName() + "' regions in transition"); 093 regions = env.getAssignmentManager().getRegionStates().getRegionsOfTable(getTableName()); 094 RegionReplicaUtil.removeNonDefaultRegions(regions); 095 assert regions != null && !regions.isEmpty() : "unexpected 0 regions"; 096 ProcedureSyncWait.waitRegionInTransition(env, regions); 097 098 // Call coprocessors 099 preTruncate(env); 100 101 setNextState(TruncateTableState.TRUNCATE_TABLE_REMOVE_FROM_META); 102 break; 103 case TRUNCATE_TABLE_REMOVE_FROM_META: 104 tableDescriptor = env.getMasterServices().getTableDescriptors() 105 .get(tableName); 106 DeleteTableProcedure.deleteFromMeta(env, getTableName(), regions); 107 DeleteTableProcedure.deleteAssignmentState(env, getTableName()); 108 setNextState(TruncateTableState.TRUNCATE_TABLE_CLEAR_FS_LAYOUT); 109 break; 110 case TRUNCATE_TABLE_CLEAR_FS_LAYOUT: 111 DeleteTableProcedure.deleteFromFs(env, getTableName(), regions, true); 112 // NOTE: It's very important that we create new HRegions before next state, so that 113 // they get persisted in procedure state before we start using them for anything. 114 // Otherwise, if we create them in next step and master crashes after creating fs 115 // layout but before saving state, region re-created after recovery will have different 116 // regionId(s) and encoded names. That will lead to unwanted regions in FS layout 117 // (which were created before the crash). 118 if (!preserveSplits) { 119 // if we are not preserving splits, generate a new single region 120 regions = Arrays.asList(ModifyRegionUtils.createRegionInfos(tableDescriptor, null)); 121 } else { 122 regions = recreateRegionInfo(regions); 123 } 124 setNextState(TruncateTableState.TRUNCATE_TABLE_CREATE_FS_LAYOUT); 125 break; 126 case TRUNCATE_TABLE_CREATE_FS_LAYOUT: 127 DeleteTableProcedure.deleteFromFs(env, getTableName(), regions, true); 128 regions = CreateTableProcedure.createFsLayout(env, tableDescriptor, regions); 129 CreateTableProcedure.updateTableDescCache(env, getTableName()); 130 setNextState(TruncateTableState.TRUNCATE_TABLE_ADD_TO_META); 131 break; 132 case TRUNCATE_TABLE_ADD_TO_META: 133 regions = CreateTableProcedure.addTableToMeta(env, tableDescriptor, regions); 134 setNextState(TruncateTableState.TRUNCATE_TABLE_ASSIGN_REGIONS); 135 break; 136 case TRUNCATE_TABLE_ASSIGN_REGIONS: 137 CreateTableProcedure.setEnablingState(env, getTableName()); 138 addChildProcedure(env.getAssignmentManager().createRoundRobinAssignProcedures(regions)); 139 setNextState(TruncateTableState.TRUNCATE_TABLE_POST_OPERATION); 140 tableDescriptor = null; 141 regions = null; 142 break; 143 case TRUNCATE_TABLE_POST_OPERATION: 144 CreateTableProcedure.setEnabledState(env, getTableName()); 145 postTruncate(env); 146 LOG.debug("truncate '" + getTableName() + "' completed"); 147 return Flow.NO_MORE_STATE; 148 default: 149 throw new UnsupportedOperationException("unhandled state=" + state); 150 } 151 } catch (IOException e) { 152 if (isRollbackSupported(state)) { 153 setFailure("master-truncate-table", e); 154 } else { 155 LOG.warn("Retriable error trying to truncate table=" + getTableName() 156 + " state=" + state, e); 157 } 158 } 159 return Flow.HAS_MORE_STATE; 160 } 161 162 @Override 163 protected void rollbackState(final MasterProcedureEnv env, final TruncateTableState state) { 164 if (state == TruncateTableState.TRUNCATE_TABLE_PRE_OPERATION) { 165 // nothing to rollback, pre-truncate is just table-state checks. 166 // We can fail if the table does not exist or is not disabled. 167 // TODO: coprocessor rollback semantic is still undefined. 168 return; 169 } 170 171 // The truncate doesn't have a rollback. The execution will succeed, at some point. 172 throw new UnsupportedOperationException("unhandled state=" + state); 173 } 174 175 @Override 176 protected void completionCleanup(final MasterProcedureEnv env) { 177 releaseSyncLatch(); 178 } 179 180 @Override 181 protected boolean isRollbackSupported(final TruncateTableState state) { 182 switch (state) { 183 case TRUNCATE_TABLE_PRE_OPERATION: 184 return true; 185 default: 186 return false; 187 } 188 } 189 190 @Override 191 protected TruncateTableState getState(final int stateId) { 192 return TruncateTableState.valueOf(stateId); 193 } 194 195 @Override 196 protected int getStateId(final TruncateTableState state) { 197 return state.getNumber(); 198 } 199 200 @Override 201 protected TruncateTableState getInitialState() { 202 return TruncateTableState.TRUNCATE_TABLE_PRE_OPERATION; 203 } 204 205 @Override 206 public TableName getTableName() { 207 return tableName; 208 } 209 210 @Override 211 public TableOperationType getTableOperationType() { 212 return TableOperationType.EDIT; 213 } 214 215 @Override 216 public void toStringClassDetails(StringBuilder sb) { 217 sb.append(getClass().getSimpleName()); 218 sb.append(" (table="); 219 sb.append(getTableName()); 220 sb.append(" preserveSplits="); 221 sb.append(preserveSplits); 222 sb.append(")"); 223 } 224 225 @Override 226 protected void serializeStateData(ProcedureStateSerializer serializer) 227 throws IOException { 228 super.serializeStateData(serializer); 229 230 MasterProcedureProtos.TruncateTableStateData.Builder state = 231 MasterProcedureProtos.TruncateTableStateData.newBuilder() 232 .setUserInfo(MasterProcedureUtil.toProtoUserInfo(getUser())) 233 .setPreserveSplits(preserveSplits); 234 if (tableDescriptor != null) { 235 state.setTableSchema(ProtobufUtil.toTableSchema(tableDescriptor)); 236 } else { 237 state.setTableName(ProtobufUtil.toProtoTableName(tableName)); 238 } 239 if (regions != null) { 240 for (RegionInfo hri: regions) { 241 state.addRegionInfo(ProtobufUtil.toRegionInfo(hri)); 242 } 243 } 244 serializer.serialize(state.build()); 245 } 246 247 @Override 248 protected void deserializeStateData(ProcedureStateSerializer serializer) 249 throws IOException { 250 super.deserializeStateData(serializer); 251 252 MasterProcedureProtos.TruncateTableStateData state = 253 serializer.deserialize(MasterProcedureProtos.TruncateTableStateData.class); 254 setUser(MasterProcedureUtil.toUserInfo(state.getUserInfo())); 255 if (state.hasTableSchema()) { 256 tableDescriptor = ProtobufUtil.toTableDescriptor(state.getTableSchema()); 257 tableName = tableDescriptor.getTableName(); 258 } else { 259 tableName = ProtobufUtil.toTableName(state.getTableName()); 260 } 261 preserveSplits = state.getPreserveSplits(); 262 if (state.getRegionInfoCount() == 0) { 263 regions = null; 264 } else { 265 regions = new ArrayList<>(state.getRegionInfoCount()); 266 for (HBaseProtos.RegionInfo hri: state.getRegionInfoList()) { 267 regions.add(ProtobufUtil.toRegionInfo(hri)); 268 } 269 } 270 } 271 272 private static List<RegionInfo> recreateRegionInfo(final List<RegionInfo> regions) { 273 ArrayList<RegionInfo> newRegions = new ArrayList<>(regions.size()); 274 for (RegionInfo hri: regions) { 275 newRegions.add(RegionInfoBuilder.newBuilder(hri.getTable()) 276 .setStartKey(hri.getStartKey()) 277 .setEndKey(hri.getEndKey()) 278 .build()); 279 } 280 return newRegions; 281 } 282 283 private boolean prepareTruncate(final MasterProcedureEnv env) throws IOException { 284 try { 285 env.getMasterServices().checkTableModifiable(getTableName()); 286 } catch (TableNotFoundException|TableNotDisabledException e) { 287 setFailure("master-truncate-table", e); 288 return false; 289 } 290 return true; 291 } 292 293 private boolean preTruncate(final MasterProcedureEnv env) 294 throws IOException, InterruptedException { 295 final MasterCoprocessorHost cpHost = env.getMasterCoprocessorHost(); 296 if (cpHost != null) { 297 final TableName tableName = getTableName(); 298 cpHost.preTruncateTableAction(tableName, getUser()); 299 } 300 return true; 301 } 302 303 private void postTruncate(final MasterProcedureEnv env) 304 throws IOException, InterruptedException { 305 final MasterCoprocessorHost cpHost = env.getMasterCoprocessorHost(); 306 if (cpHost != null) { 307 final TableName tableName = getTableName(); 308 cpHost.postCompletedTruncateTableAction(tableName, getUser()); 309 } 310 } 311 312 @VisibleForTesting 313 RegionInfo getFirstRegionInfo() { 314 if (regions == null || regions.isEmpty()) { 315 return null; 316 } 317 return regions.get(0); 318 } 319}