@InterfaceAudience.LimitedPrivate(value="Coprocesssor") @InterfaceStability.Evolving public interface Region extends ConfigurationObserver
An Region is defined by its table and its key extent.
Locking at the Region level serves only one purpose: preventing the region from being closed (and consequently split) while other operations are ongoing. Each row level operation obtains both a row lock and a region read lock for the duration of the operation. While a scanner is being constructed, getScanner holds a read lock. If the scanner is successfully constructed, it holds a read lock until it is closed. A close takes out a write lock and consequently will block for ongoing operations and will block new operations from starting while the close is in progress.
Modifier and Type | Interface and Description |
---|---|
static interface |
Region.BulkLoadListener
Listener class to enable callers of
bulkLoadHFile() to perform any necessary
pre/post processing of a given bulkload call
|
static interface |
Region.FlushResult |
static class |
Region.Operation
Operation enum is used in
startRegionOperation() to provide context for
various checks before any region operation begins. |
static interface |
Region.RowLock
Row lock held by a given thread.
|
Modifier and Type | Method and Description |
---|---|
Result |
append(Append append,
long nonceGroup,
long nonce)
Perform one or more append operations on a row.
|
OperationStatus[] |
batchMutate(Mutation[] mutations,
long nonceGroup,
long nonce)
Perform a batch of mutations.
|
OperationStatus[] |
batchReplay(WALSplitter.MutationReplay[] mutations,
long replaySeqId)
Replay a batch of mutations.
|
boolean |
bulkLoadHFiles(Collection<Pair<byte[],String>> familyPaths,
boolean assignSeqId,
Region.BulkLoadListener bulkLoadListener)
Attempts to atomically load a group of hfiles.
|
boolean |
checkAndMutate(byte[] row,
byte[] family,
byte[] qualifier,
CompareFilter.CompareOp compareOp,
ByteArrayComparable comparator,
Mutation mutation,
boolean writeToWAL)
Atomically checks if a row/family/qualifier value matches the expected val
If it does, it performs the row mutations.
|
boolean |
checkAndRowMutate(byte[] row,
byte[] family,
byte[] qualifier,
CompareFilter.CompareOp compareOp,
ByteArrayComparable comparator,
RowMutations mutations,
boolean writeToWAL)
Atomically checks if a row/family/qualifier value matches the expected val
If it does, it performs the row mutations.
|
void |
checkFamilies(Collection<byte[]> families)
Check the collection of families for validity.
|
void |
checkTimestamps(Map<byte[],List<Cell>> familyMap,
long now)
Check the collection of families for valid timestamps
|
void |
closeRegionOperation()
Closes the region operation lock.
|
void |
compact(boolean majorCompaction)
Synchronously compact all stores in the region.
|
void |
delete(Delete delete)
Deletes the specified cells/row.
|
com.google.protobuf.Message |
execService(com.google.protobuf.RpcController controller,
ClientProtos.CoprocessorServiceCall call)
Executes a single protocol buffer coprocessor endpoint
Service method using
the registered protocol handlers. |
Region.FlushResult |
flush(boolean force)
Flush the cache.
|
Result |
get(Get get)
Do a get based on the get parameter.
|
List<Cell> |
get(Get get,
boolean withCoprocessor)
Do a get based on the get parameter.
|
long |
getBlockedRequestsCount() |
long |
getCheckAndMutateChecksFailed() |
long |
getCheckAndMutateChecksPassed() |
Result |
getClosestRowBefore(byte[] row,
byte[] family)
Return all the data for the row that matches row exactly,
or the one that immediately preceeds it, at or immediately before
ts.
|
AdminProtos.GetRegionInfoResponse.CompactionState |
getCompactionState() |
RegionCoprocessorHost |
getCoprocessorHost() |
long |
getDataInMemoryWithoutWAL() |
long |
getEarliestFlushTimeForAllStores() |
HDFSBlocksDistribution |
getHDFSBlocksDistribution() |
long |
getMaxFlushedSeqId() |
Map<byte[],Long> |
getMaxStoreSeqId() |
long |
getMemstoreSize() |
MetricsRegion |
getMetrics() |
long |
getNumMutationsWithoutWAL() |
long |
getOldestHfileTs(boolean majorCompactioOnly)
This can be used to determine the last time all files of this region were major compacted.
|
long |
getOldestSeqIdOfStore(byte[] familyName) |
long |
getOpenSeqNum() |
long |
getReadpoint(IsolationLevel isolationLevel) |
long |
getReadRequestsCount() |
HRegionInfo |
getRegionInfo() |
Region.RowLock |
getRowLock(byte[] row,
boolean waitForLock)
Tries to acquire a lock on the given row.
|
RegionScanner |
getScanner(Scan scan)
Return an iterator that scans over the HRegion, returning the indicated
columns and rows specified by the
Scan . |
Store |
getStore(byte[] family)
Return the Store for the given family
|
List<String> |
getStoreFileList(byte[][] columns) |
List<Store> |
getStores()
Return the list of Stores managed by this region
|
HTableDescriptor |
getTableDesc() |
long |
getWriteRequestsCount() |
Result |
increment(Increment increment,
long nonceGroup,
long nonce)
Perform one or more increment operations on a row.
|
boolean |
isAvailable() |
boolean |
isClosed() |
boolean |
isClosing() |
boolean |
isLoadingCfsOnDemandDefault() |
boolean |
isReadOnly() |
boolean |
isRecovering() |
void |
mutateRow(RowMutations mutations)
Performs multiple mutations atomically on a single row.
|
void |
mutateRowsWithLocks(Collection<Mutation> mutations,
Collection<byte[]> rowsToLock,
long nonceGroup,
long nonce)
Perform atomic mutations within the region.
|
void |
prepareDelete(Delete delete)
Prepare a delete for a row mutation processor
|
void |
prepareDeleteTimestamps(Mutation mutation,
Map<byte[],List<Cell>> familyCellMap,
byte[] now)
Set up correct timestamps in the KVs in Delete object.
|
void |
processRowsWithLocks(RowProcessor<?,?> processor)
Performs atomic multiple reads and writes on a given row.
|
void |
processRowsWithLocks(RowProcessor<?,?> processor,
long nonceGroup,
long nonce)
Performs atomic multiple reads and writes on a given row.
|
void |
processRowsWithLocks(RowProcessor<?,?> processor,
long timeout,
long nonceGroup,
long nonce)
Performs atomic multiple reads and writes on a given row.
|
void |
put(Put put)
Puts some data in the table.
|
boolean |
refreshStoreFiles()
Check the region's underlying store files, open the files that have not
been opened yet, and remove the store file readers for store files no
longer available.
|
boolean |
registerService(com.google.protobuf.Service instance)
Registers a new protocol buffer
Service subclass as a coprocessor endpoint to
be available for handling
execService(com.google.protobuf.RpcController,
org.apache.hadoop.hbase.protobuf.generated.ClientProtos.CoprocessorServiceCall) } calls. |
void |
releaseRowLocks(List<Region.RowLock> rowLocks)
If the given list of row locks is not null, releases all locks.
|
void |
startRegionOperation()
This method needs to be called before any public call that reads or
modifies data.
|
void |
startRegionOperation(Region.Operation op)
This method needs to be called before any public call that reads or
modifies data.
|
void |
triggerMajorCompaction()
Trigger major compaction on all stores in the region.
|
void |
updateCellTimestamps(Iterable<List<Cell>> values,
byte[] now)
Replace any cell timestamps set to HConstants#LATEST_TIMESTAMP with the
provided current timestamp.
|
void |
updateReadRequestsCount(long i)
Update the read request count for this region
|
void |
updateWriteRequestsCount(long i)
Update the write request count for this region
|
void |
waitForFlushesAndCompactions()
Wait for all current flushes and compactions of the region to complete
|
onConfigurationChange
HRegionInfo getRegionInfo()
HTableDescriptor getTableDesc()
boolean isAvailable()
boolean isClosed()
boolean isClosing()
boolean isRecovering()
boolean isReadOnly()
List<Store> getStores()
Use with caution. Exposed for use of fixup utilities.
Store getStore(byte[] family)
Use with caution. Exposed for use of fixup utilities.
List<String> getStoreFileList(byte[][] columns)
boolean refreshStoreFiles() throws IOException
IOException
long getOpenSeqNum()
long getMaxFlushedSeqId()
long getOldestSeqIdOfStore(byte[] familyName)
long getOldestHfileTs(boolean majorCompactioOnly) throws IOException
majorCompactioOnly
- Only consider HFile that are the result of major compactionIOException
Map<byte[],Long> getMaxStoreSeqId()
boolean isLoadingCfsOnDemandDefault()
long getReadpoint(IsolationLevel isolationLevel)
long getEarliestFlushTimeForAllStores()
long getReadRequestsCount()
void updateReadRequestsCount(long i)
i
- incrementlong getWriteRequestsCount()
void updateWriteRequestsCount(long i)
i
- incrementlong getMemstoreSize()
long getNumMutationsWithoutWAL()
long getDataInMemoryWithoutWAL()
long getBlockedRequestsCount()
long getCheckAndMutateChecksPassed()
long getCheckAndMutateChecksFailed()
MetricsRegion getMetrics()
HDFSBlocksDistribution getHDFSBlocksDistribution()
void startRegionOperation() throws IOException
closeRegionOperation()
MUST then always be called after
the operation has completed, whether it succeeded or failed.
IOException
void startRegionOperation(Region.Operation op) throws IOException
closeRegionOperation()
MUST then always be called after
the operation has completed, whether it succeeded or failed.
op
- The operation is about to be taken on the regionIOException
void closeRegionOperation() throws IOException
IOException
Region.RowLock getRowLock(byte[] row, boolean waitForLock) throws IOException
waitForLock
- if true, will block until the lock is available.
Otherwise, just tries to obtain the lock and returns
false if unavailable.IOException
- if waitForLock was true and the lock could not be acquired after waitingvoid releaseRowLocks(List<Region.RowLock> rowLocks)
Result append(Append append, long nonceGroup, long nonce) throws IOException
append
- nonceGroup
- nonce
- IOException
OperationStatus[] batchMutate(Mutation[] mutations, long nonceGroup, long nonce) throws IOException
Note this supports only Put and Delete mutations and will ignore other types passed.
mutations
- the list of mutationsnonceGroup
- nonce
- IOException
OperationStatus[] batchReplay(WALSplitter.MutationReplay[] mutations, long replaySeqId) throws IOException
mutations
- mutations to replay.replaySeqId
- IOException
boolean checkAndMutate(byte[] row, byte[] family, byte[] qualifier, CompareFilter.CompareOp compareOp, ByteArrayComparable comparator, Mutation mutation, boolean writeToWAL) throws IOException
row
- to checkfamily
- column family to checkqualifier
- column qualifier to checkcompareOp
- the comparison operatorcomparator
- mutation
- writeToWAL
- IOException
boolean checkAndRowMutate(byte[] row, byte[] family, byte[] qualifier, CompareFilter.CompareOp compareOp, ByteArrayComparable comparator, RowMutations mutations, boolean writeToWAL) throws IOException
row
- to checkfamily
- column family to checkqualifier
- column qualifier to checkcompareOp
- the comparison operatorcomparator
- mutations
- writeToWAL
- IOException
void delete(Delete delete) throws IOException
delete
- IOException
Result get(Get get) throws IOException
get
- query parametersIOException
List<Cell> get(Get get, boolean withCoprocessor) throws IOException
get
- query parameterswithCoprocessor
- invoke coprocessor or not. We don't want to
always invoke cp.IOException
Result getClosestRowBefore(byte[] row, byte[] family) throws IOException
row
- family
- IOException
RegionScanner getScanner(Scan scan) throws IOException
Scan
.
This Iterator must be closed by the caller.
scan
- configured Scan
IOException
- read exceptionsResult increment(Increment increment, long nonceGroup, long nonce) throws IOException
increment
- nonceGroup
- nonce
- IOException
void mutateRow(RowMutations mutations) throws IOException
mutations
- object that specifies the set of mutations to perform atomicallyIOException
void mutateRowsWithLocks(Collection<Mutation> mutations, Collection<byte[]> rowsToLock, long nonceGroup, long nonce) throws IOException
mutations
- The list of mutations to perform.
mutations
can contain operations for multiple rows.
Caller has to ensure that all rows are contained in this region.rowsToLock
- Rows to locknonceGroup
- Optional nonce group of the operation (client Id)nonce
- Optional nonce of the operation (unique random id to ensure "more idempotence")
If multiple rows are locked care should be taken that
rowsToLock
is sorted in order to avoid deadlocks.IOException
void processRowsWithLocks(RowProcessor<?,?> processor) throws IOException
processor
- The object defines the reads and writes to a row.IOException
void processRowsWithLocks(RowProcessor<?,?> processor, long nonceGroup, long nonce) throws IOException
processor
- The object defines the reads and writes to a row.nonceGroup
- Optional nonce group of the operation (client Id)nonce
- Optional nonce of the operation (unique random id to ensure "more idempotence")IOException
void processRowsWithLocks(RowProcessor<?,?> processor, long timeout, long nonceGroup, long nonce) throws IOException
processor
- The object defines the reads and writes to a row.timeout
- The timeout of the processor.process() execution
Use a negative number to switch off the time boundnonceGroup
- Optional nonce group of the operation (client Id)nonce
- Optional nonce of the operation (unique random id to ensure "more idempotence")IOException
void put(Put put) throws IOException
put
- IOException
boolean bulkLoadHFiles(Collection<Pair<byte[],String>> familyPaths, boolean assignSeqId, Region.BulkLoadListener bulkLoadListener) throws IOException
familyPaths
- List of PairbulkLoadListener
- Internal hooks enabling massaging/preparation of a
file about to be bulk loadedassignSeqId
- IOException
- if failed unrecoverably.RegionCoprocessorHost getCoprocessorHost()
com.google.protobuf.Message execService(com.google.protobuf.RpcController controller, ClientProtos.CoprocessorServiceCall call) throws IOException
Service
method using
the registered protocol handlers. Service
implementations must be registered via the
registerService(com.google.protobuf.Service)
method before they are available.controller
- an RpcContoller
implementation to pass to the invoked servicecall
- a CoprocessorServiceCall
instance identifying the service, method,
and parameters for the method invocationMessage
instance containing the method's resultIOException
- if no registered service handler is found or an error
occurs during the invocationregisterService(com.google.protobuf.Service)
boolean registerService(com.google.protobuf.Service instance)
Service
subclass as a coprocessor endpoint to
be available for handling
execService(com.google.protobuf.RpcController,
org.apache.hadoop.hbase.protobuf.generated.ClientProtos.CoprocessorServiceCall)
} calls.
Only a single instance may be registered per region for a given Service
subclass (the
instances are keyed on Descriptors.ServiceDescriptor.getFullName()
.
After the first registration, subsequent calls with the same service name will fail with
a return value of false
.
instance
- the Service
subclass instance to expose as a coprocessor endpointtrue
if the registration was successful, false
otherwisevoid checkFamilies(Collection<byte[]> families) throws NoSuchColumnFamilyException
families
- NoSuchColumnFamilyException
void checkTimestamps(Map<byte[],List<Cell>> familyMap, long now) throws FailedSanityCheckException
familyMap
- now
- current timestampFailedSanityCheckException
void prepareDelete(Delete delete) throws IOException
delete
- The passed delete is modified by this method. WARNING!IOException
void prepareDeleteTimestamps(Mutation mutation, Map<byte[],List<Cell>> familyCellMap, byte[] now) throws IOException
Caller should have the row and region locks.
mutation
- familyCellMap
- now
- IOException
void updateCellTimestamps(Iterable<List<Cell>> values, byte[] now) throws IOException
values
- now
- IOException
Region.FlushResult flush(boolean force) throws IOException
When this method is called the cache will be flushed unless:
This method may block for some time, so it should not be called from a time-sensitive thread.
force
- whether we want to force a flush of all storesIOException
- general io exceptionsDroppedSnapshotException
- Thrown when abort is required. The caller MUST catch this
exception and MUST abort. Any further operation to the region may cause data loss.
because a snapshot was not properly persisted.void compact(boolean majorCompaction) throws IOException
This operation could block for a long time, so don't call it from a time-sensitive thread.
Note that no locks are taken to prevent possible conflicts between compaction and splitting activities. The regionserver does not normally compact and split in parallel. However by calling this method you may introduce unexpected and unhandled concurrency. Don't do this unless you know what you are doing.
majorCompaction
- True to force a major compaction regardless of thresholdsIOException
void triggerMajorCompaction() throws IOException
Compaction will be performed asynchronously to this call by the RegionServer's
CompactSplitThread. See also Store.triggerMajorCompaction()
IOException
AdminProtos.GetRegionInfoResponse.CompactionState getCompactionState()
void waitForFlushesAndCompactions()
Copyright © 2007-2016 The Apache Software Foundation. All Rights Reserved.