@InterfaceAudience.Private public class CompactionPipeline extends Object
CompactingMemStore
, is a FIFO queue of segments.
It supports pushing a segment at the head of the pipeline and removing a segment from the
tail when it is flushed to disk.
It also supports swap method to allow the in-memory compaction swap a subset of the segments
at the tail of the pipeline with a new (compacted) one. This swap succeeds only if the version
number passed with the list of segments to swap is the same as the current version of the
pipeline.
Essentially, there are two methods which can change the structure of the pipeline: pushHead()
and swap(), the later is used both by a flush to disk and by an in-memory compaction.
The pipeline version is updated by swap(); it allows to identify conflicting operations at the
suffix of the pipeline.
The synchronization model is copy-on-write. Methods which change the structure of the
pipeline (pushHead() and swap()) apply their changes in the context of a lock. They also make
a read-only copy of the pipeline's list. Read methods read from a read-only copy. If a read
method accesses the read-only copy more than once it makes a local copy of it
to ensure it accesses the same copy.
The methods getVersionedList(), getVersionedTail(), and flattenOneSegment() are also
protected by a lock since they need to have a consistent (atomic) view of the pipeline list
and version number.Modifier and Type | Field and Description |
---|---|
static long |
DEEP_OVERHEAD |
static long |
FIXED_OVERHEAD |
private static org.slf4j.Logger |
LOG |
private LinkedList<ImmutableSegment> |
pipeline |
private LinkedList<ImmutableSegment> |
readOnlyCopy |
private RegionServicesForStores |
region |
private long |
version |
Constructor and Description |
---|
CompactionPipeline(RegionServicesForStores region) |
Modifier and Type | Method and Description |
---|---|
private boolean |
addFirst(ImmutableSegment segment) |
boolean |
flattenOneSegment(long requesterVersion,
CompactingMemStore.IndexType idxType,
MemStoreCompactionStrategy.Action action)
If the caller holds the current version, go over the the pipeline and try to flatten each
segment.
|
long |
getMinSequenceId() |
MemStoreSize |
getPipelineSize() |
List<? extends Segment> |
getSegments() |
private static int |
getSegmentsCellsCount(List<? extends Segment> list) |
private static long |
getSegmentsHeapSize(List<? extends Segment> list) |
private static long |
getSegmentsKeySize(List<? extends Segment> list) |
private static long |
getSegmentsOffHeapSize(List<? extends Segment> list) |
Segment |
getTail() |
MemStoreSize |
getTailSize() |
VersionedSegmentsList |
getVersionedList() |
VersionedSegmentsList |
getVersionedTail() |
boolean |
isEmpty() |
boolean |
pushHead(MutableSegment segment) |
private void |
replaceAtIndex(int idx,
ImmutableSegment newSegment) |
long |
size() |
boolean |
swap(VersionedSegmentsList versionedList,
ImmutableSegment segment,
boolean closeSuffix,
boolean updateRegionSize)
Swaps the versioned list at the tail of the pipeline with a new segment.
|
private void |
swapSuffix(List<? extends Segment> suffix,
ImmutableSegment segment,
boolean closeSegmentsInSuffix) |
private boolean |
validateSuffixList(LinkedList<ImmutableSegment> suffix) |
private static final org.slf4j.Logger LOG
public static final long FIXED_OVERHEAD
public static final long DEEP_OVERHEAD
private final RegionServicesForStores region
private final LinkedList<ImmutableSegment> pipeline
private volatile LinkedList<ImmutableSegment> readOnlyCopy
private volatile long version
public CompactionPipeline(RegionServicesForStores region)
public boolean pushHead(MutableSegment segment)
public VersionedSegmentsList getVersionedList()
public VersionedSegmentsList getVersionedTail()
public boolean swap(VersionedSegmentsList versionedList, ImmutableSegment segment, boolean closeSuffix, boolean updateRegionSize)
versionedList
- suffix of the pipeline to be replaced can be tail or all the pipelinesegment
- new segment to replace the suffix. Can be null if the suffix just needs to be
removed.closeSuffix
- whether to close the suffix (to release memory), as part of swapping it out
During index merge op this will be false and for compaction it will be true.updateRegionSize
- whether to update the region size. Update the region size,
when the pipeline is swapped as part of in-memory-flush and
further merge/compaction. Don't update the region size when the
swap is result of the snapshot (flush-to-disk).private static long getSegmentsHeapSize(List<? extends Segment> list)
private static long getSegmentsOffHeapSize(List<? extends Segment> list)
private static long getSegmentsKeySize(List<? extends Segment> list)
private static int getSegmentsCellsCount(List<? extends Segment> list)
public boolean flattenOneSegment(long requesterVersion, CompactingMemStore.IndexType idxType, MemStoreCompactionStrategy.Action action)
public boolean isEmpty()
public List<? extends Segment> getSegments()
public long size()
public long getMinSequenceId()
public MemStoreSize getTailSize()
public MemStoreSize getPipelineSize()
private void swapSuffix(List<? extends Segment> suffix, ImmutableSegment segment, boolean closeSegmentsInSuffix)
private void replaceAtIndex(int idx, ImmutableSegment newSegment)
private boolean addFirst(ImmutableSegment segment)
private boolean validateSuffixList(LinkedList<ImmutableSegment> suffix)
Copyright © 2007–2020 The Apache Software Foundation. All rights reserved.