View Javadoc

1   /**
2    * Licensed to the Apache Software Foundation (ASF) under one
3    * or more contributor license agreements.  See the NOTICE file
4    * distributed with this work for additional information
5    * regarding copyright ownership.  The ASF licenses this file
6    * to you under the Apache License, Version 2.0 (the
7    * "License"); you may not use this file except in compliance
8    * with the License.  You may obtain a copy of the License at
9    *
10   *     http://www.apache.org/licenses/LICENSE-2.0
11   *
12   * Unless required by applicable law or agreed to in writing, software
13   * distributed under the License is distributed on an "AS IS" BASIS,
14   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15   * See the License for the specific language governing permissions and
16   * limitations under the License.
17   */
18  
19  package org.apache.hadoop.hbase.client;
20  
21  import java.io.IOException;
22  import java.io.InterruptedIOException;
23  import java.util.ArrayList;
24  import java.util.HashSet;
25  import java.util.List;
26  import java.util.Set;
27  import java.util.concurrent.Callable;
28  import java.util.concurrent.CancellationException;
29  import java.util.concurrent.ExecutionException;
30  import java.util.concurrent.ExecutorService;
31  import java.util.concurrent.Future;
32  import java.util.concurrent.TimeUnit;
33  import java.util.concurrent.atomic.AtomicBoolean;
34  
35  import org.apache.commons.logging.Log;
36  import org.apache.commons.logging.LogFactory;
37  import org.apache.hadoop.conf.Configuration;
38  import org.apache.hadoop.hbase.HRegionInfo;
39  import org.apache.hadoop.hbase.RegionLocations;
40  import org.apache.hadoop.hbase.TableName;
41  import org.apache.hadoop.hbase.classification.InterfaceAudience;
42  import org.apache.hadoop.hbase.util.BoundedCompletionService;
43  import org.apache.hadoop.hbase.util.Pair;
44  /**
45   * This class has the logic for handling scanners for regions with and without replicas.
46   * 1. A scan is attempted on the default (primary) region
47   * 2. The scanner sends all the RPCs to the default region until it is done, or, there
48   * is a timeout on the default (a timeout of zero is disallowed).
49   * 3. If there is a timeout in (2) above, scanner(s) is opened on the non-default replica(s)
50   * 4. The results from the first successful scanner are taken, and it is stored which server
51   * returned the results.
52   * 5. The next RPCs are done on the above stored server until it is done or there is a timeout,
53   * in which case, the other replicas are queried (as in (3) above).
54   *
55   */
56  @InterfaceAudience.Private
57  class ScannerCallableWithReplicas implements RetryingCallable<Result[]> {
58    private final Log LOG = LogFactory.getLog(this.getClass());
59    volatile ScannerCallable currentScannerCallable;
60    AtomicBoolean replicaSwitched = new AtomicBoolean(false);
61    final ClusterConnection cConnection;
62    protected final ExecutorService pool;
63    protected final int timeBeforeReplicas;
64    private final Scan scan;
65    private final int retries;
66    private Result lastResult;
67    private final RpcRetryingCaller<Result[]> caller;
68    private final TableName tableName;
69    private Configuration conf;
70    private int scannerTimeout;
71    private Set<ScannerCallable> outstandingCallables = new HashSet<ScannerCallable>();
72  
73    public ScannerCallableWithReplicas (TableName tableName, ClusterConnection cConnection,
74        ScannerCallable baseCallable, ExecutorService pool, int timeBeforeReplicas, Scan scan,
75        int retries, int scannerTimeout, int caching, Configuration conf,
76        RpcRetryingCaller<Result []> caller) {
77      this.currentScannerCallable = baseCallable;
78      this.cConnection = cConnection;
79      this.pool = pool;
80      if (timeBeforeReplicas < 0) {
81        throw new IllegalArgumentException("Invalid value of operation timeout on the primary");
82      }
83      this.timeBeforeReplicas = timeBeforeReplicas;
84      this.scan = scan;
85      this.retries = retries;
86      this.tableName = tableName;
87      this.conf = conf;
88      this.scannerTimeout = scannerTimeout;
89      this.caller = caller;
90    }
91  
92    public void setClose() {
93      currentScannerCallable.setClose();
94    }
95  
96    public void setCaching(int caching) {
97      currentScannerCallable.setCaching(caching);
98    }
99  
100   public int getCaching() {
101     return currentScannerCallable.getCaching();
102   }
103 
104   public HRegionInfo getHRegionInfo() {
105     return currentScannerCallable.getHRegionInfo();
106   }
107 
108   @Override
109   public Result [] call(int timeout) throws IOException {
110     // If the active replica callable was closed somewhere, invoke the RPC to
111     // really close it. In the case of regular scanners, this applies. We make couple
112     // of RPCs to a RegionServer, and when that region is exhausted, we set
113     // the closed flag. Then an RPC is required to actually close the scanner.
114     if (currentScannerCallable != null && currentScannerCallable.closed) {
115       // For closing we target that exact scanner (and not do replica fallback like in
116       // the case of normal reads)
117       if (LOG.isTraceEnabled()) {
118         LOG.trace("Closing scanner id=" + currentScannerCallable.scannerId);
119       }
120       Result[] r = currentScannerCallable.call(timeout);
121       currentScannerCallable = null;
122       return r;
123     }
124     // We need to do the following:
125     //1. When a scan goes out to a certain replica (default or not), we need to
126     //   continue to hit that until there is a failure. So store the last successfully invoked
127     //   replica
128     //2. We should close the "losing" scanners (scanners other than the ones we hear back
129     //   from first)
130     //
131     RegionLocations rl = RpcRetryingCallerWithReadReplicas.getRegionLocations(true,
132         RegionReplicaUtil.DEFAULT_REPLICA_ID, cConnection, tableName,
133         currentScannerCallable.getRow());
134 
135     // allocate a boundedcompletion pool of some multiple of number of replicas.
136     // We want to accomodate some RPCs for redundant replica scans (but are still in progress)
137     BoundedCompletionService<Pair<Result[], ScannerCallable>> cs =
138         new BoundedCompletionService<Pair<Result[], ScannerCallable>>(pool, rl.size() * 5);
139 
140     List<ExecutionException> exceptions = null;
141     int submitted = 0, completed = 0;
142     AtomicBoolean done = new AtomicBoolean(false);
143     replicaSwitched.set(false);
144     // submit call for the primary replica.
145     submitted += addCallsForCurrentReplica(cs, rl);
146     try {
147       // wait for the timeout to see whether the primary responds back
148       Future<Pair<Result[], ScannerCallable>> f = cs.poll(timeBeforeReplicas,
149           TimeUnit.MICROSECONDS); // Yes, microseconds
150       if (f != null) {
151         Pair<Result[], ScannerCallable> r = f.get();
152         if (r != null && r.getSecond() != null) {
153           updateCurrentlyServingReplica(r.getSecond(), r.getFirst(), done, pool);
154         }
155         return r == null ? null : r.getFirst(); //great we got a response
156       }
157     } catch (ExecutionException e) {
158       // the primary call failed with RetriesExhaustedException or DoNotRetryIOException
159       // but the secondaries might still succeed. Continue on the replica RPCs.
160       exceptions = new ArrayList<ExecutionException>(rl.size());
161       exceptions.add(e);
162       completed++;
163     } catch (CancellationException e) {
164       throw new InterruptedIOException(e.getMessage());
165     } catch (InterruptedException e) {
166       throw new InterruptedIOException(e.getMessage());
167     }
168     // submit call for the all of the secondaries at once
169     // TODO: this may be an overkill for large region replication
170     submitted += addCallsForOtherReplicas(cs, rl, 0, rl.size() - 1);
171     try {
172       while (completed < submitted) {
173         try {
174           Future<Pair<Result[], ScannerCallable>> f = cs.take();
175           Pair<Result[], ScannerCallable> r = f.get();
176           if (r != null && r.getSecond() != null) {
177             updateCurrentlyServingReplica(r.getSecond(), r.getFirst(), done, pool);
178           }
179           return r == null ? null : r.getFirst(); // great we got an answer
180         } catch (ExecutionException e) {
181           // if not cancel or interrupt, wait until all RPC's are done
182           // one of the tasks failed. Save the exception for later.
183           if (exceptions == null) exceptions = new ArrayList<ExecutionException>(rl.size());
184           exceptions.add(e);
185           completed++;
186         }
187       }
188     } catch (CancellationException e) {
189       throw new InterruptedIOException(e.getMessage());
190     } catch (InterruptedException e) {
191       throw new InterruptedIOException(e.getMessage());
192     } finally {
193       // We get there because we were interrupted or because one or more of the
194       // calls succeeded or failed. In all case, we stop all our tasks.
195       cs.cancelAll(true);
196     }
197 
198     if (exceptions != null && !exceptions.isEmpty()) {
199       RpcRetryingCallerWithReadReplicas.throwEnrichedException(exceptions.get(0),
200           retries); // just rethrow the first exception for now.
201     }
202     return null; // unreachable
203   }
204 
205   private void updateCurrentlyServingReplica(ScannerCallable scanner, Result[] result,
206       AtomicBoolean done, ExecutorService pool) {
207     if (done.compareAndSet(false, true)) {
208       if (currentScannerCallable != scanner) replicaSwitched.set(true);
209       currentScannerCallable = scanner;
210       // store where to start the replica scanner from if we need to.
211       if (result != null && result.length != 0) this.lastResult = result[result.length - 1];
212       if (LOG.isTraceEnabled()) {
213         LOG.trace("Setting current scanner as id=" + currentScannerCallable.scannerId +
214             " associated with replica=" + currentScannerCallable.getHRegionInfo().getReplicaId());
215       }
216       // close all outstanding replica scanners but the one we heard back from
217       outstandingCallables.remove(scanner);
218       for (ScannerCallable s : outstandingCallables) {
219         if (LOG.isTraceEnabled()) {
220           LOG.trace("Closing scanner id=" + s.scannerId +
221             ", replica=" + s.getHRegionInfo().getRegionId() +
222             " because slow and replica=" +
223             this.currentScannerCallable.getHRegionInfo().getReplicaId() + " succeeded");
224         }
225         // Submit the "close" to the pool since this might take time, and we don't
226         // want to wait for the "close" to happen yet. The "wait" will happen when
227         // the table is closed (when the awaitTermination of the underlying pool is called)
228         s.setClose();
229         RetryingRPC r = new RetryingRPC(s);
230         pool.submit(r);
231       }
232       // now clear outstandingCallables since we scheduled a close for all the contained scanners
233       outstandingCallables.clear();
234     }
235   }
236 
237   /**
238    * When a scanner switches in the middle of scanning (the 'next' call fails
239    * for example), the upper layer {@link ClientScanner} needs to know
240    * @return
241    */
242   public boolean switchedToADifferentReplica() {
243     return replicaSwitched.get();
244   }
245 
246   private int addCallsForCurrentReplica(
247       BoundedCompletionService<Pair<Result[], ScannerCallable>> cs, RegionLocations rl) {
248     RetryingRPC retryingOnReplica = new RetryingRPC(currentScannerCallable);
249     outstandingCallables.add(currentScannerCallable);
250     cs.submit(retryingOnReplica);
251     return 1;
252   }
253 
254   private int addCallsForOtherReplicas(
255       BoundedCompletionService<Pair<Result[], ScannerCallable>> cs, RegionLocations rl, int min,
256       int max) {
257     if (scan.getConsistency() == Consistency.STRONG) {
258       return 0; // not scheduling on other replicas for strong consistency
259     }
260     for (int id = min; id <= max; id++) {
261       if (currentScannerCallable.getHRegionInfo().getReplicaId() == id) {
262         continue; //this was already scheduled earlier
263       }
264       ScannerCallable s = currentScannerCallable.getScannerCallableForReplica(id);
265       if (this.lastResult != null) {
266         s.getScan().setStartRow(this.lastResult.getRow());
267       }
268       outstandingCallables.add(s);
269       RetryingRPC retryingOnReplica = new RetryingRPC(s);
270       cs.submit(retryingOnReplica);
271     }
272     return max - min + 1;
273   }
274 
275   class RetryingRPC implements Callable<Pair<Result[], ScannerCallable>> {
276     final ScannerCallable callable;
277 
278     RetryingRPC(ScannerCallable callable) {
279       this.callable = callable;
280     }
281 
282     @Override
283     public Pair<Result[], ScannerCallable> call() throws IOException {
284       // For the Consistency.STRONG (default case), we reuse the caller
285       // to keep compatibility with what is done in the past
286       // For the Consistency.TIMELINE case, we can't reuse the caller
287       // since we could be making parallel RPCs (caller.callWithRetries is synchronized
288       // and we can't invoke it multiple times at the same time)
289       RpcRetryingCaller<Result[]> caller = ScannerCallableWithReplicas.this.caller;
290       if (scan.getConsistency() == Consistency.TIMELINE) {
291         caller = new RpcRetryingCallerFactory(ScannerCallableWithReplicas.this.conf).
292             <Result[]>newCaller();
293       }
294       Result[] res = caller.callWithRetries(callable, scannerTimeout);
295       return new Pair<Result[], ScannerCallable>(res, callable);
296     }
297   }
298 
299   @Override
300   public void prepare(boolean reload) throws IOException {
301   }
302 
303   @Override
304   public void throwable(Throwable t, boolean retrying) {
305     currentScannerCallable.throwable(t, retrying);
306   }
307 
308   @Override
309   public String getExceptionMessageAdditionalDetail() {
310     return currentScannerCallable.getExceptionMessageAdditionalDetail();
311   }
312 
313   @Override
314   public long sleep(long pause, int tries) {
315     return currentScannerCallable.sleep(pause, tries);
316   }
317 }