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.assignment; 019 020import java.io.IOException; 021import org.apache.hadoop.fs.FileSystem; 022import org.apache.hadoop.fs.Path; 023import org.apache.hadoop.hbase.MetaTableAccessor; 024import org.apache.hadoop.hbase.backup.HFileArchiver; 025import org.apache.hadoop.hbase.client.RegionInfo; 026import org.apache.hadoop.hbase.favored.FavoredNodesManager; 027import org.apache.hadoop.hbase.master.MasterFileSystem; 028import org.apache.hadoop.hbase.master.MasterServices; 029import org.apache.hadoop.hbase.master.procedure.AbstractStateMachineRegionProcedure; 030import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv; 031import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer; 032import org.apache.hadoop.hbase.procedure2.ProcedureSuspendedException; 033import org.apache.hadoop.hbase.procedure2.ProcedureYieldException; 034import org.apache.hadoop.hbase.util.CommonFSUtils; 035import org.apache.yetus.audience.InterfaceAudience; 036import org.slf4j.Logger; 037import org.slf4j.LoggerFactory; 038 039import org.apache.hbase.thirdparty.com.google.common.collect.Lists; 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.GCRegionState; 044 045/** 046 * GC a Region that is no longer in use. It has been split or merged away. Caller determines if it 047 * is GC time. This Procedure does not check. 048 * <p> 049 * This is a Region StateMachine Procedure. We take a read lock on the Table and then exclusive on 050 * the Region. 051 */ 052@InterfaceAudience.Private 053public class GCRegionProcedure extends AbstractStateMachineRegionProcedure<GCRegionState> { 054 private static final Logger LOG = LoggerFactory.getLogger(GCRegionProcedure.class); 055 056 public GCRegionProcedure(final MasterProcedureEnv env, final RegionInfo hri) { 057 super(env, hri); 058 } 059 060 public GCRegionProcedure() { 061 // Required by the Procedure framework to create the procedure on replay 062 super(); 063 } 064 065 @Override 066 public TableOperationType getTableOperationType() { 067 return TableOperationType.REGION_GC; 068 } 069 070 @Override 071 protected Flow executeFromState(MasterProcedureEnv env, GCRegionState state) 072 throws ProcedureSuspendedException, ProcedureYieldException, InterruptedException { 073 if (LOG.isTraceEnabled()) { 074 LOG.trace(this + " execute state=" + state); 075 } 076 MasterServices masterServices = env.getMasterServices(); 077 try { 078 switch (state) { 079 case GC_REGION_PREPARE: 080 // Nothing to do to prepare. 081 setNextState(GCRegionState.GC_REGION_ARCHIVE); 082 break; 083 case GC_REGION_ARCHIVE: 084 MasterFileSystem mfs = masterServices.getMasterFileSystem(); 085 FileSystem fs = mfs.getFileSystem(); 086 if (HFileArchiver.exists(masterServices.getConfiguration(), fs, getRegion())) { 087 if (LOG.isDebugEnabled()) { 088 LOG.debug("Archiving region=" + getRegion().getShortNameToLog()); 089 } 090 HFileArchiver.archiveRegion(masterServices.getConfiguration(), fs, getRegion()); 091 } 092 FileSystem walFs = mfs.getWALFileSystem(); 093 // Cleanup the directories on WAL filesystem also 094 Path regionWALDir = CommonFSUtils.getWALRegionDir(env.getMasterConfiguration(), 095 getRegion().getTable(), getRegion().getEncodedName()); 096 if (walFs.exists(regionWALDir)) { 097 if (!walFs.delete(regionWALDir, true)) { 098 LOG.debug("Failed to delete {}", regionWALDir); 099 } 100 } 101 Path wrongRegionWALDir = CommonFSUtils.getWrongWALRegionDir(env.getMasterConfiguration(), 102 getRegion().getTable(), getRegion().getEncodedName()); 103 if (walFs.exists(wrongRegionWALDir)) { 104 if (!walFs.delete(wrongRegionWALDir, true)) { 105 LOG.debug("Failed to delete {}", regionWALDir); 106 } 107 } 108 setNextState(GCRegionState.GC_REGION_PURGE_METADATA); 109 break; 110 case GC_REGION_PURGE_METADATA: 111 // TODO: Purge metadata before removing from HDFS? This ordering is copied 112 // from CatalogJanitor. 113 AssignmentManager am = masterServices.getAssignmentManager(); 114 if (am != null) { 115 if (am.getRegionStates() != null) { 116 am.getRegionStates().deleteRegion(getRegion()); 117 } 118 } 119 MetaTableAccessor.deleteRegionInfo(masterServices.getConnection(), getRegion()); 120 masterServices.getServerManager().removeRegion(getRegion()); 121 FavoredNodesManager fnm = masterServices.getFavoredNodesManager(); 122 if (fnm != null) { 123 fnm.deleteFavoredNodesForRegions(Lists.newArrayList(getRegion())); 124 } 125 return Flow.NO_MORE_STATE; 126 default: 127 throw new UnsupportedOperationException(this + " unhandled state=" + state); 128 } 129 } catch (IOException ioe) { 130 // TODO: This is going to spew log? Add retry backoff 131 LOG.warn("Error trying to GC " + getRegion().getShortNameToLog() + "; retrying...", ioe); 132 } 133 return Flow.HAS_MORE_STATE; 134 } 135 136 @Override 137 protected void rollbackState(MasterProcedureEnv env, GCRegionState state) 138 throws IOException, InterruptedException { 139 // no-op 140 } 141 142 @Override 143 protected GCRegionState getState(int stateId) { 144 return GCRegionState.forNumber(stateId); 145 } 146 147 @Override 148 protected int getStateId(GCRegionState state) { 149 return state.getNumber(); 150 } 151 152 @Override 153 protected GCRegionState getInitialState() { 154 return GCRegionState.GC_REGION_PREPARE; 155 } 156 157 @Override 158 protected void serializeStateData(ProcedureStateSerializer serializer) throws IOException { 159 super.serializeStateData(serializer); 160 // Double serialization of regionname. Superclass is also serializing. Fix. 161 final MasterProcedureProtos.GCRegionStateData.Builder msg = 162 MasterProcedureProtos.GCRegionStateData.newBuilder() 163 .setRegionInfo(ProtobufUtil.toRegionInfo(getRegion())); 164 serializer.serialize(msg.build()); 165 } 166 167 @Override 168 protected void deserializeStateData(ProcedureStateSerializer serializer) throws IOException { 169 super.deserializeStateData(serializer); 170 final MasterProcedureProtos.GCRegionStateData msg = 171 serializer.deserialize(MasterProcedureProtos.GCRegionStateData.class); 172 setRegion(ProtobufUtil.toRegionInfo(msg.getRegionInfo())); 173 } 174}