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.Map;
022import org.apache.hadoop.hbase.CellScanner;
023import org.apache.hadoop.hbase.HConstants;
024import org.apache.hadoop.hbase.HRegionInfo;
025import org.apache.hadoop.hbase.HRegionLocation;
026import org.apache.hadoop.hbase.ServerName;
027import org.apache.hadoop.hbase.TableName;
028import org.apache.hadoop.hbase.TableNotEnabledException;
029import org.apache.hadoop.hbase.ipc.HBaseRpcController;
030import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
031import org.apache.hadoop.hbase.util.Bytes;
032import org.apache.yetus.audience.InterfaceAudience;
033
034import org.apache.hbase.thirdparty.com.google.protobuf.RpcController;
035
036/**
037 * Implementations make a RPC call against a RegionService via a protobuf Service. Implement
038 * rpcCall() and the parent class setClientByServiceName; this latter is where the RPC stub gets set
039 * (the appropriate protobuf 'Service'/Client). Be sure to make use of the RpcController that this
040 * instance is carrying via #getRpcController().
041 * <p>
042 * TODO: this class is actually tied to one region, because most of the paths make use of the
043 * regioninfo part of location when building requests. The only reason it works for multi-region
044 * requests (e.g. batch) is that they happen to not use the region parts. This could be done cleaner
045 * (e.g. having a generic parameter and 2 derived classes, RegionCallable and actual
046 * RegionServerCallable with ServerName.
047 * @param <T> The class that the ServerCallable handles.
048 * @param <S> The protocol to use (Admin or Client or even an Endpoint over in MetaTableAccessor).
049 */
050// TODO: MasterCallable and this Class have a lot in common. UNIFY!
051// Public but should be package private only it is used by MetaTableAccessor. FIX!!
052@InterfaceAudience.Private
053public abstract class RegionServerCallable<T, S> implements RetryingCallable<T> {
054  private final Connection connection;
055  private final TableName tableName;
056  private final byte[] row;
057  /**
058   * Some subclasses want to set their own location. Make it protected.
059   */
060  protected HRegionLocation location;
061  protected S stub;
062
063  /**
064   * This is 99% of the time a HBaseRpcController but also used doing Coprocessor Endpoints and in
065   * this case, it is a ServerRpcControllable which is not a HBaseRpcController. Can be null!
066   */
067  protected final RpcController rpcController;
068  private int priority = HConstants.NORMAL_QOS;
069  protected final Map<String, byte[]> requestAttributes;
070
071  /**
072   * @param connection    Connection to use.
073   * @param rpcController Controller to use; can be shaded or non-shaded.
074   * @param tableName     Table name to which <code>row</code> belongs.
075   * @param row           The row we want in <code>tableName</code>.
076   */
077  public RegionServerCallable(Connection connection, TableName tableName, byte[] row,
078    RpcController rpcController, Map<String, byte[]> requestAttributes) {
079    this(connection, tableName, row, rpcController, HConstants.NORMAL_QOS, requestAttributes);
080  }
081
082  public RegionServerCallable(Connection connection, TableName tableName, byte[] row,
083    RpcController rpcController, int priority, Map<String, byte[]> requestAttributes) {
084    super();
085    this.connection = connection;
086    this.tableName = tableName;
087    this.row = row;
088    this.rpcController = rpcController;
089    this.priority = priority;
090    this.requestAttributes = requestAttributes;
091  }
092
093  protected RpcController getRpcController() {
094    return this.rpcController;
095  }
096
097  protected void setStub(S stub) {
098    this.stub = stub;
099  }
100
101  protected S getStub() {
102    return this.stub;
103  }
104
105  /**
106   * Override that changes call Exception from {@link Exception} to {@link IOException}. Also does
107   * set up of the rpcController.
108   */
109  @Override
110  public T call(int callTimeout) throws IOException {
111    try {
112      // Iff non-null and an instance of a SHADED rpcController, do config! Unshaded -- i.e.
113      // com.google.protobuf.RpcController or null -- will just skip over this config.
114      if (getRpcController() != null) {
115        RpcController shadedRpcController = (RpcController) getRpcController();
116        // Do a reset to clear previous states, such as CellScanner.
117        shadedRpcController.reset();
118        if (shadedRpcController instanceof HBaseRpcController) {
119          HBaseRpcController hrc = (HBaseRpcController) getRpcController();
120          // If it is an instance of HBaseRpcController, we can set priority on the controller based
121          // off the tableName. Set call timeout too.
122          hrc.setPriority(tableName);
123          hrc.setPriority(priority);
124          hrc.setCallTimeout(callTimeout);
125          hrc.setRequestAttributes(requestAttributes);
126          if (tableName != null) {
127            hrc.setTableName(tableName);
128          }
129        }
130      }
131      return rpcCall();
132    } catch (Exception e) {
133      throw ProtobufUtil.handleRemoteException(e);
134    }
135  }
136
137  /**
138   * Run the RPC call. Implement this method. To get at the rpcController that has been created and
139   * configured to make this rpc call, use getRpcController(). We are trying to contain
140   * rpcController references so we don't pollute codebase with protobuf references; keep the
141   * protobuf references contained and only present in a few classes rather than all about the code
142   * base.
143   */
144  protected abstract T rpcCall() throws Exception;
145
146  /**
147   * Get the RpcController CellScanner. If the RpcController is a HBaseRpcController, which it is in
148   * all cases except when we are processing Coprocessor Endpoint, then this method returns a
149   * reference to the CellScanner that the HBaseRpcController is carrying. Do it up here in this
150   * Callable so we don't have to scatter ugly instanceof tests around the codebase. Will return
151   * null if called in a Coproccessor Endpoint context. Should never happen.
152   */
153  protected CellScanner getRpcControllerCellScanner() {
154    return (getRpcController() != null && getRpcController() instanceof HBaseRpcController)
155      ? ((HBaseRpcController) getRpcController()).cellScanner()
156      : null;
157  }
158
159  protected void setRpcControllerCellScanner(CellScanner cellScanner) {
160    if (getRpcController() != null && getRpcController() instanceof HBaseRpcController) {
161      ((HBaseRpcController) this.rpcController).setCellScanner(cellScanner);
162    }
163  }
164
165  /** Returns {@link ClusterConnection} instance used by this Callable. */
166  protected ClusterConnection getConnection() {
167    return (ClusterConnection) this.connection;
168  }
169
170  protected HRegionLocation getLocation() {
171    return this.location;
172  }
173
174  protected void setLocation(final HRegionLocation location) {
175    this.location = location;
176  }
177
178  public TableName getTableName() {
179    return this.tableName;
180  }
181
182  public byte[] getRow() {
183    return this.row;
184  }
185
186  protected int getPriority() {
187    return this.priority;
188  }
189
190  @Override
191  public void throwable(Throwable t, boolean retrying) {
192    if (location != null) {
193      getConnection().updateCachedLocations(tableName, location.getRegionInfo().getRegionName(),
194        row, t, location.getServerName());
195    }
196  }
197
198  @Override
199  public String getExceptionMessageAdditionalDetail() {
200    return "row '" + Bytes.toStringBinary(row) + "' on table '" + tableName + "' at " + location;
201  }
202
203  @Override
204  public long sleep(long pause, int tries) {
205    return ConnectionUtils.getPauseTime(pause, tries);
206  }
207
208  /** Returns the HRegionInfo for the current region */
209  public HRegionInfo getHRegionInfo() {
210    if (this.location == null) {
211      return null;
212    }
213    return this.location.getRegionInfo();
214  }
215
216  @Override
217  public void prepare(final boolean reload) throws IOException {
218    // check table state if this is a retry
219    if (
220      reload && tableName != null && !tableName.equals(TableName.META_TABLE_NAME)
221        && getConnection().isTableDisabled(tableName)
222    ) {
223      throw new TableNotEnabledException(tableName.getNameAsString() + " is disabled.");
224    }
225    try (RegionLocator regionLocator = connection.getRegionLocator(tableName)) {
226      this.location = regionLocator.getRegionLocation(row);
227    }
228    if (this.location == null) {
229      throw new IOException("Failed to find location, tableName=" + tableName + ", row="
230        + Bytes.toString(row) + ", reload=" + reload);
231    }
232    setStubByServiceName(this.location.getServerName());
233  }
234
235  /**
236   * Set the RCP client stub
237   * @param serviceName to get the rpc stub for
238   * @throws IOException When client could not be created
239   */
240  protected abstract void setStubByServiceName(ServerName serviceName) throws IOException;
241}