hbase-commits mailing list archives

Site index · List index
Message view « Date » · « Thread »
Top « Date » · « Thread »
From apurt...@apache.org
Subject [3/4] hbase git commit: HBASE-13262 Observe ScanResponse.moreResults in ClientScanner.
Date Sun, 29 Mar 2015 02:35:42 GMT
HBASE-13262 Observe ScanResponse.moreResults in ClientScanner.

The RS already returns to the client whether or not it has additional
results to be returned in a subsequent call to scan(), but the ClientScanner
did not use or adhere to this value. Subsequently, this can lead to
bugs around moving to the next region too early. A new method was added
to ClientScanner in the name of testability.

Encapsulate server-state into RegionServerCallable to avoid
modifying parameterization of callable impls.

Signed-off-by: Andrew Purtell <apurtell@apache.org>


Project: http://git-wip-us.apache.org/repos/asf/hbase/repo
Commit: http://git-wip-us.apache.org/repos/asf/hbase/commit/20189c25
Tree: http://git-wip-us.apache.org/repos/asf/hbase/tree/20189c25
Diff: http://git-wip-us.apache.org/repos/asf/hbase/diff/20189c25

Branch: refs/heads/branch-1.0
Commit: 20189c258cbcf109de10f76f06c059681e438615
Parents: d92fcdd
Author: Josh Elser <elserj@apache.org>
Authored: Sat Mar 28 18:56:51 2015 -0700
Committer: Andrew Purtell <apurtell@apache.org>
Committed: Sat Mar 28 18:56:51 2015 -0700

----------------------------------------------------------------------
 .../hadoop/hbase/client/ClientScanner.java      | 296 +++++------
 .../client/ClientSmallReversedScanner.java      |   1 +
 .../hadoop/hbase/client/ClientSmallScanner.java |  20 +-
 .../hadoop/hbase/client/ScannerCallable.java    |  44 +-
 .../client/ScannerCallableWithReplicas.java     |  16 +
 .../hadoop/hbase/client/TestClientScanner.java  | 489 +++++++++++++++++++
 .../hbase/protobuf/generated/ClientProtos.java  | 237 +++++++--
 hbase-protocol/src/main/protobuf/Client.proto   |   5 +
 .../hbase/regionserver/RSRpcServices.java       |  13 +-
 .../hadoop/hbase/client/TestSizeFailures.java   | 162 ++++++
 10 files changed, 1088 insertions(+), 195 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/hbase/blob/20189c25/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientScanner.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientScanner.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientScanner.java
index 4921de2..b111ba7 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientScanner.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientScanner.java
@@ -276,7 +276,8 @@ public class ClientScanner extends AbstractClientScanner {
       return true;
     }
 
-  static Result[] call(Scan scan, ScannerCallableWithReplicas callable,
+
+  Result[] call(Scan scan, ScannerCallableWithReplicas callable,
       RpcRetryingCaller<Result[]> caller, int scannerTimeout)
       throws IOException, RuntimeException {
     if (Thread.interrupted()) {
@@ -328,141 +329,7 @@ public class ClientScanner extends AbstractClientScanner {
         return null;
       }
       if (cache.size() == 0) {
-        Result [] values = null;
-        long remainingResultSize = maxScannerResultSize;
-        int countdown = this.caching;
-        // We need to reset it if it's a new callable that was created
-        // with a countdown in nextScanner
-        callable.setCaching(this.caching);
-        // This flag is set when we want to skip the result returned.  We do
-        // this when we reset scanner because it split under us.
-        boolean skipFirst = false;
-        boolean retryAfterOutOfOrderException  = true;
-        do {
-          try {
-            if (skipFirst) {
-              // Skip only the first row (which was the last row of the last
-              // already-processed batch).
-              callable.setCaching(1);
-              values = call(scan, callable, caller, scannerTimeout);
-              // When the replica switch happens, we need to do certain operations
-              // again. The scannercallable will openScanner with the right startkey
-              // but we need to pick up from there. Bypass the rest of the loop
-              // and let the catch-up happen in the beginning of the loop as it
-              // happens for the cases where we see exceptions. Since only openScanner
-              // would have happened, values would be null
-              if (values == null && callable.switchedToADifferentReplica()) {
-                if (this.lastResult != null) { //only skip if there was something read earlier
-                  skipFirst = true;
-                }
-                this.currentRegion = callable.getHRegionInfo();
-                continue;
-              }
-              callable.setCaching(this.caching);
-              skipFirst = false;
-            }
-            // Server returns a null values if scanning is to stop.  Else,
-            // returns an empty array if scanning is to go on and we've just
-            // exhausted current region.
-            values = call(scan, callable, caller, scannerTimeout);
-            if (skipFirst && values != null && values.length == 1) {
-              skipFirst = false; // Already skipped, unset it before scanning again
-              values = call(scan, callable, caller, scannerTimeout);
-            }
-            // When the replica switch happens, we need to do certain operations
-            // again. The callable will openScanner with the right startkey
-            // but we need to pick up from there. Bypass the rest of the loop
-            // and let the catch-up happen in the beginning of the loop as it
-            // happens for the cases where we see exceptions. Since only openScanner
-            // would have happened, values would be null
-            if (values == null && callable.switchedToADifferentReplica()) {
-              if (this.lastResult != null) { //only skip if there was something read earlier
-                skipFirst = true;
-              }
-              this.currentRegion = callable.getHRegionInfo();
-              continue;
-            }
-            retryAfterOutOfOrderException  = true;
-          } catch (DoNotRetryIOException e) {
-            // DNRIOEs are thrown to make us break out of retries.  Some types of DNRIOEs want us
-            // to reset the scanner and come back in again.
-            if (e instanceof UnknownScannerException) {
-              long timeout = lastNext + scannerTimeout;
-              // If we are over the timeout, throw this exception to the client wrapped in
-              // a ScannerTimeoutException. Else, it's because the region moved and we used the old
-              // id against the new region server; reset the scanner.
-              if (timeout < System.currentTimeMillis()) {
-                long elapsed = System.currentTimeMillis() - lastNext;
-                ScannerTimeoutException ex = new ScannerTimeoutException(
-                    elapsed + "ms passed since the last invocation, " +
-                        "timeout is currently set to " + scannerTimeout);
-                ex.initCause(e);
-                throw ex;
-              }
-            } else {
-              // If exception is any but the list below throw it back to the client; else setup
-              // the scanner and retry.
-              Throwable cause = e.getCause();
-              if ((cause != null && cause instanceof NotServingRegionException) ||
-                (cause != null && cause instanceof RegionServerStoppedException) ||
-                e instanceof OutOfOrderScannerNextException) {
-                // Pass
-                // It is easier writing the if loop test as list of what is allowed rather than
-                // as a list of what is not allowed... so if in here, it means we do not throw.
-              } else {
-                throw e;
-              }
-            }
-            // Else, its signal from depths of ScannerCallable that we need to reset the scanner.
-            if (this.lastResult != null) {
-              // The region has moved. We need to open a brand new scanner at
-              // the new location.
-              // Reset the startRow to the row we've seen last so that the new
-              // scanner starts at the correct row. Otherwise we may see previously
-              // returned rows again.
-              // (ScannerCallable by now has "relocated" the correct region)
-              this.scan.setStartRow(this.lastResult.getRow());
-
-              // Skip first row returned.  We already let it out on previous
-              // invocation.
-              skipFirst = true;
-            }
-            if (e instanceof OutOfOrderScannerNextException) {
-              if (retryAfterOutOfOrderException) {
-                retryAfterOutOfOrderException = false;
-              } else {
-                // TODO: Why wrap this in a DNRIOE when it already is a DNRIOE?
-                throw new DoNotRetryIOException("Failed after retry of " +
-                  "OutOfOrderScannerNextException: was there a rpc timeout?", e);
-              }
-            }
-            // Clear region.
-            this.currentRegion = null;
-            // Set this to zero so we don't try and do an rpc and close on remote server when
-            // the exception we got was UnknownScanner or the Server is going down.
-            callable = null;
-            // This continue will take us to while at end of loop where we will set up new scanner.
-            continue;
-          }
-          long currentTime = System.currentTimeMillis();
-          if (this.scanMetrics != null ) {
-            this.scanMetrics.sumOfMillisSecBetweenNexts.addAndGet(currentTime-lastNext);
-          }
-          lastNext = currentTime;
-          if (values != null && values.length > 0) {
-            for (Result rs : values) {
-              cache.add(rs);
-              // We don't make Iterator here
-              for (Cell cell : rs.rawCells()) {
-                remainingResultSize -= CellUtil.estimatedHeapSizeOf(cell);
-              }
-              countdown--;
-              this.lastResult = rs;
-            }
-          }
-          // Values == null means server-side filter has determined we must STOP
-        } while (remainingResultSize > 0 && countdown > 0 &&
-            possiblyNextScanner(countdown, values == null));
+        loadCache();
       }
 
       if (cache.size() > 0) {
@@ -474,6 +341,163 @@ public class ClientScanner extends AbstractClientScanner {
       return null;
     }
 
+  /**
+   * Contact the servers to load more {@link Result}s in the cache.
+   */
+  protected void loadCache() throws IOException {
+    Result[] values = null;
+    long remainingResultSize = maxScannerResultSize;
+    int countdown = this.caching;
+
+    // We need to reset it if it's a new callable that was created
+    // with a countdown in nextScanner
+    callable.setCaching(this.caching);
+    // This flag is set when we want to skip the result returned.  We do
+    // this when we reset scanner because it split under us.
+    boolean skipFirst = false;
+    boolean retryAfterOutOfOrderException = true;
+    // We don't expect that the server will have more results for us if
+    // it doesn't tell us otherwise. We rely on the size or count of results
+    boolean serverHasMoreResults = false;
+    do {
+      try {
+        if (skipFirst) {
+          // Skip only the first row (which was the last row of the last
+          // already-processed batch).
+          callable.setCaching(1);
+          values = call(scan, callable, caller, scannerTimeout);
+          // When the replica switch happens, we need to do certain operations
+          // again. The scannercallable will openScanner with the right startkey
+          // but we need to pick up from there. Bypass the rest of the loop
+          // and let the catch-up happen in the beginning of the loop as it
+          // happens for the cases where we see exceptions. Since only openScanner
+          // would have happened, values would be null
+          if (values == null && callable.switchedToADifferentReplica()) {
+            if (this.lastResult != null) { //only skip if there was something read earlier
+              skipFirst = true;
+            }
+            this.currentRegion = callable.getHRegionInfo();
+            continue;
+          }
+          callable.setCaching(this.caching);
+          skipFirst = false;
+        }
+        // Server returns a null values if scanning is to stop. Else,
+        // returns an empty array if scanning is to go on and we've just
+        // exhausted current region.
+        values = call(scan,   callable, caller, scannerTimeout);
+        if (skipFirst && values != null && values.length == 1) {
+          skipFirst = false; // Already skipped, unset it before scanning again
+          values = call(scan, callable, caller, scannerTimeout);
+        }
+        // When the replica switch happens, we need to do certain operations
+        // again. The callable will openScanner with the right startkey
+        // but we need to pick up from there. Bypass the rest of the loop
+        // and let the catch-up happen in the beginning of the loop as it
+        // happens for the cases where we see exceptions. Since only openScanner
+        // would have happened, values would be null
+        if (values == null && callable.switchedToADifferentReplica()) {
+          if (this.lastResult != null) { //only skip if there was something read earlier
+            skipFirst = true;
+          }
+          this.currentRegion = callable.getHRegionInfo();
+          continue;
+        }
+        retryAfterOutOfOrderException = true;
+      } catch (DoNotRetryIOException e) {
+        // DNRIOEs are thrown to make us break out of retries. Some types of DNRIOEs want us
+        // to reset the scanner and come back in again.
+        if (e instanceof UnknownScannerException) {
+          long timeout = lastNext + scannerTimeout;
+          // If we are over the timeout, throw this exception to the client wrapped in
+          // a ScannerTimeoutException. Else, it's because the region moved and we used the old
+          // id against the new region server; reset the scanner.
+          if (timeout < System.currentTimeMillis()) {
+            long elapsed = System.currentTimeMillis() - lastNext;
+            ScannerTimeoutException ex =
+                new ScannerTimeoutException(elapsed + "ms passed since the last invocation, "
+                    + "timeout is currently set to " + scannerTimeout);
+            ex.initCause(e);
+            throw ex;
+          }
+        } else {
+          // If exception is any but the list below throw it back to the client; else setup
+          // the scanner and retry.
+          Throwable cause = e.getCause();
+          if ((cause != null && cause instanceof NotServingRegionException) ||
+              (cause != null && cause instanceof RegionServerStoppedException) ||
+              e instanceof OutOfOrderScannerNextException) {
+            // Pass
+            // It is easier writing the if loop test as list of what is allowed rather than
+            // as a list of what is not allowed... so if in here, it means we do not throw.
+          } else {
+            throw e;
+          }
+        }
+        // Else, its signal from depths of ScannerCallable that we need to reset the scanner.
+        if (this.lastResult != null) {
+          // The region has moved. We need to open a brand new scanner at
+          // the new location.
+          // Reset the startRow to the row we've seen last so that the new
+          // scanner starts at the correct row. Otherwise we may see previously
+          // returned rows again.
+          // (ScannerCallable by now has "relocated" the correct region)
+          this.scan.setStartRow(this.lastResult.getRow());
+
+          // Skip first row returned.  We already let it out on previous
+          // invocation.
+          skipFirst = true;
+        }
+        if (e instanceof OutOfOrderScannerNextException) {
+          if (retryAfterOutOfOrderException) {
+            retryAfterOutOfOrderException = false;
+          } else {
+            // TODO: Why wrap this in a DNRIOE when it already is a DNRIOE?
+            throw new DoNotRetryIOException("Failed after retry of " +
+                "OutOfOrderScannerNextException: was there a rpc timeout?", e);
+          }
+        }
+        // Clear region.
+        this.currentRegion = null;
+        // Set this to zero so we don't try and do an rpc and close on remote server when
+        // the exception we got was UnknownScanner or the Server is going down.
+        callable = null;
+        // This continue will take us to while at end of loop where we will set up new scanner.
+        continue;
+      }
+      long currentTime = System.currentTimeMillis();
+      if (this.scanMetrics != null) {
+        this.scanMetrics.sumOfMillisSecBetweenNexts.addAndGet(currentTime - lastNext);
+      }
+      lastNext = currentTime;
+      if (values != null && values.length > 0) {
+        for (Result rs : values) {
+          cache.add(rs);
+          // We don't make Iterator here
+          for (Cell cell : rs.rawCells()) {
+            remainingResultSize -= CellUtil.estimatedHeapSizeOf(cell);
+          }
+          countdown--;
+          this.lastResult = rs;
+        }
+      }
+      // We expect that the server won't have more results for us when we exhaust
+      // the size (bytes or count) of the results returned. If the server *does* inform us that
+      // there are more results, we want to avoid possiblyNextScanner(...). Only when we actually
+      // get results is the moreResults context valid.
+      if (null != values && values.length > 0 && callable.hasMoreResultsContext()) {
+        // Only adhere to more server results when we don't have any partialResults
+        // as it keeps the outer loop logic the same.
+        serverHasMoreResults = callable.getServerHasMoreResults();
+      }
+      // Values == null means server-side filter has determined we must STOP
+      // !partialResults.isEmpty() means that we are still accumulating partial Results for a
+      // row. We should not change scanners before we receive all the partial Results for that
+      // row.
+    } while (remainingResultSize > 0 && countdown > 0 && !serverHasMoreResults
+        && possiblyNextScanner(countdown, values == null));
+  }
+
     @Override
     public void close() {
       if (!scanMetricsPublished) writeScanMetrics();

http://git-wip-us.apache.org/repos/asf/hbase/blob/20189c25/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientSmallReversedScanner.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientSmallReversedScanner.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientSmallReversedScanner.java
index 2cab830..1e94820 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientSmallReversedScanner.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientSmallReversedScanner.java
@@ -142,6 +142,7 @@ public class ClientSmallReversedScanner extends ReversedClientScanner {
         // exhausted current region.
         // callWithoutRetries is at this layer. Within the ScannerCallableWithReplicas,
         // we do a callWithRetries
+        // TODO use context from server
         values = this.caller.callWithoutRetries(smallScanCallable, scannerTimeout);
         this.currentRegion = smallScanCallable.getHRegionInfo();
         long currentTime = System.currentTimeMillis();

http://git-wip-us.apache.org/repos/asf/hbase/blob/20189c25/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientSmallScanner.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientSmallScanner.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientSmallScanner.java
index 478ba76..6d721e8 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientSmallScanner.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ClientSmallScanner.java
@@ -137,11 +137,11 @@ public class ClientSmallScanner extends ClientScanner {
   }
 
 
-  static ScannerCallableWithReplicas getSmallScanCallable(
-      ClusterConnection connection, TableName table, Scan scan,
-      ScanMetrics scanMetrics,  byte[] localStartKey, final int cacheNum,
-      RpcControllerFactory controllerFactory, ExecutorService pool, int primaryOperationTimeout,
-      int retries, int scannerTimeout, Configuration conf, RpcRetryingCaller<Result []> caller) {
+  static ScannerCallableWithReplicas getSmallScanCallable(ClusterConnection connection,
+      TableName table, Scan scan, ScanMetrics scanMetrics, byte[] localStartKey,
+      final int cacheNum, RpcControllerFactory controllerFactory, ExecutorService pool,
+      int primaryOperationTimeout, int retries, int scannerTimeout, Configuration conf,
+      RpcRetryingCaller<Result[]> caller) {
     scan.setStartRow(localStartKey);
     SmallScannerCallable s = new SmallScannerCallable(
       connection, table, scan, scanMetrics, controllerFactory, cacheNum, 0);
@@ -174,8 +174,15 @@ public class ClientSmallScanner extends ClientScanner {
         controller.setPriority(getTableName());
         controller.setCallTimeout(timeout);
         response = getStub().scan(controller, request);
-        return ResponseConverter.getResults(controller.cellScanner(),
+        Result[] results = ResponseConverter.getResults(controller.cellScanner(),
             response);
+        if (response.hasMoreResultsInRegion()) {
+          setHasMoreResultsContext(true);
+          setServerHasMoreResults(response.getMoreResultsInRegion());
+        } else {
+          setHasMoreResultsContext(false);
+        }
+        return results;
       } catch (ServiceException se) {
         throw ProtobufUtil.getRemoteException(se);
       }
@@ -208,6 +215,7 @@ public class ClientSmallScanner extends ClientScanner {
         // exhausted current region.
         // callWithoutRetries is at this layer. Within the ScannerCallableWithReplicas,
         // we do a callWithRetries
+        // TODO Use the server's response about more results
         values = this.caller.callWithoutRetries(smallScanCallable, scannerTimeout);
         this.currentRegion = smallScanCallable.getHRegionInfo();
         long currentTime = System.currentTimeMillis();

http://git-wip-us.apache.org/repos/asf/hbase/blob/20189c25/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ScannerCallable.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ScannerCallable.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ScannerCallable.java
index 5ecc363..9c5cacf 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ScannerCallable.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ScannerCallable.java
@@ -77,6 +77,8 @@ public class ScannerCallable extends RegionServerCallable<Result[]> {
   private int logCutOffLatency = 1000;
   private static String myAddress;
   protected final int id;
+  protected boolean serverHasMoreResultsContext;
+  protected boolean serverHasMoreResults;
   static {
     try {
       myAddress = DNS.getDefaultHost("default", "default");
@@ -173,7 +175,6 @@ public class ScannerCallable extends RegionServerCallable<Result[]> {
 
 
   @Override
-  @SuppressWarnings("deprecation")
   public Result [] call(int callTimeout) throws IOException {
     if (Thread.interrupted()) {
       throw new InterruptedIOException();
@@ -219,12 +220,23 @@ public class ScannerCallable extends RegionServerCallable<Result[]> {
                   + rows + " rows from scanner=" + scannerId);
               }
             }
-            if (response.hasMoreResults()
-                && !response.getMoreResults()) {
+            // moreResults is only used for the case where a filter exhausts all elements
+            if (response.hasMoreResults() && !response.getMoreResults()) {
               scannerId = -1L;
               closed = true;
+              // Implied that no results were returned back, either.
               return null;
             }
+            // moreResultsInRegion explicitly defines when a RS may choose to terminate a batch due
+            // to size or quantity of results in the response.
+            if (response.hasMoreResultsInRegion()) {
+              // Set what the RS said
+              setHasMoreResultsContext(true);
+              setServerHasMoreResults(response.getMoreResultsInRegion());
+            } else {
+              // Server didn't respond whether it has more results or not.
+              setHasMoreResultsContext(false);
+            }
           } catch (ServiceException se) {
             throw ProtobufUtil.getRemoteException(se);
           }
@@ -390,4 +402,30 @@ public class ScannerCallable extends RegionServerCallable<Result[]> {
     s.setCaching(this.caching);
     return s;
   }
+
+  /**
+   * Should the client attempt to fetch more results from this region
+   * @return True if the client should attempt to fetch more results, false otherwise.
+   */
+  protected boolean getServerHasMoreResults() {
+    assert serverHasMoreResultsContext;
+    return this.serverHasMoreResults;
+  }
+
+  protected void setServerHasMoreResults(boolean serverHasMoreResults) {
+    this.serverHasMoreResults = serverHasMoreResults;
+  }
+
+  /**
+   * Did the server respond with information about whether more results might exist.
+   * Not guaranteed to respond with older server versions
+   * @return True if the server responded with information about more results.
+   */
+  protected boolean hasMoreResultsContext() {
+    return serverHasMoreResultsContext;
+  }
+
+  protected void setHasMoreResultsContext(boolean serverHasMoreResultsContext) {
+    this.serverHasMoreResultsContext = serverHasMoreResultsContext;
+  }
 }

http://git-wip-us.apache.org/repos/asf/hbase/blob/20189c25/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ScannerCallableWithReplicas.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ScannerCallableWithReplicas.java b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ScannerCallableWithReplicas.java
index 440cddf..6c81197 100644
--- a/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ScannerCallableWithReplicas.java
+++ b/hbase-client/src/main/java/org/apache/hadoop/hbase/client/ScannerCallableWithReplicas.java
@@ -105,6 +105,22 @@ class ScannerCallableWithReplicas implements RetryingCallable<Result[]> {
     return currentScannerCallable.getHRegionInfo();
   }
 
+  public boolean getServerHasMoreResults() {
+    return currentScannerCallable.getServerHasMoreResults();
+  }
+
+  public void setServerHasMoreResults(boolean serverHasMoreResults) {
+    currentScannerCallable.setServerHasMoreResults(serverHasMoreResults);
+  }
+
+  public boolean hasMoreResultsContext() {
+    return currentScannerCallable.hasMoreResultsContext();
+  }
+
+  public void setHasMoreResultsContext(boolean serverHasMoreResultsContext) {
+    currentScannerCallable.setHasMoreResultsContext(serverHasMoreResultsContext);
+  }
+
   @Override
   public Result [] call(int timeout) throws IOException {
     // If the active replica callable was closed somewhere, invoke the RPC to

http://git-wip-us.apache.org/repos/asf/hbase/blob/20189c25/hbase-client/src/test/java/org/apache/hadoop/hbase/client/TestClientScanner.java
----------------------------------------------------------------------
diff --git a/hbase-client/src/test/java/org/apache/hadoop/hbase/client/TestClientScanner.java b/hbase-client/src/test/java/org/apache/hadoop/hbase/client/TestClientScanner.java
new file mode 100644
index 0000000..a91def3
--- /dev/null
+++ b/hbase-client/src/test/java/org/apache/hadoop/hbase/client/TestClientScanner.java
@@ -0,0 +1,489 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.hbase.client;
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertNotNull;
+import static org.junit.Assert.assertTrue;
+
+import java.io.IOException;
+import java.util.concurrent.ExecutorService;
+import java.util.concurrent.Executors;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.hbase.Cell;
+import org.apache.hadoop.hbase.CellScanner;
+import org.apache.hadoop.hbase.KeyValue;
+import org.apache.hadoop.hbase.KeyValue.Type;
+import org.apache.hadoop.hbase.TableName;
+import org.apache.hadoop.hbase.ipc.RpcControllerFactory;
+import org.apache.hadoop.hbase.testclassification.SmallTests;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+import org.mockito.InOrder;
+import org.mockito.Mockito;
+import org.mockito.invocation.InvocationOnMock;
+import org.mockito.stubbing.Answer;
+
+/**
+ * Test the ClientScanner.
+ */
+@Category(SmallTests.class)
+public class TestClientScanner {
+
+  Scan scan;
+  ExecutorService pool;
+  Configuration conf;
+
+  ClusterConnection clusterConn;
+  RpcRetryingCallerFactory rpcFactory;
+  RpcControllerFactory controllerFactory;
+
+  @Before
+  @SuppressWarnings("deprecation")
+  public void setup() throws IOException {
+    clusterConn = Mockito.mock(ClusterConnection.class);
+    rpcFactory = Mockito.mock(RpcRetryingCallerFactory.class);
+    controllerFactory = Mockito.mock(RpcControllerFactory.class);
+    pool = Executors.newSingleThreadExecutor();
+    scan = new Scan();
+    conf = new Configuration();
+    Mockito.when(clusterConn.getConfiguration()).thenReturn(conf);
+  }
+
+  @After
+  public void teardown() {
+    if (null != pool) {
+      pool.shutdownNow();
+    }
+  }
+
+  private static class MockClientScanner extends ClientScanner {
+
+    private boolean rpcFinished = false;
+    private boolean rpcFinishedFired = false;
+
+    public MockClientScanner(final Configuration conf, final Scan scan, final TableName tableName,
+        ClusterConnection connection, RpcRetryingCallerFactory rpcFactory,
+        RpcControllerFactory controllerFactory, ExecutorService pool, int primaryOperationTimeout)
+        throws IOException {
+      super(conf, scan, tableName, connection, rpcFactory, controllerFactory, pool,
+          primaryOperationTimeout);
+    }
+
+    @Override
+    protected boolean nextScanner(int nbRows, final boolean done) throws IOException {
+      if (!rpcFinished) {
+        return super.nextScanner(nbRows, done);
+      }
+
+      // Enforce that we don't short-circuit more than once
+      if (rpcFinishedFired) {
+        throw new RuntimeException("Expected nextScanner to only be called once after " +
+            " short-circuit was triggered.");
+      }
+      rpcFinishedFired = true;
+      return false;
+    }
+
+    @Override
+    protected ScannerCallableWithReplicas getScannerCallable(byte [] localStartKey,
+        int nbRows) {
+      scan.setStartRow(localStartKey);
+      ScannerCallable s =
+          new ScannerCallable(getConnection(), getTable(), scan, this.scanMetrics,
+              this.rpcControllerFactory);
+      s.setCaching(nbRows);
+      ScannerCallableWithReplicas sr = new ScannerCallableWithReplicas(getTable(), getConnection(),
+       s, pool, primaryOperationTimeout, scan,
+       getRetries(), scannerTimeout, caching, conf, caller);
+      return sr;
+    }
+
+    public void setRpcFinished(boolean rpcFinished) {
+      this.rpcFinished = rpcFinished;
+    }
+  }
+
+  @Test
+  @SuppressWarnings("unchecked")
+  public void testNoResultsHint() throws IOException {
+    final Result[] results = new Result[1];
+    KeyValue kv1 = new KeyValue("row".getBytes(), "cf".getBytes(), "cq".getBytes(), 1,
+        Type.Maximum);
+    results[0] = Result.create(new Cell[] {kv1});
+
+    RpcRetryingCaller<Result[]> caller = Mockito.mock(RpcRetryingCaller.class);
+
+    Mockito.when(rpcFactory.<Result[]> newCaller()).thenReturn(caller);
+    Mockito.when(caller.callWithoutRetries(Mockito.any(RetryingCallable.class),
+      Mockito.anyInt())).thenAnswer(new Answer<Result[]>() {
+        private int count = 0;
+        @Override
+        public Result[] answer(InvocationOnMock invocation) throws Throwable {
+            ScannerCallableWithReplicas callable = invocation.getArgumentAt(0,
+                ScannerCallableWithReplicas.class);
+          switch (count) {
+            case 0: // initialize
+            case 2: // close
+              count++;
+              return null;
+            case 1:
+              count++;
+              callable.setHasMoreResultsContext(false);
+              return results;
+            default:
+              throw new RuntimeException("Expected only 2 invocations");
+          }
+        }
+    });
+
+    // Set a much larger cache and buffer size than we'll provide
+    scan.setCaching(100);
+    scan.setMaxResultSize(1000*1000);
+
+    try (MockClientScanner scanner = new MockClientScanner(conf, scan, TableName.valueOf("table"),
+        clusterConn, rpcFactory, controllerFactory, pool, Integer.MAX_VALUE)) {
+
+      scanner.setRpcFinished(true);
+
+      InOrder inOrder = Mockito.inOrder(caller);
+
+      scanner.loadCache();
+
+      // One more call due to initializeScannerInConstruction()
+      inOrder.verify(caller, Mockito.times(2)).callWithoutRetries(
+          Mockito.any(RetryingCallable.class), Mockito.anyInt());
+
+      assertEquals(1, scanner.cache.size());
+      Result r = scanner.cache.poll();
+      assertNotNull(r);
+      CellScanner cs = r.cellScanner();
+      assertTrue(cs.advance());
+      assertEquals(kv1, cs.current());
+      assertFalse(cs.advance());
+    }
+  }
+
+  @Test
+  @SuppressWarnings("unchecked")
+  public void testSizeLimit() throws IOException {
+    final Result[] results = new Result[1];
+    KeyValue kv1 = new KeyValue("row".getBytes(), "cf".getBytes(), "cq".getBytes(), 1,
+        Type.Maximum);
+    results[0] = Result.create(new Cell[] {kv1});
+
+    RpcRetryingCaller<Result[]> caller = Mockito.mock(RpcRetryingCaller.class);
+
+    Mockito.when(rpcFactory.<Result[]> newCaller()).thenReturn(caller);
+    Mockito.when(caller.callWithoutRetries(Mockito.any(RetryingCallable.class),
+      Mockito.anyInt())).thenAnswer(new Answer<Result[]>() {
+        private int count = 0;
+        @Override
+        public Result[] answer(InvocationOnMock invocation) throws Throwable {
+          ScannerCallableWithReplicas callable = invocation.getArgumentAt(0,
+              ScannerCallableWithReplicas.class);
+          switch (count) {
+            case 0: // initialize
+            case 2: // close
+              count++;
+              return null;
+            case 1:
+              count++;
+              callable.setHasMoreResultsContext(true);
+              callable.setServerHasMoreResults(false);
+              return results;
+            default:
+              throw new RuntimeException("Expected only 2 invocations");
+          }
+        }
+    });
+
+    Mockito.when(rpcFactory.<Result[]> newCaller()).thenReturn(caller);
+
+    // Set a much larger cache
+    scan.setCaching(100);
+    // The single key-value will exit the loop
+    scan.setMaxResultSize(1);
+
+    try (MockClientScanner scanner = new MockClientScanner(conf, scan, TableName.valueOf("table"),
+        clusterConn, rpcFactory, controllerFactory, pool, Integer.MAX_VALUE)) {
+
+      // Due to initializeScannerInConstruction()
+      Mockito.verify(caller).callWithoutRetries(Mockito.any(RetryingCallable.class),
+          Mockito.anyInt());
+
+      InOrder inOrder = Mockito.inOrder(caller);
+
+      scanner.loadCache();
+
+      inOrder.verify(caller, Mockito.times(2)).callWithoutRetries(
+          Mockito.any(RetryingCallable.class), Mockito.anyInt());
+
+      assertEquals(1, scanner.cache.size());
+      Result r = scanner.cache.poll();
+      assertNotNull(r);
+      CellScanner cs = r.cellScanner();
+      assertTrue(cs.advance());
+      assertEquals(kv1, cs.current());
+      assertFalse(cs.advance());
+    }
+  }
+
+  @Test
+  @SuppressWarnings("unchecked")
+  public void testCacheLimit() throws IOException {
+    KeyValue kv1 = new KeyValue("row1".getBytes(), "cf".getBytes(), "cq".getBytes(), 1,
+        Type.Maximum), kv2 = new KeyValue("row2".getBytes(), "cf".getBytes(), "cq".getBytes(), 1,
+        Type.Maximum), kv3 = new KeyValue("row3".getBytes(), "cf".getBytes(), "cq".getBytes(), 1,
+        Type.Maximum);
+    final Result[] results = new Result[] {Result.create(new Cell[] {kv1}),
+        Result.create(new Cell[] {kv2}), Result.create(new Cell[] {kv3})};
+
+    RpcRetryingCaller<Result[]> caller = Mockito.mock(RpcRetryingCaller.class);
+
+    Mockito.when(rpcFactory.<Result[]> newCaller()).thenReturn(caller);
+    Mockito.when(caller.callWithoutRetries(Mockito.any(RetryingCallable.class),
+      Mockito.anyInt())).thenAnswer(new Answer<Result[]>() {
+        private int count = 0;
+        @Override
+        public Result[] answer(InvocationOnMock invocation) throws Throwable {
+          ScannerCallableWithReplicas callable = invocation.getArgumentAt(0,
+              ScannerCallableWithReplicas.class);
+          switch (count) {
+            case 0: // initialize
+            case 2: // close
+              count++;
+              return null;
+            case 1:
+              count++;
+              callable.setHasMoreResultsContext(true);
+              callable.setServerHasMoreResults(false);
+              return results;
+            default:
+              throw new RuntimeException("Expected only 2 invocations");
+          }
+        }
+    });
+
+    Mockito.when(rpcFactory.<Result[]> newCaller()).thenReturn(caller);
+
+    // Set a small cache
+    scan.setCaching(1);
+    // Set a very large size
+    scan.setMaxResultSize(1000*1000);
+
+    try (MockClientScanner scanner = new MockClientScanner(conf, scan, TableName.valueOf("table"),
+        clusterConn, rpcFactory, controllerFactory, pool, Integer.MAX_VALUE)) {
+
+      // Due to initializeScannerInConstruction()
+      Mockito.verify(caller).callWithoutRetries(Mockito.any(RetryingCallable.class),
+          Mockito.anyInt());
+
+      InOrder inOrder = Mockito.inOrder(caller);
+
+      scanner.loadCache();
+
+      // Ensures that possiblyNextScanner isn't called at the end which would trigger
+      // another call to callWithoutRetries
+      inOrder.verify(caller, Mockito.times(2)).callWithoutRetries(
+          Mockito.any(RetryingCallable.class), Mockito.anyInt());
+
+      assertEquals(3, scanner.cache.size());
+      Result r = scanner.cache.poll();
+      assertNotNull(r);
+      CellScanner cs = r.cellScanner();
+      assertTrue(cs.advance());
+      assertEquals(kv1, cs.current());
+      assertFalse(cs.advance());
+
+      r = scanner.cache.poll();
+      assertNotNull(r);
+      cs = r.cellScanner();
+      assertTrue(cs.advance());
+      assertEquals(kv2, cs.current());
+      assertFalse(cs.advance());
+
+      r = scanner.cache.poll();
+      assertNotNull(r);
+      cs = r.cellScanner();
+      assertTrue(cs.advance());
+      assertEquals(kv3, cs.current());
+      assertFalse(cs.advance());
+    }
+  }
+
+  @Test
+  @SuppressWarnings("unchecked")
+  public void testNoMoreResults() throws IOException {
+    final Result[] results = new Result[1];
+    KeyValue kv1 = new KeyValue("row".getBytes(), "cf".getBytes(), "cq".getBytes(), 1,
+        Type.Maximum);
+    results[0] = Result.create(new Cell[] {kv1});
+
+    RpcRetryingCaller<Result[]> caller = Mockito.mock(RpcRetryingCaller.class);
+
+    Mockito.when(rpcFactory.<Result[]> newCaller()).thenReturn(caller);
+    Mockito.when(caller.callWithoutRetries(Mockito.any(RetryingCallable.class),
+      Mockito.anyInt())).thenAnswer(new Answer<Result[]>() {
+        private int count = 0;
+        @Override
+        public Result[] answer(InvocationOnMock invocation) throws Throwable {
+          ScannerCallableWithReplicas callable = invocation.getArgumentAt(0,
+              ScannerCallableWithReplicas.class);
+          switch (count) {
+            case 0: // initialize
+            case 2: // close
+              count++;
+              return null;
+            case 1:
+              count++;
+              callable.setHasMoreResultsContext(true);
+              callable.setServerHasMoreResults(false);
+              return results;
+            default:
+              throw new RuntimeException("Expected only 2 invocations");
+          }
+        }
+    });
+
+    Mockito.when(rpcFactory.<Result[]> newCaller()).thenReturn(caller);
+
+    // Set a much larger cache and buffer size than we'll provide
+    scan.setCaching(100);
+    scan.setMaxResultSize(1000*1000);
+
+    try (MockClientScanner scanner = new MockClientScanner(conf, scan, TableName.valueOf("table"),
+        clusterConn, rpcFactory, controllerFactory, pool, Integer.MAX_VALUE)) {
+
+      // Due to initializeScannerInConstruction()
+      Mockito.verify(caller).callWithoutRetries(Mockito.any(RetryingCallable.class),
+          Mockito.anyInt());
+
+      scanner.setRpcFinished(true);
+
+      InOrder inOrder = Mockito.inOrder(caller);
+
+      scanner.loadCache();
+
+      inOrder.verify(caller, Mockito.times(2)).callWithoutRetries(
+          Mockito.any(RetryingCallable.class), Mockito.anyInt());
+
+      assertEquals(1, scanner.cache.size());
+      Result r = scanner.cache.poll();
+      assertNotNull(r);
+      CellScanner cs = r.cellScanner();
+      assertTrue(cs.advance());
+      assertEquals(kv1, cs.current());
+      assertFalse(cs.advance());
+    }
+  }
+
+  @Test
+  @SuppressWarnings("unchecked")
+  public void testMoreResults() throws IOException {
+    final Result[] results1 = new Result[1];
+    KeyValue kv1 = new KeyValue("row".getBytes(), "cf".getBytes(), "cq".getBytes(), 1,
+        Type.Maximum);
+    results1[0] = Result.create(new Cell[] {kv1});
+
+    final Result[] results2 = new Result[1];
+    KeyValue kv2 = new KeyValue("row2".getBytes(), "cf".getBytes(), "cq".getBytes(), 1,
+        Type.Maximum);
+    results2[0] = Result.create(new Cell[] {kv2});
+
+
+    RpcRetryingCaller<Result[]> caller = Mockito.mock(RpcRetryingCaller.class);
+
+    Mockito.when(rpcFactory.<Result[]> newCaller()).thenReturn(caller);
+    Mockito.when(caller.callWithoutRetries(Mockito.any(RetryingCallable.class),
+        Mockito.anyInt())).thenAnswer(new Answer<Result[]>() {
+          private int count = 0;
+          @Override
+          public Result[] answer(InvocationOnMock invocation) throws Throwable {
+            ScannerCallableWithReplicas callable = invocation.getArgumentAt(0,
+                ScannerCallableWithReplicas.class);
+            switch (count) {
+              case 0: // initialize
+              case 3: // close
+                count++;
+                return null;
+              case 1:
+                count++;
+                callable.setHasMoreResultsContext(true);
+                callable.setServerHasMoreResults(true);
+                return results1;
+              case 2:
+                count++;
+                // The server reports back false WRT more results
+                callable.setHasMoreResultsContext(true);
+                callable.setServerHasMoreResults(false);
+                return results2;
+              default:
+                throw new RuntimeException("Expected only 2 invocations");
+            }
+          }
+      });
+
+    // Set a much larger cache and buffer size than we'll provide
+    scan.setCaching(100);
+    scan.setMaxResultSize(1000*1000);
+
+    try (MockClientScanner scanner = new MockClientScanner(conf, scan, TableName.valueOf("table"),
+        clusterConn, rpcFactory, controllerFactory, pool, Integer.MAX_VALUE)) {
+
+      // Due to initializeScannerInConstruction()
+      Mockito.verify(caller).callWithoutRetries(Mockito.any(RetryingCallable.class),
+          Mockito.anyInt());
+
+      InOrder inOrder = Mockito.inOrder(caller);
+
+      scanner.loadCache();
+
+      inOrder.verify(caller, Mockito.times(2)).callWithoutRetries(
+          Mockito.any(RetryingCallable.class), Mockito.anyInt());
+
+      assertEquals(1, scanner.cache.size());
+      Result r = scanner.cache.poll();
+      assertNotNull(r);
+      CellScanner cs = r.cellScanner();
+      assertTrue(cs.advance());
+      assertEquals(kv1, cs.current());
+      assertFalse(cs.advance());
+
+      scanner.setRpcFinished(true);
+
+      inOrder = Mockito.inOrder(caller);
+
+      scanner.loadCache();
+
+      inOrder.verify(caller, Mockito.times(3)).callWithoutRetries(
+          Mockito.any(RetryingCallable.class), Mockito.anyInt());
+
+      r = scanner.cache.poll();
+      assertNotNull(r);
+      cs = r.cellScanner();
+      assertTrue(cs.advance());
+      assertEquals(kv2, cs.current());
+      assertFalse(cs.advance());
+    }
+  }
+}

http://git-wip-us.apache.org/repos/asf/hbase/blob/20189c25/hbase-protocol/src/main/java/org/apache/hadoop/hbase/protobuf/generated/ClientProtos.java
----------------------------------------------------------------------
diff --git a/hbase-protocol/src/main/java/org/apache/hadoop/hbase/protobuf/generated/ClientProtos.java b/hbase-protocol/src/main/java/org/apache/hadoop/hbase/protobuf/generated/ClientProtos.java
index afd67a1..61088e7 100644
--- a/hbase-protocol/src/main/java/org/apache/hadoop/hbase/protobuf/generated/ClientProtos.java
+++ b/hbase-protocol/src/main/java/org/apache/hadoop/hbase/protobuf/generated/ClientProtos.java
@@ -17504,6 +17504,28 @@ public final class ClientProtos {
      * <code>optional bool stale = 6;</code>
      */
     boolean getStale();
+
+    // optional bool more_results_in_region = 8;
+    /**
+     * <code>optional bool more_results_in_region = 8;</code>
+     *
+     * <pre>
+     * A server may choose to limit the number of results returned to the client for
+     * reasons such as the size in bytes or quantity of results accumulated. This field
+     * will true when more results exist in the current region.
+     * </pre>
+     */
+    boolean hasMoreResultsInRegion();
+    /**
+     * <code>optional bool more_results_in_region = 8;</code>
+     *
+     * <pre>
+     * A server may choose to limit the number of results returned to the client for
+     * reasons such as the size in bytes or quantity of results accumulated. This field
+     * will true when more results exist in the current region.
+     * </pre>
+     */
+    boolean getMoreResultsInRegion();
   }
   /**
    * Protobuf type {@code ScanResponse}
@@ -17611,6 +17633,11 @@ public final class ClientProtos {
               stale_ = input.readBool();
               break;
             }
+            case 64: {
+              bitField0_ |= 0x00000010;
+              moreResultsInRegion_ = input.readBool();
+              break;
+            }
           }
         }
       } catch (com.google.protobuf.InvalidProtocolBufferException e) {
@@ -17840,6 +17867,34 @@ public final class ClientProtos {
       return stale_;
     }
 
+    // optional bool more_results_in_region = 8;
+    public static final int MORE_RESULTS_IN_REGION_FIELD_NUMBER = 8;
+    private boolean moreResultsInRegion_;
+    /**
+     * <code>optional bool more_results_in_region = 8;</code>
+     *
+     * <pre>
+     * A server may choose to limit the number of results returned to the client for
+     * reasons such as the size in bytes or quantity of results accumulated. This field
+     * will true when more results exist in the current region.
+     * </pre>
+     */
+    public boolean hasMoreResultsInRegion() {
+      return ((bitField0_ & 0x00000010) == 0x00000010);
+    }
+    /**
+     * <code>optional bool more_results_in_region = 8;</code>
+     *
+     * <pre>
+     * A server may choose to limit the number of results returned to the client for
+     * reasons such as the size in bytes or quantity of results accumulated. This field
+     * will true when more results exist in the current region.
+     * </pre>
+     */
+    public boolean getMoreResultsInRegion() {
+      return moreResultsInRegion_;
+    }
+
     private void initFields() {
       cellsPerResult_ = java.util.Collections.emptyList();
       scannerId_ = 0L;
@@ -17847,6 +17902,7 @@ public final class ClientProtos {
       ttl_ = 0;
       results_ = java.util.Collections.emptyList();
       stale_ = false;
+      moreResultsInRegion_ = false;
     }
     private byte memoizedIsInitialized = -1;
     public final boolean isInitialized() {
@@ -17878,6 +17934,9 @@ public final class ClientProtos {
       if (((bitField0_ & 0x00000008) == 0x00000008)) {
         output.writeBool(6, stale_);
       }
+      if (((bitField0_ & 0x00000010) == 0x00000010)) {
+        output.writeBool(8, moreResultsInRegion_);
+      }
       getUnknownFields().writeTo(output);
     }
 
@@ -17916,6 +17975,10 @@ public final class ClientProtos {
         size += com.google.protobuf.CodedOutputStream
           .computeBoolSize(6, stale_);
       }
+      if (((bitField0_ & 0x00000010) == 0x00000010)) {
+        size += com.google.protobuf.CodedOutputStream
+          .computeBoolSize(8, moreResultsInRegion_);
+      }
       size += getUnknownFields().getSerializedSize();
       memoizedSerializedSize = size;
       return size;
@@ -17963,6 +18026,11 @@ public final class ClientProtos {
         result = result && (getStale()
             == other.getStale());
       }
+      result = result && (hasMoreResultsInRegion() == other.hasMoreResultsInRegion());
+      if (hasMoreResultsInRegion()) {
+        result = result && (getMoreResultsInRegion()
+            == other.getMoreResultsInRegion());
+      }
       result = result &&
           getUnknownFields().equals(other.getUnknownFields());
       return result;
@@ -18000,6 +18068,10 @@ public final class ClientProtos {
         hash = (37 * hash) + STALE_FIELD_NUMBER;
         hash = (53 * hash) + hashBoolean(getStale());
       }
+      if (hasMoreResultsInRegion()) {
+        hash = (37 * hash) + MORE_RESULTS_IN_REGION_FIELD_NUMBER;
+        hash = (53 * hash) + hashBoolean(getMoreResultsInRegion());
+      }
       hash = (29 * hash) + getUnknownFields().hashCode();
       memoizedHashCode = hash;
       return hash;
@@ -18132,6 +18204,8 @@ public final class ClientProtos {
         }
         stale_ = false;
         bitField0_ = (bitField0_ & ~0x00000020);
+        moreResultsInRegion_ = false;
+        bitField0_ = (bitField0_ & ~0x00000040);
         return this;
       }
 
@@ -18190,6 +18264,10 @@ public final class ClientProtos {
           to_bitField0_ |= 0x00000008;
         }
         result.stale_ = stale_;
+        if (((from_bitField0_ & 0x00000040) == 0x00000040)) {
+          to_bitField0_ |= 0x00000010;
+        }
+        result.moreResultsInRegion_ = moreResultsInRegion_;
         result.bitField0_ = to_bitField0_;
         onBuilt();
         return result;
@@ -18254,6 +18332,9 @@ public final class ClientProtos {
         if (other.hasStale()) {
           setStale(other.getStale());
         }
+        if (other.hasMoreResultsInRegion()) {
+          setMoreResultsInRegion(other.getMoreResultsInRegion());
+        }
         this.mergeUnknownFields(other.getUnknownFields());
         return this;
       }
@@ -18897,6 +18978,63 @@ public final class ClientProtos {
         return this;
       }
 
+      // optional bool more_results_in_region = 8;
+      private boolean moreResultsInRegion_ ;
+      /**
+       * <code>optional bool more_results_in_region = 8;</code>
+       *
+       * <pre>
+       * A server may choose to limit the number of results returned to the client for
+       * reasons such as the size in bytes or quantity of results accumulated. This field
+       * will true when more results exist in the current region.
+       * </pre>
+       */
+      public boolean hasMoreResultsInRegion() {
+        return ((bitField0_ & 0x00000040) == 0x00000040);
+      }
+      /**
+       * <code>optional bool more_results_in_region = 8;</code>
+       *
+       * <pre>
+       * A server may choose to limit the number of results returned to the client for
+       * reasons such as the size in bytes or quantity of results accumulated. This field
+       * will true when more results exist in the current region.
+       * </pre>
+       */
+      public boolean getMoreResultsInRegion() {
+        return moreResultsInRegion_;
+      }
+      /**
+       * <code>optional bool more_results_in_region = 8;</code>
+       *
+       * <pre>
+       * A server may choose to limit the number of results returned to the client for
+       * reasons such as the size in bytes or quantity of results accumulated. This field
+       * will true when more results exist in the current region.
+       * </pre>
+       */
+      public Builder setMoreResultsInRegion(boolean value) {
+        bitField0_ |= 0x00000040;
+        moreResultsInRegion_ = value;
+        onChanged();
+        return this;
+      }
+      /**
+       * <code>optional bool more_results_in_region = 8;</code>
+       *
+       * <pre>
+       * A server may choose to limit the number of results returned to the client for
+       * reasons such as the size in bytes or quantity of results accumulated. This field
+       * will true when more results exist in the current region.
+       * </pre>
+       */
+      public Builder clearMoreResultsInRegion() {
+        bitField0_ = (bitField0_ & ~0x00000040);
+        moreResultsInRegion_ = false;
+        onChanged();
+        return this;
+      }
+
       // @@protoc_insertion_point(builder_scope:ScanResponse)
     }
 
@@ -32029,56 +32167,57 @@ public final class ClientProtos {
       "on\030\001 \001(\0132\020.RegionSpecifier\022\023\n\004scan\030\002 \001(\013",
       "2\005.Scan\022\022\n\nscanner_id\030\003 \001(\004\022\026\n\016number_of" +
       "_rows\030\004 \001(\r\022\025\n\rclose_scanner\030\005 \001(\010\022\025\n\rne" +
-      "xt_call_seq\030\006 \001(\004\"\210\001\n\014ScanResponse\022\030\n\020ce" +
+      "xt_call_seq\030\006 \001(\004\"\250\001\n\014ScanResponse\022\030\n\020ce" +
       "lls_per_result\030\001 \003(\r\022\022\n\nscanner_id\030\002 \001(\004" +
       "\022\024\n\014more_results\030\003 \001(\010\022\013\n\003ttl\030\004 \001(\r\022\030\n\007r" +
-      "esults\030\005 \003(\0132\007.Result\022\r\n\005stale\030\006 \001(\010\"\263\001\n" +
-      "\024BulkLoadHFileRequest\022 \n\006region\030\001 \002(\0132\020." +
-      "RegionSpecifier\0225\n\013family_path\030\002 \003(\0132 .B" +
-      "ulkLoadHFileRequest.FamilyPath\022\026\n\016assign" +
-      "_seq_num\030\003 \001(\010\032*\n\nFamilyPath\022\016\n\006family\030\001",
-      " \002(\014\022\014\n\004path\030\002 \002(\t\"\'\n\025BulkLoadHFileRespo" +
-      "nse\022\016\n\006loaded\030\001 \002(\010\"a\n\026CoprocessorServic" +
-      "eCall\022\013\n\003row\030\001 \002(\014\022\024\n\014service_name\030\002 \002(\t" +
-      "\022\023\n\013method_name\030\003 \002(\t\022\017\n\007request\030\004 \002(\014\"9" +
-      "\n\030CoprocessorServiceResult\022\035\n\005value\030\001 \001(" +
-      "\0132\016.NameBytesPair\"d\n\031CoprocessorServiceR" +
-      "equest\022 \n\006region\030\001 \002(\0132\020.RegionSpecifier" +
-      "\022%\n\004call\030\002 \002(\0132\027.CoprocessorServiceCall\"" +
-      "]\n\032CoprocessorServiceResponse\022 \n\006region\030" +
-      "\001 \002(\0132\020.RegionSpecifier\022\035\n\005value\030\002 \002(\0132\016",
-      ".NameBytesPair\"{\n\006Action\022\r\n\005index\030\001 \001(\r\022" +
-      " \n\010mutation\030\002 \001(\0132\016.MutationProto\022\021\n\003get" +
-      "\030\003 \001(\0132\004.Get\022-\n\014service_call\030\004 \001(\0132\027.Cop" +
-      "rocessorServiceCall\"Y\n\014RegionAction\022 \n\006r" +
-      "egion\030\001 \002(\0132\020.RegionSpecifier\022\016\n\006atomic\030" +
-      "\002 \001(\010\022\027\n\006action\030\003 \003(\0132\007.Action\"D\n\017Region" +
-      "LoadStats\022\027\n\014memstoreLoad\030\001 \001(\005:\0010\022\030\n\rhe" +
-      "apOccupancy\030\002 \001(\005:\0010\"\266\001\n\021ResultOrExcepti" +
-      "on\022\r\n\005index\030\001 \001(\r\022\027\n\006result\030\002 \001(\0132\007.Resu" +
-      "lt\022!\n\texception\030\003 \001(\0132\016.NameBytesPair\0221\n",
-      "\016service_result\030\004 \001(\0132\031.CoprocessorServi" +
-      "ceResult\022#\n\tloadStats\030\005 \001(\0132\020.RegionLoad" +
-      "Stats\"f\n\022RegionActionResult\022-\n\021resultOrE" +
-      "xception\030\001 \003(\0132\022.ResultOrException\022!\n\tex" +
-      "ception\030\002 \001(\0132\016.NameBytesPair\"f\n\014MultiRe" +
-      "quest\022#\n\014regionAction\030\001 \003(\0132\r.RegionActi" +
-      "on\022\022\n\nnonceGroup\030\002 \001(\004\022\035\n\tcondition\030\003 \001(" +
-      "\0132\n.Condition\"S\n\rMultiResponse\022/\n\022region" +
-      "ActionResult\030\001 \003(\0132\023.RegionActionResult\022" +
-      "\021\n\tprocessed\030\002 \001(\010*\'\n\013Consistency\022\n\n\006STR",
-      "ONG\020\000\022\014\n\010TIMELINE\020\0012\205\003\n\rClientService\022 \n" +
-      "\003Get\022\013.GetRequest\032\014.GetResponse\022)\n\006Mutat" +
-      "e\022\016.MutateRequest\032\017.MutateResponse\022#\n\004Sc" +
-      "an\022\014.ScanRequest\032\r.ScanResponse\022>\n\rBulkL" +
-      "oadHFile\022\025.BulkLoadHFileRequest\032\026.BulkLo" +
-      "adHFileResponse\022F\n\013ExecService\022\032.Coproce" +
-      "ssorServiceRequest\032\033.CoprocessorServiceR" +
-      "esponse\022R\n\027ExecRegionServerService\022\032.Cop" +
-      "rocessorServiceRequest\032\033.CoprocessorServ" +
-      "iceResponse\022&\n\005Multi\022\r.MultiRequest\032\016.Mu",
-      "ltiResponseBB\n*org.apache.hadoop.hbase.p" +
-      "rotobuf.generatedB\014ClientProtosH\001\210\001\001\240\001\001"
+      "esults\030\005 \003(\0132\007.Result\022\r\n\005stale\030\006 \001(\010\022\036\n\026" +
+      "more_results_in_region\030\010 \001(\010\"\263\001\n\024BulkLoa" +
+      "dHFileRequest\022 \n\006region\030\001 \002(\0132\020.RegionSp" +
+      "ecifier\0225\n\013family_path\030\002 \003(\0132 .BulkLoadH" +
+      "FileRequest.FamilyPath\022\026\n\016assign_seq_num",
+      "\030\003 \001(\010\032*\n\nFamilyPath\022\016\n\006family\030\001 \002(\014\022\014\n\004" +
+      "path\030\002 \002(\t\"\'\n\025BulkLoadHFileResponse\022\016\n\006l" +
+      "oaded\030\001 \002(\010\"a\n\026CoprocessorServiceCall\022\013\n" +
+      "\003row\030\001 \002(\014\022\024\n\014service_name\030\002 \002(\t\022\023\n\013meth" +
+      "od_name\030\003 \002(\t\022\017\n\007request\030\004 \002(\014\"9\n\030Coproc" +
+      "essorServiceResult\022\035\n\005value\030\001 \001(\0132\016.Name" +
+      "BytesPair\"d\n\031CoprocessorServiceRequest\022 " +
+      "\n\006region\030\001 \002(\0132\020.RegionSpecifier\022%\n\004call" +
+      "\030\002 \002(\0132\027.CoprocessorServiceCall\"]\n\032Copro" +
+      "cessorServiceResponse\022 \n\006region\030\001 \002(\0132\020.",
+      "RegionSpecifier\022\035\n\005value\030\002 \002(\0132\016.NameByt" +
+      "esPair\"{\n\006Action\022\r\n\005index\030\001 \001(\r\022 \n\010mutat" +
+      "ion\030\002 \001(\0132\016.MutationProto\022\021\n\003get\030\003 \001(\0132\004" +
+      ".Get\022-\n\014service_call\030\004 \001(\0132\027.Coprocessor" +
+      "ServiceCall\"Y\n\014RegionAction\022 \n\006region\030\001 " +
+      "\002(\0132\020.RegionSpecifier\022\016\n\006atomic\030\002 \001(\010\022\027\n" +
+      "\006action\030\003 \003(\0132\007.Action\"D\n\017RegionLoadStat" +
+      "s\022\027\n\014memstoreLoad\030\001 \001(\005:\0010\022\030\n\rheapOccupa" +
+      "ncy\030\002 \001(\005:\0010\"\266\001\n\021ResultOrException\022\r\n\005in" +
+      "dex\030\001 \001(\r\022\027\n\006result\030\002 \001(\0132\007.Result\022!\n\tex",
+      "ception\030\003 \001(\0132\016.NameBytesPair\0221\n\016service" +
+      "_result\030\004 \001(\0132\031.CoprocessorServiceResult" +
+      "\022#\n\tloadStats\030\005 \001(\0132\020.RegionLoadStats\"f\n" +
+      "\022RegionActionResult\022-\n\021resultOrException" +
+      "\030\001 \003(\0132\022.ResultOrException\022!\n\texception\030" +
+      "\002 \001(\0132\016.NameBytesPair\"f\n\014MultiRequest\022#\n" +
+      "\014regionAction\030\001 \003(\0132\r.RegionAction\022\022\n\nno" +
+      "nceGroup\030\002 \001(\004\022\035\n\tcondition\030\003 \001(\0132\n.Cond" +
+      "ition\"S\n\rMultiResponse\022/\n\022regionActionRe" +
+      "sult\030\001 \003(\0132\023.RegionActionResult\022\021\n\tproce",
+      "ssed\030\002 \001(\010*\'\n\013Consistency\022\n\n\006STRONG\020\000\022\014\n" +
+      "\010TIMELINE\020\0012\205\003\n\rClientService\022 \n\003Get\022\013.G" +
+      "etRequest\032\014.GetResponse\022)\n\006Mutate\022\016.Muta" +
+      "teRequest\032\017.MutateResponse\022#\n\004Scan\022\014.Sca" +
+      "nRequest\032\r.ScanResponse\022>\n\rBulkLoadHFile" +
+      "\022\025.BulkLoadHFileRequest\032\026.BulkLoadHFileR" +
+      "esponse\022F\n\013ExecService\022\032.CoprocessorServ" +
+      "iceRequest\032\033.CoprocessorServiceResponse\022" +
+      "R\n\027ExecRegionServerService\022\032.Coprocessor" +
+      "ServiceRequest\032\033.CoprocessorServiceRespo",
+      "nse\022&\n\005Multi\022\r.MultiRequest\032\016.MultiRespo" +
+      "nseBB\n*org.apache.hadoop.hbase.protobuf." +
+      "generatedB\014ClientProtosH\001\210\001\001\240\001\001"
     };
     com.google.protobuf.Descriptors.FileDescriptor.InternalDescriptorAssigner assigner =
       new com.google.protobuf.Descriptors.FileDescriptor.InternalDescriptorAssigner() {
@@ -32180,7 +32319,7 @@ public final class ClientProtos {
           internal_static_ScanResponse_fieldAccessorTable = new
             com.google.protobuf.GeneratedMessage.FieldAccessorTable(
               internal_static_ScanResponse_descriptor,
-              new java.lang.String[] { "CellsPerResult", "ScannerId", "MoreResults", "Ttl", "Results", "Stale", });
+              new java.lang.String[] { "CellsPerResult", "ScannerId", "MoreResults", "Ttl", "Results", "Stale", "MoreResultsInRegion", });
           internal_static_BulkLoadHFileRequest_descriptor =
             getDescriptor().getMessageTypes().get(14);
           internal_static_BulkLoadHFileRequest_fieldAccessorTable = new

http://git-wip-us.apache.org/repos/asf/hbase/blob/20189c25/hbase-protocol/src/main/protobuf/Client.proto
----------------------------------------------------------------------
diff --git a/hbase-protocol/src/main/protobuf/Client.proto b/hbase-protocol/src/main/protobuf/Client.proto
index 606ca8d..b5439f7 100644
--- a/hbase-protocol/src/main/protobuf/Client.proto
+++ b/hbase-protocol/src/main/protobuf/Client.proto
@@ -291,6 +291,11 @@ message ScanResponse {
   // be inside the pb'd Result)
   repeated Result results = 5;
   optional bool stale = 6;
+
+  // A server may choose to limit the number of results returned to the client for
+  // reasons such as the size in bytes or quantity of results accumulated. This field
+  // will true when more results exist in the current region.
+  optional bool more_results_in_region = 8;
 }
 
 /**

http://git-wip-us.apache.org/repos/asf/hbase/blob/20189c25/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/RSRpcServices.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/RSRpcServices.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/RSRpcServices.java
index df01875..3165854 100644
--- a/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/RSRpcServices.java
+++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/RSRpcServices.java
@@ -2091,14 +2091,16 @@ public class RSRpcServices implements HBaseRPCErrorHandler,
               int i = 0;
               synchronized(scanner) {
                 boolean stale = (region.getRegionInfo().getReplicaId() != 0);
+                boolean moreRows = false;
                 while (i < rows) {
                   // Stop collecting results if maxScannerResultSize is set and we have exceeded it
                   if ((maxScannerResultSize < Long.MAX_VALUE) &&
                       (currentScanResultSize >= maxResultSize)) {
+                    builder.setMoreResultsInRegion(true);
                     break;
                   }
                   // Collect values to be returned here
-                  boolean moreRows = scanner.nextRaw(values);
+                  moreRows = scanner.nextRaw(values);
                   if (!values.isEmpty()) {
                     for (Cell cell : values) {
                       currentScanResultSize += CellUtil.estimatedHeapSizeOfWithoutTags(cell);
@@ -2112,6 +2114,15 @@ public class RSRpcServices implements HBaseRPCErrorHandler,
                   }
                   values.clear();
                 }
+                // currentScanResultSize >= maxResultSize should be functionally equivalent to
+                // state.sizeLimitReached()
+                if (currentScanResultSize >= maxResultSize || i >= rows || moreRows) {
+                  // We stopped prematurely
+                  builder.setMoreResultsInRegion(true);
+                } else {
+                  // We didn't get a single batch
+                  builder.setMoreResultsInRegion(false);
+                }
               }
               region.readRequestsCount.add(i);
               region.getMetrics().updateScanNext(totalCellSize);

http://git-wip-us.apache.org/repos/asf/hbase/blob/20189c25/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestSizeFailures.java
----------------------------------------------------------------------
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestSizeFailures.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestSizeFailures.java
new file mode 100644
index 0000000..a77c50c
--- /dev/null
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/client/TestSizeFailures.java
@@ -0,0 +1,162 @@
+/**
+ *
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.hbase.client;
+
+import static org.junit.Assert.assertEquals;
+
+import java.io.IOException;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.LinkedList;
+import java.util.List;
+import java.util.TreeSet;
+
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.hbase.HBaseTestingUtility;
+import org.apache.hadoop.hbase.HColumnDescriptor;
+import org.apache.hadoop.hbase.HTableDescriptor;
+import org.apache.hadoop.hbase.TableName;
+import org.apache.hadoop.hbase.testclassification.LargeTests;
+import org.apache.hadoop.hbase.util.Bytes;
+import org.junit.AfterClass;
+import org.junit.BeforeClass;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+
+@Category(LargeTests.class)
+public class TestSizeFailures {
+  final Log LOG = LogFactory.getLog(getClass());
+  protected final static HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
+  private static byte [] FAMILY = Bytes.toBytes("testFamily");
+  protected static int SLAVES = 1;
+
+  /**
+   * @throws java.lang.Exception
+   */
+  @BeforeClass
+  public static void setUpBeforeClass() throws Exception {
+    // Uncomment the following lines if more verbosity is needed for
+    // debugging (see HBASE-12285 for details).
+    //((Log4JLogger)RpcServer.LOG).getLogger().setLevel(Level.ALL);
+    //((Log4JLogger)RpcClient.LOG).getLogger().setLevel(Level.ALL);
+    //((Log4JLogger)ScannerCallable.LOG).getLogger().setLevel(Level.ALL);
+    Configuration conf = TEST_UTIL.getConfiguration();
+    conf.setBoolean("hbase.table.sanity.checks", true); // ignore sanity checks in the server
+    TEST_UTIL.startMiniCluster(SLAVES);
+  }
+
+  /**
+   * @throws java.lang.Exception
+   */
+  @AfterClass
+  public static void tearDownAfterClass() throws Exception {
+    TEST_UTIL.shutdownMiniCluster();
+  }
+
+  /**
+   * Basic client side validation of HBASE-13262
+   */
+   @Test
+   public void testScannerSeesAllRecords() throws Exception {
+     final int NUM_ROWS = 1000 * 1000, NUM_COLS = 10;
+     final TableName TABLENAME = TableName.valueOf("testScannerSeesAllRecords");
+     List<byte[]> qualifiers = new ArrayList<>();
+     for (int i = 1; i <= 10; i++) {
+       qualifiers.add(Bytes.toBytes(Integer.toString(i)));
+     }
+
+     HColumnDescriptor hcd = new HColumnDescriptor(FAMILY);
+     HTableDescriptor desc = new HTableDescriptor(TABLENAME);
+     desc.addFamily(hcd);
+     byte[][] splits = new byte[9][2];
+     for (int i = 1; i < 10; i++) {
+       int split = 48 + i;
+       splits[i - 1][0] = (byte) (split >>> 8);
+       splits[i - 1][0] = (byte) (split);
+     }
+     TEST_UTIL.getHBaseAdmin().createTable(desc, splits);
+     Connection conn = TEST_UTIL.getConnection();
+
+     try (Table table = conn.getTable(TABLENAME)) {
+       List<Put> puts = new LinkedList<>();
+       for (int i = 0; i < NUM_ROWS; i++) {
+         Put p = new Put(Bytes.toBytes(Integer.toString(i)));
+         for (int j = 0; j < NUM_COLS; j++) {
+           byte[] value = new byte[50];
+           Bytes.random(value);
+           p.addColumn(FAMILY, Bytes.toBytes(Integer.toString(j)), value);
+         }
+         puts.add(p);
+
+         if (puts.size() == 1000) {
+           Object[] results = new Object[1000];
+           try {
+             table.batch(puts, results);
+           } catch (IOException e) {
+             LOG.error("Failed to write data", e);
+             LOG.debug("Errors: " +  Arrays.toString(results));
+           }
+
+           puts.clear();
+         }
+       }
+
+       if (puts.size() > 0) {
+         Object[] results = new Object[puts.size()];
+         try {
+           table.batch(puts, results);
+         } catch (IOException e) {
+           LOG.error("Failed to write data", e);
+           LOG.debug("Errors: " +  Arrays.toString(results));
+         }
+       }
+
+       // Flush the memstore to disk
+       TEST_UTIL.getHBaseAdmin().flush(TABLENAME);
+
+       TreeSet<Integer> rows = new TreeSet<>();
+       long rowsObserved = 0l;
+       long entriesObserved = 0l;
+       Scan s = new Scan();
+       s.addFamily(FAMILY);
+       s.setMaxResultSize(-1);
+       s.setBatch(-1);
+       s.setCaching(500);
+       ResultScanner scanner = table.getScanner(s);
+       // Read all the records in the table
+       for (Result result : scanner) {
+         rowsObserved++;
+         String row = new String(result.getRow());
+         rows.add(Integer.parseInt(row));
+         while (result.advance()) {
+           entriesObserved++;
+           // result.current();
+         }
+       }
+
+       // Verify that we see 1M rows and 10M cells
+       assertEquals(NUM_ROWS, rowsObserved);
+       assertEquals(NUM_ROWS * NUM_COLS, entriesObserved);
+     }
+
+     conn.close();
+   }
+}


Mime
View raw message