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.client; 019 020import java.io.IOException; 021import java.util.HashMap; 022import java.util.List; 023import java.util.Map; 024import java.util.concurrent.Callable; 025import java.util.stream.Collectors; 026import org.apache.hadoop.conf.Configuration; 027import org.apache.hadoop.hbase.ServerName; 028import org.apache.hadoop.hbase.ipc.RpcControllerFactory; 029import org.apache.hadoop.hbase.master.RegionState; 030import org.apache.yetus.audience.InterfaceAudience; 031import org.slf4j.Logger; 032import org.slf4j.LoggerFactory; 033 034import org.apache.hbase.thirdparty.com.google.protobuf.ServiceException; 035 036import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil; 037import org.apache.hadoop.hbase.shaded.protobuf.RequestConverter; 038import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos; 039import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.AssignsResponse; 040import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.BypassProcedureRequest; 041import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.BypassProcedureResponse; 042import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.FixMetaRequest; 043import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.GetTableStateResponse; 044import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.HbckService.BlockingInterface; 045import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.RegionSpecifierAndState; 046import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.RunHbckChoreRequest; 047import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.RunHbckChoreResponse; 048import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.ScheduleSCPsForUnknownServersRequest; 049import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.ScheduleSCPsForUnknownServersResponse; 050import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.ScheduleServerCrashProcedureResponse; 051import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProtos.UnassignsResponse; 052 053/** 054 * Use {@link ClusterConnection#getHbck()} to obtain an instance of {@link Hbck} instead of 055 * constructing an HBaseHbck directly. 056 * <p> 057 * Connection should be an <i>unmanaged</i> connection obtained via 058 * {@link ConnectionFactory#createConnection(Configuration)}. 059 * </p> 060 * <p> 061 * NOTE: The methods in here can do damage to a cluster if applied in the wrong sequence or at the 062 * wrong time. Use with caution. For experts only. These methods are only for the extreme case where 063 * the cluster has been damaged or has achieved an inconsistent state because of some unforeseen 064 * circumstance or bug and requires manual intervention. 065 * <p> 066 * An instance of this class is lightweight and not-thread safe. A new instance should be created by 067 * each thread. Pooling or caching of the instance is not recommended. 068 * </p> 069 * @see ConnectionFactory 070 * @see ClusterConnection 071 * @see Hbck 072 */ 073@InterfaceAudience.Private 074public class HBaseHbck implements Hbck { 075 private static final Logger LOG = LoggerFactory.getLogger(HBaseHbck.class); 076 077 private boolean aborted; 078 private final BlockingInterface hbck; 079 080 private RpcControllerFactory rpcControllerFactory; 081 082 HBaseHbck(BlockingInterface hbck, RpcControllerFactory rpcControllerFactory) { 083 this.hbck = hbck; 084 this.rpcControllerFactory = rpcControllerFactory; 085 } 086 087 @Override 088 public void close() throws IOException { 089 // currently does nothing 090 } 091 092 @Override 093 public void abort(String why, Throwable e) { 094 this.aborted = true; 095 // Currently does nothing but throw the passed message and exception 096 throw new RuntimeException(why, e); 097 } 098 099 @Override 100 public boolean isAborted() { 101 return this.aborted; 102 } 103 104 @Override 105 public TableState setTableStateInMeta(TableState state) throws IOException { 106 try { 107 GetTableStateResponse response = 108 hbck.setTableStateInMeta(rpcControllerFactory.newController(), 109 RequestConverter.buildSetTableStateInMetaRequest(state)); 110 return TableState.convert(state.getTableName(), response.getTableState()); 111 } catch (ServiceException se) { 112 LOG.debug("table={}, state={}", state.getTableName(), state.getState(), se); 113 throw new IOException(se); 114 } 115 } 116 117 @Override 118 public Map<String, RegionState.State> setRegionStateInMeta( 119 Map<String, RegionState.State> nameOrEncodedName2State) throws IOException { 120 try { 121 if (LOG.isDebugEnabled()) { 122 nameOrEncodedName2State.forEach((k, v) -> LOG.debug("region={}, state={}", k, v)); 123 } 124 MasterProtos.SetRegionStateInMetaResponse response = 125 hbck.setRegionStateInMeta(rpcControllerFactory.newController(), 126 RequestConverter.buildSetRegionStateInMetaRequest(nameOrEncodedName2State)); 127 Map<String, RegionState.State> result = new HashMap<>(); 128 for (RegionSpecifierAndState nameAndState : response.getStatesList()) { 129 result.put(nameAndState.getRegionSpecifier().getValue().toStringUtf8(), 130 RegionState.State.convert(nameAndState.getState())); 131 } 132 return result; 133 } catch (ServiceException se) { 134 throw new IOException(se); 135 } 136 } 137 138 @Override 139 public List<Long> assigns(List<String> encodedRegionNames, boolean override) throws IOException { 140 try { 141 AssignsResponse response = this.hbck.assigns(rpcControllerFactory.newController(), 142 RequestConverter.toAssignRegionsRequest(encodedRegionNames, override)); 143 return response.getPidList(); 144 } catch (ServiceException se) { 145 LOG.debug(toCommaDelimitedString(encodedRegionNames), se); 146 throw new IOException(se); 147 } 148 } 149 150 @Override 151 public List<Long> unassigns(List<String> encodedRegionNames, boolean override) 152 throws IOException { 153 try { 154 UnassignsResponse response = this.hbck.unassigns(rpcControllerFactory.newController(), 155 RequestConverter.toUnassignRegionsRequest(encodedRegionNames, override)); 156 return response.getPidList(); 157 } catch (ServiceException se) { 158 LOG.debug(toCommaDelimitedString(encodedRegionNames), se); 159 throw new IOException(se); 160 } 161 } 162 163 private static String toCommaDelimitedString(List<String> list) { 164 return list.stream().collect(Collectors.joining(", ")); 165 } 166 167 @Override 168 public List<Boolean> bypassProcedure(List<Long> pids, long waitTime, boolean override, 169 boolean recursive) throws IOException { 170 BypassProcedureResponse response = ProtobufUtil.call(new Callable<BypassProcedureResponse>() { 171 @Override 172 public BypassProcedureResponse call() throws Exception { 173 try { 174 return hbck.bypassProcedure(rpcControllerFactory.newController(), 175 BypassProcedureRequest.newBuilder().addAllProcId(pids).setWaitTime(waitTime) 176 .setOverride(override).setRecursive(recursive).build()); 177 } catch (Throwable t) { 178 LOG.error(pids.stream().map(i -> i.toString()).collect(Collectors.joining(", ")), t); 179 throw t; 180 } 181 } 182 }); 183 return response.getBypassedList(); 184 } 185 186 @Override 187 public List<Long> scheduleServerCrashProcedures(List<ServerName> serverNames) throws IOException { 188 try { 189 ScheduleServerCrashProcedureResponse response = 190 this.hbck.scheduleServerCrashProcedure(rpcControllerFactory.newController(), 191 RequestConverter.toScheduleServerCrashProcedureRequest(serverNames)); 192 return response.getPidList(); 193 } catch (ServiceException se) { 194 LOG.debug(toCommaDelimitedString( 195 serverNames.stream().map(serverName -> ProtobufUtil.toServerName(serverName).toString()) 196 .collect(Collectors.toList())), 197 se); 198 throw new IOException(se); 199 } 200 } 201 202 @Override 203 public List<Long> scheduleSCPsForUnknownServers() throws IOException { 204 try { 205 ScheduleSCPsForUnknownServersResponse response = 206 this.hbck.scheduleSCPsForUnknownServers(rpcControllerFactory.newController(), 207 ScheduleSCPsForUnknownServersRequest.newBuilder().build()); 208 return response.getPidList(); 209 } catch (ServiceException se) { 210 LOG.debug("Failed to run ServerCrashProcedures for unknown servers", se); 211 throw new IOException(se); 212 } 213 } 214 215 @Override 216 public boolean runHbckChore() throws IOException { 217 try { 218 RunHbckChoreResponse response = this.hbck.runHbckChore(rpcControllerFactory.newController(), 219 RunHbckChoreRequest.newBuilder().build()); 220 return response.getRan(); 221 } catch (ServiceException se) { 222 LOG.debug("Failed to run HBCK chore", se); 223 throw new IOException(se); 224 } 225 } 226 227 @Override 228 public void fixMeta() throws IOException { 229 try { 230 this.hbck.fixMeta(rpcControllerFactory.newController(), FixMetaRequest.newBuilder().build()); 231 } catch (ServiceException se) { 232 throw new IOException(se); 233 } 234 } 235}