@InterfaceAudience.Private class SequenceIdAccounting extends Object
 For the implementation, we assume that all the encodedRegionName passed in are gotten by
 RegionInfo.getEncodedNameAsBytes(). So it is safe to use
 it as a hash key. And for family name, we use ImmutableByteArray as key. This is because
 hash based map is much faster than RBTree or CSLM and here we are on the critical write path. See
 HBASE-16278 for more details.
 
| Modifier and Type | Field and Description | 
|---|---|
| private Map<byte[],Map<ImmutableByteArray,Long>> | flushingSequenceIdsMap of encoded region names and family names to their lowest or OLDEST sequence/edit id
 currently being flushed out to hfiles. | 
| private Map<byte[],Long> | highestSequenceIds
 Map of region encoded names to the latest/highest region sequence id. | 
| private static org.slf4j.Logger | LOG | 
| private ConcurrentMap<byte[],ConcurrentMap<ImmutableByteArray,Long>> | lowestUnflushedSequenceIdsMap of encoded region names and family names to their OLDEST -- i.e. | 
| private Object | tieLockThis lock ties all operations on  flushingSequenceIdsandlowestUnflushedSequenceIdsMaps. | 
| Constructor and Description | 
|---|
| SequenceIdAccounting() | 
| Modifier and Type | Method and Description | 
|---|---|
| (package private) void | abortCacheFlush(byte[] encodedRegionName) | 
| (package private) boolean | areAllLower(Map<byte[],Long> sequenceids)See if passed  sequenceidsare lower -- i.e. | 
| (package private) void | completeCacheFlush(byte[] encodedRegionName,
                  long maxFlushedSeqId) | 
| (package private) Map<byte[],List<byte[]>> | findLower(Map<byte[],Long> sequenceids)Iterates over the given Map and compares sequence ids with corresponding entries in
  lowestUnflushedSequenceIds. | 
| private <T extends Map<?,Long>> | flattenToLowestSequenceId(Map<byte[],T> src) | 
| (package private) long | getLowestSequenceId(byte[] encodedRegionName)Returns the lowest unflushed sequence id for the region. | 
| (package private) long | getLowestSequenceId(byte[] encodedRegionName,
                   byte[] familyName) | 
| private static long | getLowestSequenceId(Map<?,Long> sequenceids) | 
| (package private) ConcurrentMap<ImmutableByteArray,Long> | getOrCreateLowestSequenceIds(byte[] encodedRegionName) | 
| (package private) void | onRegionClose(byte[] encodedRegionName)Clear all the records of the given region as it is going to be closed. | 
| (package private) Map<byte[],Long> | resetHighest()Reset the accounting of highest sequenceid by regionname. | 
| (package private) Long | startCacheFlush(byte[] encodedRegionName,
               Map<byte[],Long> familyToSeq) | 
| (package private) Long | startCacheFlush(byte[] encodedRegionName,
               Set<byte[]> families) | 
| (package private) void | update(byte[] encodedRegionName,
      Set<byte[]> families,
      long sequenceid,
      boolean lowest)We've been passed a new sequenceid for the region. | 
| (package private) void | updateStore(byte[] encodedRegionName,
           byte[] familyName,
           Long sequenceId,
           boolean onlyIfGreater)Update the store sequence id, e.g., upon executing in-memory compaction | 
private static final org.slf4j.Logger LOG
private final Object tieLock
flushingSequenceIds and
 lowestUnflushedSequenceIds Maps. lowestUnflushedSequenceIds has the
 lowest outstanding sequence ids EXCEPT when flushing. When we flush, the current
 lowest set for the region/column family are moved (atomically because of this lock) to
 flushingSequenceIds.
 
 The two Maps are tied by this locking object EXCEPT when we go to update the lowest
 entry; see lowestUnflushedSequenceIds. In here is a putIfAbsent call on
 lowestUnflushedSequenceIds. In this latter case, we will add this lowest
 sequence id if we find that there is no entry for the current column family. There will be no
 entry only if we just came up OR we have moved aside current set of lowest sequence ids
 because the current set are being flushed (by putting them into flushingSequenceIds).
 This is how we pick up the next 'lowest' sequence id per region per column family to be used
 figuring what is in the next flush.
private final ConcurrentMap<byte[],ConcurrentMap<ImmutableByteArray,Long>> lowestUnflushedSequenceIds
When we flush, the current lowest sequence ids get cleared and added to
 flushingSequenceIds. The next append that comes in, is then added
 here to lowestUnflushedSequenceIds as the next lowest sequenceid.
 
If flush fails, currently server is aborted so no need to restore previous sequence ids.
Needs to be concurrent Maps because we use putIfAbsent updating oldest.
private final Map<byte[],Map<ImmutableByteArray,Long>> flushingSequenceIds
lowestUnflushedSequenceIds while the lock tieLock is held
 (so movement between the Maps is atomic).private Map<byte[],Long> highestSequenceIds
Map of region encoded names to the latest/highest region sequence id. Updated on each call to append.
 This map uses byte[] as the key, and uses reference equality. It works in our use case as we
 use RegionInfo.getEncodedNameAsBytes() as keys. For a
 given region, it always returns the same array.
 
SequenceIdAccounting()
long getLowestSequenceId(byte[] encodedRegionName)
encodedRegionName. Will
 return HConstants.NO_SEQNUM when none.long getLowestSequenceId(byte[] encodedRegionName, byte[] familyName)
encodedRegionname and
         familyName. Returned sequenceid may be for an edit currently being
         flushed.Map<byte[],Long> resetHighest()
void update(byte[] encodedRegionName, Set<byte[]> families, long sequenceid, boolean lowest)
encodedRegionName - families - sequenceid - lowest - Whether to keep running account of oldest sequence id.void onRegionClose(byte[] encodedRegionName)
void updateStore(byte[] encodedRegionName, byte[] familyName, Long sequenceId, boolean onlyIfGreater)
ConcurrentMap<ImmutableByteArray,Long> getOrCreateLowestSequenceIds(byte[] encodedRegionName)
private static long getLowestSequenceId(Map<?,Long> sequenceids)
sequenceids - Map to search for lowest value.sequenceids.private <T extends Map<?,Long>> Map<byte[],Long> flattenToLowestSequenceId(Map<byte[],T> src)
src - src but instead of a Map for a value, it
         instead has found the smallest sequence id and it returns that as the value instead.Long startCacheFlush(byte[] encodedRegionName, Set<byte[]> families)
encodedRegionName - Region to flush.families - Families to flush. May be a subset of all families in the region.HConstants.NO_SEQNUM if we are flushing the whole region OR if
 we are flushing a subset of all families but there are no edits in those families not
 being flushed; in other words, this is effectively same as a flush of all of the region
 though we were passed a subset of regions. Otherwise, it returns the sequence id of the
 oldest/lowest outstanding edit.Long startCacheFlush(byte[] encodedRegionName, Map<byte[],Long> familyToSeq)
void completeCacheFlush(byte[] encodedRegionName, long maxFlushedSeqId)
void abortCacheFlush(byte[] encodedRegionName)
boolean areAllLower(Map<byte[],Long> sequenceids)
sequenceids are lower -- i.e. earlier -- than any outstanding
 sequenceids, sequenceids we are holding on to in this accounting instance.sequenceids - Keyed by encoded region name. Cannot be null (doesn't make sense for it to
          be null).Map<byte[],List<byte[]>> findLower(Map<byte[],Long> sequenceids)
lowestUnflushedSequenceIds. If a region in
 lowestUnflushedSequenceIds has a sequence id less than that passed in
 sequenceids then return it.sequenceids - Sequenceids keyed by encoded region name.Copyright © 2007–2021 The Apache Software Foundation. All rights reserved.