@InterfaceAudience.Private public class StoreScanner extends NonReversedNonLazyKeyValueScanner implements KeyValueScanner, InternalScanner, ChangedReadersObserver
| Modifier and Type | Field and Description |
|---|---|
protected boolean |
cacheBlocks |
protected long |
cellsPerHeartbeatCheck |
protected boolean |
closing |
protected NavigableSet<byte[]> |
columns |
protected long |
countPerRow |
static long |
DEFAULT_HBASE_CELLS_SCANNED_PER_HEARTBEAT_CHECK
Default value of
HBASE_CELLS_SCANNED_PER_HEARTBEAT_CHECK. |
protected ExecutorService |
executor |
protected boolean |
explicitColumnQuery |
protected boolean |
get |
static String |
HBASE_CELLS_SCANNED_PER_HEARTBEAT_CHECK
The number of cells scanned in between timeout checks.
|
protected KeyValueHeap |
heap |
protected Cell |
lastTop |
protected static boolean |
lazySeekEnabledGlobally
Used during unit testing to ensure that lazy seek does save seek ops
|
protected ScanQueryMatcher |
matcher |
protected long |
maxRowSize |
protected int |
minVersions |
protected long |
now |
protected long |
oldestUnexpiredTS |
protected boolean |
parallelSeekEnabled
A flag that enables StoreFileScanner parallel-seeking
|
protected Scan |
scan |
protected Store |
store |
protected int |
storeLimit |
protected int |
storeOffset |
static String |
STORESCANNER_PARALLEL_SEEK_ENABLE |
protected boolean |
useRowColBloom |
NO_NEXT_INDEXED_KEY| Modifier | Constructor and Description |
|---|---|
|
StoreScanner(Scan scan,
ScanInfo scanInfo,
ScanType scanType,
NavigableSet<byte[]> columns,
List<KeyValueScanner> scanners,
long earliestPutTs,
long readPt) |
|
StoreScanner(Store store,
ScanInfo scanInfo,
Scan scan,
List<? extends KeyValueScanner> scanners,
long smallestReadPoint,
long earliestPutTs,
byte[] dropDeletesFromRow,
byte[] dropDeletesToRow)
Used for compactions that drop deletes from a limited range of rows.
|
|
StoreScanner(Store store,
ScanInfo scanInfo,
Scan scan,
List<? extends KeyValueScanner> scanners,
ScanType scanType,
long smallestReadPoint,
long earliestPutTs)
Used for compactions.
|
|
StoreScanner(Store store,
ScanInfo scanInfo,
Scan scan,
NavigableSet<byte[]> columns,
long readPt)
Opens a scanner across memstore, snapshot, and all StoreFiles.
|
protected |
StoreScanner(Store store,
Scan scan,
ScanInfo scanInfo,
NavigableSet<byte[]> columns,
long readPt,
boolean cacheBlocks)
An internal constructor.
|
| Modifier and Type | Method and Description |
|---|---|
protected void |
addCurrentScanners(List<? extends KeyValueScanner> scanners) |
protected boolean |
checkFlushed() |
protected boolean |
checkReseek(boolean flushed) |
protected void |
checkScanOrder(Cell prevKV,
Cell kv,
KeyValue.KVComparator comparator)
Check whether scan as expected order
|
void |
close()
Close the KeyValue scanner.
|
long |
getEstimatedNumberOfKvsScanned() |
Cell |
getNextIndexedKey() |
long |
getReadPoint() |
long |
getScannerOrder()
Get the order of this KeyValueScanner.
|
protected List<KeyValueScanner> |
getScannersNoCompaction()
Get a filtered list of scanners.
|
KeyValue |
next()
Return the next Cell in this scanner, iterating the scanner
|
boolean |
next(List<Cell> outResult)
Grab the next row's worth of values.
|
boolean |
next(List<Cell> outResult,
ScannerContext scannerContext)
Get the next row of values from this Store.
|
protected void |
nullifyCurrentHeap() |
Cell |
peek()
Look at the next Cell in this scanner, but do not iterate scanner.
|
boolean |
reseek(Cell kv)
Reseek the scanner at or after the specified KeyValue.
|
protected void |
resetKVHeap(List<? extends KeyValueScanner> scanners,
KeyValue.KVComparator comparator) |
protected void |
resetScannerStack(Cell lastTopKey) |
boolean |
seek(Cell key)
Seek the scanner at or after the specified KeyValue.
|
protected boolean |
seekAsDirection(Cell kv)
Do a reseek in a normal StoreScanner(scan forward)
|
protected void |
seekScanners(List<? extends KeyValueScanner> scanners,
Cell seekKey,
boolean isLazy,
boolean isParallelSeek)
Seek the specified scanners with the given key
|
protected boolean |
seekToNextRow(Cell kv) |
protected List<KeyValueScanner> |
selectScannersFrom(List<? extends KeyValueScanner> allScanners)
Filters the given list of scanners using Bloom filter, time range, and
TTL.
|
protected boolean |
trySkipToNextColumn(Cell cell)
|
protected boolean |
trySkipToNextRow(Cell cell)
See if we should actually SEEK or rather just SKIP to the next Cell (see HBASE-13109).
|
void |
updateReaders(List<StoreFile> sfs,
List<KeyValueScanner> memStoreScanners)
Notify observers.
|
backwardSeek, seekToLastRow, seekToPreviousRowdoRealSeek, enforceSeek, isFileScanner, realSeekDone, requestSeek, shouldUseScannerclone, equals, finalize, getClass, hashCode, notify, notifyAll, toString, wait, wait, waitbackwardSeek, enforceSeek, isFileScanner, realSeekDone, requestSeek, seekToLastRow, seekToPreviousRow, shouldUseScannerprotected final Store store
protected ScanQueryMatcher matcher
protected KeyValueHeap heap
protected boolean cacheBlocks
protected long countPerRow
protected int storeLimit
protected int storeOffset
protected boolean closing
protected final boolean get
protected final boolean explicitColumnQuery
protected final boolean useRowColBloom
protected boolean parallelSeekEnabled
protected ExecutorService executor
protected final Scan scan
protected final NavigableSet<byte[]> columns
protected final long oldestUnexpiredTS
protected final long now
protected final int minVersions
protected final long maxRowSize
protected final long cellsPerHeartbeatCheck
public static final String STORESCANNER_PARALLEL_SEEK_ENABLE
protected static boolean lazySeekEnabledGlobally
public static final String HBASE_CELLS_SCANNED_PER_HEARTBEAT_CHECK
public static final long DEFAULT_HBASE_CELLS_SCANNED_PER_HEARTBEAT_CHECK
HBASE_CELLS_SCANNED_PER_HEARTBEAT_CHECK.protected Cell lastTop
protected StoreScanner(Store store, Scan scan, ScanInfo scanInfo, NavigableSet<byte[]> columns, long readPt, boolean cacheBlocks)
public StoreScanner(Store store, ScanInfo scanInfo, Scan scan, NavigableSet<byte[]> columns, long readPt) throws IOException
store - who we scanscan - the speccolumns - which columns we are scanningIOExceptionpublic StoreScanner(Store store, ScanInfo scanInfo, Scan scan, List<? extends KeyValueScanner> scanners, ScanType scanType, long smallestReadPoint, long earliestPutTs) throws IOException
Opens a scanner across specified StoreFiles.
store - who we scanscan - the specscanners - ancillary scannerssmallestReadPoint - the readPoint that we should use for tracking
versionsIOExceptionpublic StoreScanner(Store store, ScanInfo scanInfo, Scan scan, List<? extends KeyValueScanner> scanners, long smallestReadPoint, long earliestPutTs, byte[] dropDeletesFromRow, byte[] dropDeletesToRow) throws IOException
Opens a scanner across specified StoreFiles.
store - who we scanscan - the specscanners - ancillary scannerssmallestReadPoint - the readPoint that we should use for tracking versionsdropDeletesFromRow - The inclusive left bound of the range; can be EMPTY_START_ROW.dropDeletesToRow - The exclusive right bound of the range; can be EMPTY_END_ROW.IOExceptionpublic StoreScanner(Scan scan, ScanInfo scanInfo, ScanType scanType, NavigableSet<byte[]> columns, List<KeyValueScanner> scanners, long earliestPutTs, long readPt) throws IOException
IOExceptionprotected void addCurrentScanners(List<? extends KeyValueScanner> scanners)
protected List<KeyValueScanner> getScannersNoCompaction() throws IOException
IOExceptionprotected void seekScanners(List<? extends KeyValueScanner> scanners, Cell seekKey, boolean isLazy, boolean isParallelSeek) throws IOException
scanners - seekKey - isLazy - true if using lazy seekisParallelSeek - true if using parallel seekIOExceptionprotected void resetKVHeap(List<? extends KeyValueScanner> scanners, KeyValue.KVComparator comparator) throws IOException
IOExceptionprotected List<KeyValueScanner> selectScannersFrom(List<? extends KeyValueScanner> allScanners)
public Cell peek()
KeyValueScannerpeek in interface KeyValueScannerpublic KeyValue next()
KeyValueScannernext in interface KeyValueScannerpublic void close()
KeyValueScannerclose in interface Closeableclose in interface AutoCloseableclose in interface InternalScannerclose in interface KeyValueScannerpublic boolean seek(Cell key) throws IOException
KeyValueScannerseek in interface KeyValueScannerkey - seek valueIOExceptionpublic boolean next(List<Cell> outResult) throws IOException
InternalScannernext in interface InternalScanneroutResult - return output arrayIOException - epublic boolean next(List<Cell> outResult, ScannerContext scannerContext) throws IOException
next in interface InternalScanneroutResult - scannerContext - IOException - eprotected boolean trySkipToNextRow(Cell cell) throws IOException
Other notes:
A good proxy (best effort) to determine whether SKIP is better than SEEK is whether we'll likely end up seeking to the next block (or past the next block) to get our next column. Example:
| BLOCK 1 | BLOCK 2 |
| r1/c1, r1/c2, r1/c3 | r1/c4, r1/c5, r2/c1 |
^ ^
| |
Next Index Key SEEK_NEXT_ROW (before r2/c1)
| BLOCK 1 | BLOCK 2 |
| r1/c1/t5, r1/c1/t4, r1/c1/t3 | r1/c1/t2, r1/c1/T1, r1/c2/T3 |
^ ^
| |
Next Index Key SEEK_NEXT_COL
Now imagine we want columns c1 and c3 (see first diagram above), the 'Next Index Key' of r1/c4
is > r1/c3 so we should seek to get to the c1 on the next row, r2. In second case, say we only
want one version of c1, after we have it, a SEEK_COL will be issued to get to c2. Looking at
the 'Next Index Key', it would land us in the next block, so we should SEEK. In other scenarios
where the SEEK will not land us in the next block, it is very likely better to issues a series
of SKIPs.cell - current cellIOExceptionprotected boolean trySkipToNextColumn(Cell cell) throws IOException
cell - current cellIOExceptionpublic long getReadPoint()
getReadPoint in interface ChangedReadersObserverpublic void updateReaders(List<StoreFile> sfs, List<KeyValueScanner> memStoreScanners) throws IOException
ChangedReadersObserverupdateReaders in interface ChangedReadersObserversfs - The new filesmemStoreScanners - scanner of current memstoreIOException - eprotected void nullifyCurrentHeap()
throws IOException
IOExceptionprotected boolean checkReseek(boolean flushed)
throws IOException
flushed - indicates if there was a flushIOExceptionprotected void resetScannerStack(Cell lastTopKey) throws IOException
IOExceptionprotected void checkScanOrder(Cell prevKV, Cell kv, KeyValue.KVComparator comparator) throws IOException
prevKV - kv - comparator - IOExceptionprotected boolean seekToNextRow(Cell kv) throws IOException
IOExceptionprotected boolean seekAsDirection(Cell kv) throws IOException
kv - IOExceptionpublic boolean reseek(Cell kv) throws IOException
KeyValueScannerreseek in interface KeyValueScannerkv - seek value (should be non-null)IOExceptionprotected boolean checkFlushed()
public long getScannerOrder()
KeyValueScannergetScannerOrder in interface KeyValueScannerKeyValueScanner.getScannerOrder()public long getEstimatedNumberOfKvsScanned()
public Cell getNextIndexedKey()
getNextIndexedKey in interface KeyValueScannergetNextIndexedKey in class NonLazyKeyValueScannerCopyright © 2007–2019 The Apache Software Foundation. All rights reserved.