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