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 static org.apache.hadoop.hbase.HConstants.EMPTY_END_ROW;
021import static org.apache.hadoop.hbase.HConstants.EMPTY_START_ROW;
022import static org.apache.hadoop.hbase.client.ConnectionUtils.createScanResultCache;
023import static org.apache.hadoop.hbase.client.ConnectionUtils.getLocateType;
024import static org.apache.hadoop.hbase.client.ConnectionUtils.incRPCCallsMetrics;
025import static org.apache.hadoop.hbase.client.ConnectionUtils.incRPCRetriesMetrics;
026import static org.apache.hadoop.hbase.client.ConnectionUtils.incRegionCountMetrics;
027import static org.apache.hadoop.hbase.client.ConnectionUtils.isRemote;
028import static org.apache.hadoop.hbase.client.ConnectionUtils.timelineConsistentRead;
029import static org.apache.hadoop.hbase.util.FutureUtils.addListener;
030
031import io.opentelemetry.api.trace.Span;
032import io.opentelemetry.api.trace.StatusCode;
033import io.opentelemetry.context.Scope;
034import java.io.IOException;
035import java.util.Map;
036import java.util.concurrent.CompletableFuture;
037import java.util.concurrent.TimeUnit;
038import java.util.concurrent.atomic.AtomicInteger;
039import org.apache.hadoop.hbase.HRegionLocation;
040import org.apache.hadoop.hbase.TableName;
041import org.apache.hadoop.hbase.client.metrics.ScanMetrics;
042import org.apache.hadoop.hbase.client.trace.TableOperationSpanBuilder;
043import org.apache.hadoop.hbase.ipc.HBaseRpcController;
044import org.apache.hadoop.hbase.trace.TraceUtil;
045import org.apache.yetus.audience.InterfaceAudience;
046
047import org.apache.hbase.thirdparty.io.netty.util.Timer;
048
049import org.apache.hadoop.hbase.shaded.protobuf.RequestConverter;
050import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService;
051import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ClientService.Interface;
052import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ScanRequest;
053import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.ScanResponse;
054
055/**
056 * The asynchronous client scanner implementation.
057 * <p>
058 * Here we will call OpenScanner first and use the returned scannerId to create a
059 * {@link AsyncScanSingleRegionRpcRetryingCaller} to do the real scan operation. The return value of
060 * {@link AsyncScanSingleRegionRpcRetryingCaller} will tell us whether open a new scanner or finish
061 * scan.
062 */
063@InterfaceAudience.Private
064class AsyncClientScanner {
065
066  // We will use this scan object during the whole scan operation. The
067  // AsyncScanSingleRegionRpcRetryingCaller will modify this scan object directly.
068  private final Scan scan;
069
070  private final ScanMetrics scanMetrics;
071
072  private final AdvancedScanResultConsumer consumer;
073
074  private final TableName tableName;
075
076  private final AsyncConnectionImpl conn;
077
078  private final Timer retryTimer;
079
080  private final long pauseNs;
081
082  private final long pauseNsForServerOverloaded;
083
084  private final int maxAttempts;
085
086  private final long scanTimeoutNs;
087
088  private final long rpcTimeoutNs;
089
090  private final int startLogErrorsCnt;
091
092  private final ScanResultCache resultCache;
093
094  private final Span span;
095
096  private final Map<String, byte[]> requestAttributes;
097
098  public AsyncClientScanner(Scan scan, AdvancedScanResultConsumer consumer, TableName tableName,
099    AsyncConnectionImpl conn, Timer retryTimer, long pauseNs, long pauseNsForServerOverloaded,
100    int maxAttempts, long scanTimeoutNs, long rpcTimeoutNs, int startLogErrorsCnt,
101    Map<String, byte[]> requestAttributes) {
102    if (scan.getStartRow() == null) {
103      scan.withStartRow(EMPTY_START_ROW, scan.includeStartRow());
104    }
105    if (scan.getStopRow() == null) {
106      scan.withStopRow(EMPTY_END_ROW, scan.includeStopRow());
107    }
108    this.scan = scan;
109    this.consumer = consumer;
110    this.tableName = tableName;
111    this.conn = conn;
112    this.retryTimer = retryTimer;
113    this.pauseNs = pauseNs;
114    this.pauseNsForServerOverloaded = pauseNsForServerOverloaded;
115    this.maxAttempts = maxAttempts;
116    this.scanTimeoutNs = scanTimeoutNs;
117    this.rpcTimeoutNs = rpcTimeoutNs;
118    this.startLogErrorsCnt = startLogErrorsCnt;
119    this.resultCache = createScanResultCache(scan);
120    this.requestAttributes = requestAttributes;
121    if (scan.isScanMetricsEnabled()) {
122      this.scanMetrics = new ScanMetrics();
123      consumer.onScanMetricsCreated(scanMetrics);
124    } else {
125      this.scanMetrics = null;
126    }
127
128    /*
129     * Assumes that the `start()` method is called immediately after construction. If this is no
130     * longer the case, for tracing correctness, we should move the start of the span into the
131     * `start()` method. The cost of doing so would be making access to the `span` safe for
132     * concurrent threads.
133     */
134    span = new TableOperationSpanBuilder(conn).setTableName(tableName).setOperation(scan).build();
135    if (consumer instanceof AsyncTableResultScanner) {
136      AsyncTableResultScanner scanner = (AsyncTableResultScanner) consumer;
137      scanner.setSpan(span);
138    }
139  }
140
141  private static final class OpenScannerResponse {
142
143    public final HRegionLocation loc;
144
145    public final boolean isRegionServerRemote;
146
147    public final ClientService.Interface stub;
148
149    public final HBaseRpcController controller;
150
151    public final ScanResponse resp;
152
153    public OpenScannerResponse(HRegionLocation loc, boolean isRegionServerRemote, Interface stub,
154      HBaseRpcController controller, ScanResponse resp) {
155      this.loc = loc;
156      this.isRegionServerRemote = isRegionServerRemote;
157      this.stub = stub;
158      this.controller = controller;
159      this.resp = resp;
160    }
161  }
162
163  private final AtomicInteger openScannerTries = new AtomicInteger();
164
165  private CompletableFuture<OpenScannerResponse> callOpenScanner(HBaseRpcController controller,
166    HRegionLocation loc, ClientService.Interface stub) {
167    try (Scope ignored = span.makeCurrent()) {
168      boolean isRegionServerRemote = isRemote(loc.getHostname());
169      incRPCCallsMetrics(scanMetrics, isRegionServerRemote);
170      if (openScannerTries.getAndIncrement() > 1) {
171        incRPCRetriesMetrics(scanMetrics, isRegionServerRemote);
172      }
173      CompletableFuture<OpenScannerResponse> future = new CompletableFuture<>();
174      try {
175        ScanRequest request = RequestConverter.buildScanRequest(loc.getRegion().getRegionName(),
176          scan, scan.getCaching(), false);
177        stub.scan(controller, request, resp -> {
178          try (Scope ignored1 = span.makeCurrent()) {
179            if (controller.failed()) {
180              final IOException e = controller.getFailed();
181              future.completeExceptionally(e);
182              TraceUtil.setError(span, e);
183              span.end();
184              return;
185            }
186            future
187              .complete(new OpenScannerResponse(loc, isRegionServerRemote, stub, controller, resp));
188          }
189        });
190      } catch (IOException e) {
191        // span is closed by listener attached to the Future in `openScanner()`
192        future.completeExceptionally(e);
193      }
194      return future;
195    }
196  }
197
198  private void startScan(OpenScannerResponse resp) {
199    addListener(
200      conn.callerFactory.scanSingleRegion().id(resp.resp.getScannerId()).location(resp.loc)
201        .remote(resp.isRegionServerRemote)
202        .scannerLeaseTimeoutPeriod(resp.resp.getTtl(), TimeUnit.MILLISECONDS).stub(resp.stub)
203        .setScan(scan).metrics(scanMetrics).consumer(consumer).resultCache(resultCache)
204        .rpcTimeout(rpcTimeoutNs, TimeUnit.NANOSECONDS)
205        .scanTimeout(scanTimeoutNs, TimeUnit.NANOSECONDS).pause(pauseNs, TimeUnit.NANOSECONDS)
206        .pauseForServerOverloaded(pauseNsForServerOverloaded, TimeUnit.NANOSECONDS)
207        .maxAttempts(maxAttempts).startLogErrorsCnt(startLogErrorsCnt)
208        .setRequestAttributes(requestAttributes).start(resp.controller, resp.resp),
209      (hasMore, error) -> {
210        try (Scope ignored = span.makeCurrent()) {
211          if (error != null) {
212            try {
213              consumer.onError(error);
214              return;
215            } finally {
216              TraceUtil.setError(span, error);
217              span.end();
218            }
219          }
220          if (hasMore) {
221            openScanner();
222          } else {
223            try {
224              consumer.onComplete();
225            } finally {
226              span.setStatus(StatusCode.OK);
227              span.end();
228            }
229          }
230        }
231      });
232  }
233
234  private CompletableFuture<OpenScannerResponse> openScanner(int replicaId) {
235    try (Scope ignored = span.makeCurrent()) {
236      return conn.callerFactory.<OpenScannerResponse> single().table(tableName)
237        .row(scan.getStartRow()).replicaId(replicaId).locateType(getLocateType(scan))
238        .priority(scan.getPriority()).rpcTimeout(rpcTimeoutNs, TimeUnit.NANOSECONDS)
239        .operationTimeout(scanTimeoutNs, TimeUnit.NANOSECONDS).pause(pauseNs, TimeUnit.NANOSECONDS)
240        .pauseForServerOverloaded(pauseNsForServerOverloaded, TimeUnit.NANOSECONDS)
241        .maxAttempts(maxAttempts).startLogErrorsCnt(startLogErrorsCnt)
242        .setRequestAttributes(requestAttributes).action(this::callOpenScanner).call();
243    }
244  }
245
246  private long getPrimaryTimeoutNs() {
247    return TableName.isMetaTableName(tableName)
248      ? conn.connConf.getPrimaryMetaScanTimeoutNs()
249      : conn.connConf.getPrimaryScanTimeoutNs();
250  }
251
252  private void openScanner() {
253    incRegionCountMetrics(scanMetrics);
254    openScannerTries.set(1);
255    addListener(timelineConsistentRead(conn.getLocator(), tableName, scan, scan.getStartRow(),
256      getLocateType(scan), this::openScanner, rpcTimeoutNs, getPrimaryTimeoutNs(), retryTimer,
257      conn.getConnectionMetrics()), (resp, error) -> {
258        try (Scope ignored = span.makeCurrent()) {
259          if (error != null) {
260            try {
261              consumer.onError(error);
262              return;
263            } finally {
264              TraceUtil.setError(span, error);
265              span.end();
266            }
267          }
268          startScan(resp);
269        }
270      });
271  }
272
273  public void start() {
274    try (Scope ignored = span.makeCurrent()) {
275      openScanner();
276    }
277  }
278}