HBASE-5664 CP hooks in Scan flow for fast forward when filter filters out a row (Anoop)
git-svn-id: https://svn.apache.org/repos/asf/hbase/trunk@1437790 13f79535-47bb-0310-9956-ffa450edef68
This commit is contained in:
parent
507565e35f
commit
c637f33c8c
|
@ -314,6 +314,12 @@ public abstract class BaseRegionObserver implements RegionObserver {
|
|||
return hasMore;
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean postScannerFilterRow(final ObserverContext<RegionCoprocessorEnvironment> e,
|
||||
final InternalScanner s, final byte[] currentRow, final boolean hasMore) throws IOException {
|
||||
return hasMore;
|
||||
}
|
||||
|
||||
@Override
|
||||
public void preScannerClose(final ObserverContext<RegionCoprocessorEnvironment> e,
|
||||
final InternalScanner s) throws IOException {
|
||||
|
|
|
@ -734,6 +734,27 @@ public interface RegionObserver extends Coprocessor {
|
|||
final boolean hasNext)
|
||||
throws IOException;
|
||||
|
||||
/**
|
||||
* This will be called by the scan flow when the current scanned row is being filtered out by the
|
||||
* filter. The filter may be filtering out the row via any of the below scenarios
|
||||
* <ol>
|
||||
* <li>
|
||||
* <code>boolean filterRowKey(byte [] buffer, int offset, int length)</code> returning true</li>
|
||||
* <li>
|
||||
* <code>boolean filterRow()</code> returning true</li>
|
||||
* <li>
|
||||
* <code>void filterRow(List<KeyValue> kvs)</code> removing all the kvs from the passed List</li>
|
||||
* </ol>
|
||||
* @param c the environment provided by the region server
|
||||
* @param s the scanner
|
||||
* @param currentRow The current rowkey which got filtered out
|
||||
* @param hasMore the 'has more' indication
|
||||
* @return whether more rows are available for the scanner or not
|
||||
* @throws IOException
|
||||
*/
|
||||
boolean postScannerFilterRow(final ObserverContext<RegionCoprocessorEnvironment> c,
|
||||
final InternalScanner s, final byte[] currentRow, final boolean hasMore) throws IOException;
|
||||
|
||||
/**
|
||||
* Called before the client closes a scanner.
|
||||
* <p>
|
||||
|
|
|
@ -1774,7 +1774,7 @@ public class HRegion implements HeapSize { // , Writable{
|
|||
|
||||
protected RegionScanner instantiateRegionScanner(Scan scan,
|
||||
List<KeyValueScanner> additionalScanners) throws IOException {
|
||||
return new RegionScannerImpl(scan, additionalScanners);
|
||||
return new RegionScannerImpl(scan, additionalScanners, this);
|
||||
}
|
||||
|
||||
/*
|
||||
|
@ -3380,13 +3380,16 @@ public class HRegion implements HeapSize { // , Writable{
|
|||
private boolean filterClosed = false;
|
||||
private long readPt;
|
||||
private long maxResultSize;
|
||||
private HRegion region;
|
||||
|
||||
public HRegionInfo getRegionInfo() {
|
||||
return regionInfo;
|
||||
}
|
||||
RegionScannerImpl(Scan scan, List<KeyValueScanner> additionalScanners) throws IOException {
|
||||
//DebugPrint.println("HRegionScanner.<init>");
|
||||
|
||||
|
||||
RegionScannerImpl(Scan scan, List<KeyValueScanner> additionalScanners, HRegion region)
|
||||
throws IOException {
|
||||
// DebugPrint.println("HRegionScanner.<init>");
|
||||
this.region = region;
|
||||
this.maxResultSize = scan.getMaxResultSize();
|
||||
if (scan.hasFilter()) {
|
||||
this.filter = new FilterWrapper(scan.getFilter());
|
||||
|
@ -3443,8 +3446,8 @@ public class HRegion implements HeapSize { // , Writable{
|
|||
}
|
||||
}
|
||||
|
||||
RegionScannerImpl(Scan scan) throws IOException {
|
||||
this(scan, null);
|
||||
RegionScannerImpl(Scan scan, HRegion region) throws IOException {
|
||||
this(scan, null, region);
|
||||
}
|
||||
|
||||
@Override
|
||||
|
@ -3625,7 +3628,8 @@ public class HRegion implements HeapSize { // , Writable{
|
|||
// Check if rowkey filter wants to exclude this row. If so, loop to next.
|
||||
// Technically, if we hit limits before on this row, we don't need this call.
|
||||
if (filterRowKey(currentRow, offset, length)) {
|
||||
nextRow(currentRow, offset, length);
|
||||
boolean moreRows = nextRow(currentRow, offset, length);
|
||||
if (!moreRows) return false;
|
||||
results.clear();
|
||||
continue;
|
||||
}
|
||||
|
@ -3652,7 +3656,8 @@ public class HRegion implements HeapSize { // , Writable{
|
|||
filter.filterRow(results);
|
||||
}
|
||||
if (isEmptyRow) {
|
||||
nextRow(currentRow, offset, length);
|
||||
boolean moreRows = nextRow(currentRow, offset, length);
|
||||
if (!moreRows) return false;
|
||||
results.clear();
|
||||
// This row was totally filtered out, if this is NOT the last row,
|
||||
// we should continue on. Otherwise, nothing else to do.
|
||||
|
@ -3691,7 +3696,8 @@ public class HRegion implements HeapSize { // , Writable{
|
|||
// Double check to prevent empty rows from appearing in result. It could be
|
||||
// the case when SingleColumnValueExcludeFilter is used.
|
||||
if (results.isEmpty()) {
|
||||
nextRow(currentRow, offset, length);
|
||||
boolean moreRows = nextRow(currentRow, offset, length);
|
||||
if (!moreRows) return false;
|
||||
if (!stopRow) continue;
|
||||
}
|
||||
|
||||
|
@ -3705,13 +3711,18 @@ public class HRegion implements HeapSize { // , Writable{
|
|||
&& filter.filterRowKey(row, offset, length);
|
||||
}
|
||||
|
||||
protected void nextRow(byte [] currentRow, int offset, short length) throws IOException {
|
||||
protected boolean nextRow(byte [] currentRow, int offset, short length) throws IOException {
|
||||
assert this.joinedContinuationRow == null : "Trying to go to next row during joinedHeap read.";
|
||||
KeyValue next;
|
||||
while ((next = this.storeHeap.peek()) != null && next.matchingRow(currentRow, offset, length)) {
|
||||
this.storeHeap.next(MOCKED_LIST);
|
||||
}
|
||||
resetFilters();
|
||||
// Calling the hook in CP which allows it to do a fast forward
|
||||
if (this.region.getCoprocessorHost() != null) {
|
||||
return this.region.getCoprocessorHost().postScannerFilterRow(this, currentRow);
|
||||
}
|
||||
return true;
|
||||
}
|
||||
|
||||
private boolean isStopRow(byte [] currentRow, int offset, short length) {
|
||||
|
|
|
@ -1349,6 +1349,35 @@ public class RegionCoprocessorHost
|
|||
return hasMore;
|
||||
}
|
||||
|
||||
/**
|
||||
* This will be called by the scan flow when the current scanned row is being filtered out by the
|
||||
* filter.
|
||||
* @param s the scanner
|
||||
* @param currentRow The current rowkey which got filtered out
|
||||
* @return whether more rows are available for the scanner or not
|
||||
* @throws IOException
|
||||
*/
|
||||
public boolean postScannerFilterRow(final InternalScanner s, final byte[] currentRow)
|
||||
throws IOException {
|
||||
boolean hasMore = true; // By default assume more rows there.
|
||||
ObserverContext<RegionCoprocessorEnvironment> ctx = null;
|
||||
for (RegionEnvironment env : coprocessors) {
|
||||
if (env.getInstance() instanceof RegionObserver) {
|
||||
ctx = ObserverContext.createAndPrepare(env, ctx);
|
||||
try {
|
||||
hasMore = ((RegionObserver) env.getInstance()).postScannerFilterRow(ctx, s, currentRow,
|
||||
hasMore);
|
||||
} catch (Throwable e) {
|
||||
handleCoprocessorThrowable(env, e);
|
||||
}
|
||||
if (ctx.shouldComplete()) {
|
||||
break;
|
||||
}
|
||||
}
|
||||
}
|
||||
return hasMore;
|
||||
}
|
||||
|
||||
/**
|
||||
* @param s the scanner
|
||||
* @return true if default behavior should be bypassed, false otherwise
|
||||
|
|
Loading…
Reference in New Issue