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.ArrayList;
022import java.util.HashMap;
023import java.util.List;
024import java.util.Map;
025
026import org.apache.hadoop.hbase.CellScannable;
027import org.apache.hadoop.hbase.CellUtil;
028import org.apache.hadoop.hbase.DoNotRetryIOException;
029import org.apache.hadoop.hbase.HRegionInfo;
030import org.apache.hadoop.hbase.HRegionLocation;
031import org.apache.hadoop.hbase.ServerName;
032import org.apache.hadoop.hbase.TableName;
033import org.apache.yetus.audience.InterfaceAudience;
034import org.apache.hadoop.hbase.shaded.protobuf.RequestConverter;
035import org.apache.hadoop.hbase.shaded.protobuf.ResponseConverter;
036import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos;
037import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MultiRequest;
038import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.MutationProto;
039import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.RegionAction;
040import org.apache.hbase.thirdparty.com.google.protobuf.RpcController;
041
042/**
043 * Callable that handles the <code>multi</code> method call going against a single
044 * regionserver; i.e. A RegionServerCallable for the multi call (It is NOT a
045 * RegionServerCallable that goes against multiple regions).
046 * @param <R>
047 */
048@InterfaceAudience.Private
049class MultiServerCallable extends CancellableRegionServerCallable<MultiResponse> {
050  private MultiAction multiAction;
051  private boolean cellBlock;
052
053  MultiServerCallable(final ClusterConnection connection, final TableName tableName,
054      final ServerName location, final MultiAction multi, RpcController rpcController,
055      int rpcTimeout, RetryingTimeTracker tracker, int priority) {
056    super(connection, tableName, null, rpcController, rpcTimeout, tracker, priority);
057    this.multiAction = multi;
058    // RegionServerCallable has HRegionLocation field, but this is a multi-region request.
059    // Using region info from parent HRegionLocation would be a mistake for this class; so
060    // we will store the server here, and throw if someone tries to obtain location/regioninfo.
061    this.location = new HRegionLocation(null, location);
062    this.cellBlock = isCellBlock();
063  }
064
065  public void reset(ServerName location, MultiAction multiAction) {
066    this.location = new HRegionLocation(null, location);
067    this.multiAction = multiAction;
068    this.cellBlock = isCellBlock();
069  }
070
071  @Override
072  protected HRegionLocation getLocation() {
073    throw new RuntimeException("Cannot get region location for multi-region request");
074  }
075
076  @Override
077  public HRegionInfo getHRegionInfo() {
078    throw new RuntimeException("Cannot get region info for multi-region request");
079  }
080
081  MultiAction getMulti() {
082    return this.multiAction;
083  }
084
085  @Override
086  protected MultiResponse rpcCall() throws Exception {
087    int countOfActions = this.multiAction.size();
088    if (countOfActions <= 0) throw new DoNotRetryIOException("No Actions");
089    MultiRequest.Builder multiRequestBuilder = MultiRequest.newBuilder();
090    RegionAction.Builder regionActionBuilder = RegionAction.newBuilder();
091    ClientProtos.Action.Builder actionBuilder = ClientProtos.Action.newBuilder();
092    MutationProto.Builder mutationBuilder = MutationProto.newBuilder();
093
094    // Pre-size. Presume at least a KV per Action. There are likely more.
095    List<CellScannable> cells =
096        (this.cellBlock ? new ArrayList<CellScannable>(countOfActions) : null);
097
098    long nonceGroup = multiAction.getNonceGroup();
099
100    // Map from a created RegionAction to the original index for a RowMutations/CheckAndMutate
101    // within the original list of actions. This will be used to process the results when there
102    // is RowMutations/CheckAndMutate in the action list.
103    Map<Integer, Integer> indexMap = new HashMap<>();
104    // The multi object is a list of Actions by region. Iterate by region.
105    for (Map.Entry<byte[], List<Action>> e: this.multiAction.actions.entrySet()) {
106      final byte [] regionName = e.getKey();
107      final List<Action> actions = e.getValue();
108      if (this.cellBlock) {
109        // Send data in cellblocks.
110        // multiRequestBuilder will be populated with region actions.
111        // indexMap will be non-empty after the call if there is RowMutations/CheckAndMutate in
112        // the action list.
113        RequestConverter.buildNoDataRegionActions(regionName, actions, cells, multiRequestBuilder,
114          regionActionBuilder, actionBuilder, mutationBuilder, nonceGroup, indexMap);
115      } else {
116        // multiRequestBuilder will be populated with region actions.
117        // indexMap will be non-empty after the call if there is RowMutations/CheckAndMutate in
118        // the action list.
119        RequestConverter.buildRegionActions(regionName, actions, multiRequestBuilder,
120          regionActionBuilder, actionBuilder, mutationBuilder, nonceGroup, indexMap);
121      }
122    }
123
124    if (cells != null) {
125      setRpcControllerCellScanner(CellUtil.createCellScanner(cells));
126    }
127    ClientProtos.MultiRequest requestProto = multiRequestBuilder.build();
128    ClientProtos.MultiResponse responseProto = getStub().multi(getRpcController(), requestProto);
129    if (responseProto == null) return null; // Occurs on cancel
130    return ResponseConverter.getResults(requestProto, indexMap, responseProto,
131      getRpcControllerCellScanner());
132  }
133
134  /**
135   * @return True if we should send data in cellblocks.  This is an expensive call.  Cache the
136   * result if you can rather than call each time.
137   */
138  private boolean isCellBlock() {
139    // This is not exact -- the configuration could have changed on us after connection was set up
140    // but it will do for now.
141    ClusterConnection conn = getConnection();
142    return conn.hasCellBlockSupport();
143  }
144
145  @Override
146  public void prepare(boolean reload) throws IOException {
147    // Use the location we were given in the constructor rather than go look it up.
148    setStub(getConnection().getClient(this.location.getServerName()));
149  }
150
151  ServerName getServerName() {
152    return location.getServerName();
153  }
154}