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 org.apache.hadoop.fs.Path; 022import org.apache.hadoop.hbase.HBaseIOException; 023import org.apache.hadoop.hbase.MetaTableAccessor; 024import org.apache.hadoop.hbase.TableName; 025import org.apache.hadoop.hbase.TableNotDisabledException; 026import org.apache.hadoop.hbase.TableNotEnabledException; 027import org.apache.hadoop.hbase.TableNotFoundException; 028import org.apache.hadoop.hbase.UnknownRegionException; 029import org.apache.hadoop.hbase.client.DoNotRetryRegionException; 030import org.apache.hadoop.hbase.client.RegionInfo; 031import org.apache.hadoop.hbase.client.RegionOfflineException; 032import org.apache.hadoop.hbase.client.TableState; 033import org.apache.hadoop.hbase.master.MasterFileSystem; 034import org.apache.hadoop.hbase.master.MasterServices; 035import org.apache.hadoop.hbase.master.RegionState; 036import org.apache.hadoop.hbase.master.TableStateManager; 037import org.apache.hadoop.hbase.master.assignment.RegionStates; 038import org.apache.hadoop.hbase.procedure2.StateMachineProcedure; 039import org.apache.hadoop.hbase.security.User; 040import org.apache.hadoop.hbase.util.FSUtils; 041import org.apache.hadoop.hbase.util.ServerRegionReplicaUtil; 042import org.apache.yetus.audience.InterfaceAudience; 043 044/** 045 * Base class for all the Table procedures that want to use a StateMachineProcedure. 046 * It provides helpers like basic locking, sync latch, and toStringClassDetails(). 047 */ 048@InterfaceAudience.Private 049public abstract class AbstractStateMachineTableProcedure<TState> 050 extends StateMachineProcedure<MasterProcedureEnv, TState> 051 implements TableProcedureInterface { 052 053 // used for compatibility with old clients 054 private final ProcedurePrepareLatch syncLatch; 055 056 private User user; 057 058 protected AbstractStateMachineTableProcedure() { 059 // Required by the Procedure framework to create the procedure on replay 060 syncLatch = null; 061 } 062 063 protected AbstractStateMachineTableProcedure(final MasterProcedureEnv env) { 064 this(env, null); 065 } 066 067 /** 068 * @param env Uses this to set Procedure Owner at least. 069 */ 070 protected AbstractStateMachineTableProcedure(final MasterProcedureEnv env, 071 final ProcedurePrepareLatch latch) { 072 if (env != null) { 073 this.user = env.getRequestUser(); 074 this.setOwner(user); 075 } 076 // used for compatibility with clients without procedures 077 // they need a sync TableExistsException, TableNotFoundException, TableNotDisabledException, ... 078 this.syncLatch = latch; 079 } 080 081 @Override 082 public abstract TableName getTableName(); 083 084 @Override 085 public abstract TableOperationType getTableOperationType(); 086 087 @Override 088 public void toStringClassDetails(final StringBuilder sb) { 089 sb.append(getClass().getSimpleName()); 090 sb.append(" table="); 091 sb.append(getTableName()); 092 } 093 094 @Override 095 protected boolean waitInitialized(MasterProcedureEnv env) { 096 return env.waitInitialized(this); 097 } 098 099 @Override 100 protected LockState acquireLock(final MasterProcedureEnv env) { 101 if (env.getProcedureScheduler().waitTableExclusiveLock(this, getTableName())) { 102 return LockState.LOCK_EVENT_WAIT; 103 } 104 return LockState.LOCK_ACQUIRED; 105 } 106 107 @Override 108 protected void releaseLock(final MasterProcedureEnv env) { 109 env.getProcedureScheduler().wakeTableExclusiveLock(this, getTableName()); 110 } 111 112 protected User getUser() { 113 return user; 114 } 115 116 protected void setUser(final User user) { 117 this.user = user; 118 } 119 120 protected void releaseSyncLatch() { 121 ProcedurePrepareLatch.releaseLatch(syncLatch, this); 122 } 123 124 /** 125 * Check whether a table is modifiable - exists and either offline or online with config set 126 * @param env MasterProcedureEnv 127 * @throws IOException 128 */ 129 protected void checkTableModifiable(final MasterProcedureEnv env) throws IOException { 130 // Checks whether the table exists 131 if (!MetaTableAccessor.tableExists(env.getMasterServices().getConnection(), getTableName())) { 132 throw new TableNotFoundException(getTableName()); 133 } 134 } 135 136 protected final Path getRegionDir(MasterProcedureEnv env, RegionInfo region) throws IOException { 137 MasterFileSystem mfs = env.getMasterServices().getMasterFileSystem(); 138 Path tableDir = FSUtils.getTableDir(mfs.getRootDir(), getTableName()); 139 return new Path(tableDir, ServerRegionReplicaUtil.getRegionInfoForFs(region).getEncodedName()); 140 } 141 142 protected final Path getWALRegionDir(MasterProcedureEnv env, RegionInfo region) 143 throws IOException { 144 return FSUtils.getWALRegionDir(env.getMasterConfiguration(), 145 region.getTable(), region.getEncodedName()); 146 } 147 148 /** 149 * Check that cluster is up and master is running. Check table is modifiable. 150 * If <code>enabled</code>, check table is enabled else check it is disabled. 151 * Call in Procedure constructor so can pass any exception to caller. 152 * @param enabled If true, check table is enabled and throw exception if not. If false, do the 153 * inverse. If null, do no table checks. 154 */ 155 protected void preflightChecks(MasterProcedureEnv env, Boolean enabled) throws HBaseIOException { 156 MasterServices master = env.getMasterServices(); 157 if (!master.isClusterUp()) { 158 throw new HBaseIOException("Cluster not up!"); 159 } 160 if (master.isStopping() || master.isStopped()) { 161 throw new HBaseIOException("Master stopping=" + master.isStopping() + 162 ", stopped=" + master.isStopped()); 163 } 164 if (enabled == null) { 165 // Don't do any table checks. 166 return; 167 } 168 try { 169 // Checks table exists and is modifiable. 170 checkTableModifiable(env); 171 TableName tn = getTableName(); 172 TableStateManager tsm = master.getTableStateManager(); 173 TableState ts = tsm.getTableState(tn); 174 if (enabled) { 175 if (!ts.isEnabledOrEnabling()) { 176 throw new TableNotEnabledException(tn); 177 } 178 } else { 179 if (!ts.isDisabledOrDisabling()) { 180 throw new TableNotDisabledException(tn); 181 } 182 } 183 } catch (IOException ioe) { 184 if (ioe instanceof HBaseIOException) { 185 throw (HBaseIOException)ioe; 186 } 187 throw new HBaseIOException(ioe); 188 } 189 } 190 191 /** 192 * Check region is online. 193 */ 194 protected static void checkOnline(MasterProcedureEnv env, final RegionInfo ri) 195 throws DoNotRetryRegionException { 196 RegionStates regionStates = env.getAssignmentManager().getRegionStates(); 197 RegionState rs = regionStates.getRegionState(ri); 198 if (rs == null) { 199 throw new UnknownRegionException("No RegionState found for " + ri.getEncodedName()); 200 } 201 if (!rs.isOpened()) { 202 throw new DoNotRetryRegionException(ri.getEncodedName() + " is not OPEN; regionState=" + rs); 203 } 204 if (ri.isSplitParent()) { 205 throw new DoNotRetryRegionException(ri.getEncodedName() + 206 " is not online (splitParent=true)"); 207 } 208 if (ri.isSplit()) { 209 throw new DoNotRetryRegionException(ri.getEncodedName() + " has split=true"); 210 } 211 if (ri.isOffline()) { 212 // RegionOfflineException is not instance of DNRIOE so wrap it. 213 throw new DoNotRetryRegionException(new RegionOfflineException(ri.getEncodedName())); 214 } 215 } 216}