3 * Licensed to the Apache Software Foundation (ASF) under one
4 * or more contributor license agreements. See the NOTICE file
5 * distributed with this work for additional information
6 * regarding copyright ownership. The ASF licenses this file
7 * to you under the Apache License, Version 2.0 (the
8 * "License"); you may not use this file except in compliance
9 * with the License. You may obtain a copy of the License at
11 * http://www.apache.org/licenses/LICENSE-2.0
13 * Unless required by applicable law or agreed to in writing, software
14 * distributed under the License is distributed on an "AS IS" BASIS,
15 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
16 * See the License for the specific language governing permissions and
17 * limitations under the License.
19 package org
.apache
.hadoop
.hbase
.regionserver
;
21 import java
.util
.Collections
;
22 import java
.util
.Iterator
;
23 import java
.util
.List
;
24 import java
.util
.Objects
;
25 import java
.util
.SortedSet
;
26 import java
.util
.concurrent
.atomic
.AtomicReference
;
27 import java
.util
.concurrent
.locks
.ReentrantReadWriteLock
;
29 import org
.apache
.hadoop
.hbase
.Cell
;
30 import org
.apache
.hadoop
.hbase
.CellComparator
;
31 import org
.apache
.hadoop
.hbase
.KeyValue
;
32 import org
.apache
.hadoop
.hbase
.io
.TimeRange
;
33 import org
.apache
.hadoop
.hbase
.util
.Bytes
;
34 import org
.apache
.hadoop
.hbase
.util
.ClassSize
;
35 import org
.apache
.yetus
.audience
.InterfaceAudience
;
36 import org
.slf4j
.Logger
;
39 * This is an abstraction of a segment maintained in a memstore, e.g., the active
40 * cell set or its snapshot.
42 * This abstraction facilitates the management of the compaction pipeline and the shifts of these
43 * segments from active set to snapshot set in the default implementation.
45 @InterfaceAudience.Private
46 public abstract class Segment
implements MemStoreSizing
{
48 public final static long FIXED_OVERHEAD
= ClassSize
.align(ClassSize
.OBJECT
49 + 6 * ClassSize
.REFERENCE
// cellSet, comparator, updatesLock, memStoreLAB, memStoreSizing,
50 // and timeRangeTracker
51 + Bytes
.SIZEOF_LONG
// minSequenceId
52 + Bytes
.SIZEOF_BOOLEAN
); // tagsPresent
53 public final static long DEEP_OVERHEAD
= FIXED_OVERHEAD
+ ClassSize
.ATOMIC_REFERENCE
54 + ClassSize
.CELL_SET
+ 2 * ClassSize
.ATOMIC_LONG
55 + ClassSize
.REENTRANT_LOCK
;
57 private AtomicReference
<CellSet
> cellSet
= new AtomicReference
<>();
58 private final CellComparator comparator
;
59 private ReentrantReadWriteLock updatesLock
;
60 protected long minSequenceId
;
61 private MemStoreLAB memStoreLAB
;
62 // Sum of sizes of all Cells added to this Segment. Cell's HeapSize is considered. This is not
63 // including the heap overhead of this class.
64 protected final MemStoreSizing memStoreSizing
;
65 protected final TimeRangeTracker timeRangeTracker
;
66 protected volatile boolean tagsPresent
;
68 // Empty constructor to be used when Segment is used as interface,
69 // and there is no need in true Segments state
70 protected Segment(CellComparator comparator
, TimeRangeTracker trt
) {
71 this.comparator
= comparator
;
72 // Do we need to be thread safe always? What if ImmutableSegment?
73 // DITTO for the TimeRangeTracker below.
74 this.memStoreSizing
= new ThreadSafeMemStoreSizing();
75 this.timeRangeTracker
= trt
;
78 protected Segment(CellComparator comparator
, List
<ImmutableSegment
> segments
,
79 TimeRangeTracker trt
) {
84 for (Segment segment
: segments
) {
85 MemStoreSize memStoreSize
= segment
.getMemStoreSize();
86 dataSize
+= memStoreSize
.getDataSize();
87 heapSize
+= memStoreSize
.getHeapSize();
88 OffHeapSize
+= memStoreSize
.getOffHeapSize();
89 cellsCount
+= memStoreSize
.getCellsCount();
91 this.comparator
= comparator
;
92 this.updatesLock
= new ReentrantReadWriteLock();
93 // Do we need to be thread safe always? What if ImmutableSegment?
94 // DITTO for the TimeRangeTracker below.
95 this.memStoreSizing
= new ThreadSafeMemStoreSizing(dataSize
, heapSize
, OffHeapSize
, cellsCount
);
96 this.timeRangeTracker
= trt
;
99 // This constructor is used to create empty Segments.
100 protected Segment(CellSet cellSet
, CellComparator comparator
, MemStoreLAB memStoreLAB
, TimeRangeTracker trt
) {
101 this.cellSet
.set(cellSet
);
102 this.comparator
= comparator
;
103 this.updatesLock
= new ReentrantReadWriteLock();
104 this.minSequenceId
= Long
.MAX_VALUE
;
105 this.memStoreLAB
= memStoreLAB
;
106 // Do we need to be thread safe always? What if ImmutableSegment?
107 // DITTO for the TimeRangeTracker below.
108 this.memStoreSizing
= new ThreadSafeMemStoreSizing();
109 this.tagsPresent
= false;
110 this.timeRangeTracker
= trt
;
113 protected Segment(Segment segment
) {
114 this.cellSet
.set(segment
.getCellSet());
115 this.comparator
= segment
.getComparator();
116 this.updatesLock
= segment
.getUpdatesLock();
117 this.minSequenceId
= segment
.getMinSequenceId();
118 this.memStoreLAB
= segment
.getMemStoreLAB();
119 this.memStoreSizing
= segment
.memStoreSizing
;
120 this.tagsPresent
= segment
.isTagsPresent();
121 this.timeRangeTracker
= segment
.getTimeRangeTracker();
125 * Creates the scanner for the given read point
126 * @return a scanner for the given read point
128 protected KeyValueScanner
getScanner(long readPoint
) {
129 return new SegmentScanner(this, readPoint
);
132 public List
<KeyValueScanner
> getScanners(long readPoint
) {
133 return Collections
.singletonList(new SegmentScanner(this, readPoint
));
137 * @return whether the segment has any cells
139 public boolean isEmpty() {
140 return getCellSet().isEmpty();
145 * Closing a segment before it is being discarded
147 public void close() {
148 if (this.memStoreLAB
!= null) {
149 this.memStoreLAB
.close();
151 // do not set MSLab to null as scanners may still be reading the data here and need to decrease
152 // the counter when they finish
156 * If the segment has a memory allocator the cell is being cloned to this space, and returned;
157 * otherwise the given cell is returned
159 * When a cell's size is too big (bigger than maxAlloc), it is not allocated on MSLAB.
160 * Since the process of flattening to CellChunkMap assumes that all cells
161 * are allocated on MSLAB, during this process, the input parameter
162 * forceCloneOfBigCell is set to 'true' and the cell is copied into MSLAB.
164 * @return either the given cell or its clone
166 public Cell
maybeCloneWithAllocator(Cell cell
, boolean forceCloneOfBigCell
) {
167 if (this.memStoreLAB
== null) {
172 if (forceCloneOfBigCell
) {
173 cellFromMslab
= this.memStoreLAB
.forceCopyOfBigCellInto(cell
);
175 cellFromMslab
= this.memStoreLAB
.copyCellInto(cell
);
177 return (cellFromMslab
!= null) ? cellFromMslab
: cell
;
181 * Get cell length after serialized in {@link KeyValue}
183 static int getCellLength(Cell cell
) {
184 return cell
.getSerializedSize();
187 public boolean shouldSeek(TimeRange tr
, long oldestUnexpiredTS
) {
189 && (tr
.isAllTime() || timeRangeTracker
.includesTimeRange(tr
))
190 && timeRangeTracker
.getMax() >= oldestUnexpiredTS
;
193 public boolean isTagsPresent() {
197 public void incScannerCount() {
198 if (this.memStoreLAB
!= null) {
199 this.memStoreLAB
.incScannerCount();
203 public void decScannerCount() {
204 if (this.memStoreLAB
!= null) {
205 this.memStoreLAB
.decScannerCount();
210 * Setting the CellSet of the segment - used only for flat immutable segment for setting
211 * immutable CellSet after its creation in immutable segment constructor
212 * @return this object
215 protected Segment
setCellSet(CellSet cellSetOld
, CellSet cellSetNew
) {
216 this.cellSet
.compareAndSet(cellSetOld
, cellSetNew
);
221 public MemStoreSize
getMemStoreSize() {
222 return this.memStoreSizing
.getMemStoreSize();
226 public long getDataSize() {
227 return this.memStoreSizing
.getDataSize();
231 public long getHeapSize() {
232 return this.memStoreSizing
.getHeapSize();
236 public long getOffHeapSize() {
237 return this.memStoreSizing
.getOffHeapSize();
241 public int getCellsCount() {
242 return memStoreSizing
.getCellsCount();
246 public long incMemStoreSize(long delta
, long heapOverhead
, long offHeapOverhead
, int cellsCount
) {
247 return this.memStoreSizing
.incMemStoreSize(delta
, heapOverhead
, offHeapOverhead
, cellsCount
);
250 public boolean sharedLock() {
251 return updatesLock
.readLock().tryLock();
254 public void sharedUnlock() {
255 updatesLock
.readLock().unlock();
258 public void waitForUpdates() {
259 if(!updatesLock
.isWriteLocked()) {
260 updatesLock
.writeLock().lock();
265 public boolean compareAndSetDataSize(long expected
, long updated
) {
266 return memStoreSizing
.compareAndSetDataSize(expected
, updated
);
269 public long getMinSequenceId() {
270 return minSequenceId
;
273 public TimeRangeTracker
getTimeRangeTracker() {
274 return this.timeRangeTracker
;
277 //*** Methods for SegmentsScanner
279 return getCellSet().last();
282 public Iterator
<Cell
> iterator() {
283 return getCellSet().iterator();
286 public SortedSet
<Cell
> headSet(Cell firstKeyOnRow
) {
287 return getCellSet().headSet(firstKeyOnRow
);
290 public int compare(Cell left
, Cell right
) {
291 return getComparator().compare(left
, right
);
294 public int compareRows(Cell left
, Cell right
) {
295 return getComparator().compareRows(left
, right
);
299 * @return a set of all cells in the segment
301 protected CellSet
getCellSet() {
302 return cellSet
.get();
306 * Returns the Cell comparator used by this segment
307 * @return the Cell comparator used by this segment
309 protected CellComparator
getComparator() {
313 protected void internalAdd(Cell cell
, boolean mslabUsed
, MemStoreSizing memstoreSizing
,
314 boolean sizeAddedPreOperation
) {
315 boolean succ
= getCellSet().add(cell
);
316 updateMetaInfo(cell
, succ
, mslabUsed
, memstoreSizing
, sizeAddedPreOperation
);
319 protected void updateMetaInfo(Cell cellToAdd
, boolean succ
, boolean mslabUsed
,
320 MemStoreSizing memstoreSizing
, boolean sizeAddedPreOperation
) {
322 long cellSize
= getCellLength(cellToAdd
);
323 int cellsCount
= succ ?
1 : 0;
324 // If there's already a same cell in the CellSet and we are using MSLAB, we must count in the
325 // MSLAB allocation size as well, or else there will be memory leak (occupied heap size larger
326 // than the counted number)
327 if (succ
|| mslabUsed
) {
330 if (sizeAddedPreOperation
) {
333 long heapSize
= heapSizeChange(cellToAdd
, succ
|| mslabUsed
);
334 long offHeapSize
= offHeapSizeChange(cellToAdd
, succ
|| mslabUsed
);
335 incMemStoreSize(delta
, heapSize
, offHeapSize
, cellsCount
);
336 if (memstoreSizing
!= null) {
337 memstoreSizing
.incMemStoreSize(delta
, heapSize
, offHeapSize
, cellsCount
);
339 getTimeRangeTracker().includeTimestamp(cellToAdd
);
340 minSequenceId
= Math
.min(minSequenceId
, cellToAdd
.getSequenceId());
341 // In no tags case this NoTagsKeyValue.getTagsLength() is a cheap call.
342 // When we use ACL CP or Visibility CP which deals with Tags during
343 // mutation, the TagRewriteCell.getTagsLength() is a cheaper call. We do not
344 // parse the byte[] to identify the tags length.
345 if (cellToAdd
.getTagsLength() > 0) {
350 protected void updateMetaInfo(Cell cellToAdd
, boolean succ
, MemStoreSizing memstoreSizing
) {
351 updateMetaInfo(cellToAdd
, succ
, (getMemStoreLAB()!=null), memstoreSizing
, false);
355 * @return The increase in heap size because of this cell addition. This includes this cell POJO's
356 * heap size itself and additional overhead because of addition on to CSLM.
358 protected long heapSizeChange(Cell cell
, boolean allocated
) {
361 boolean onHeap
= true;
362 MemStoreLAB memStoreLAB
= getMemStoreLAB();
363 if(memStoreLAB
!= null) {
364 onHeap
= memStoreLAB
.isOnHeap();
366 res
+= indexEntryOnHeapSize(onHeap
);
368 res
+= cell
.heapSize();
370 res
= ClassSize
.align(res
);
375 protected long offHeapSizeChange(Cell cell
, boolean allocated
) {
378 boolean offHeap
= false;
379 MemStoreLAB memStoreLAB
= getMemStoreLAB();
380 if(memStoreLAB
!= null) {
381 offHeap
= memStoreLAB
.isOffHeap();
383 res
+= indexEntryOffHeapSize(offHeap
);
385 res
+= cell
.heapSize();
387 res
= ClassSize
.align(res
);
392 protected long indexEntryOnHeapSize(boolean onHeap
) {
393 // in most cases index is allocated on-heap
394 // override this method when it is not always the case, e.g., in CCM
395 return indexEntrySize();
398 protected long indexEntryOffHeapSize(boolean offHeap
) {
399 // in most cases index is allocated on-heap
400 // override this method when it is not always the case, e.g., in CCM
404 protected abstract long indexEntrySize();
407 * Returns a subset of the segment cell set, which starts with the given cell
408 * @param firstCell a cell in the segment
409 * @return a subset of the segment cell set, which starts with the given cell
411 protected SortedSet
<Cell
> tailSet(Cell firstCell
) {
412 return getCellSet().tailSet(firstCell
);
415 MemStoreLAB
getMemStoreLAB() {
421 * Dumps all cells of the segment into the given log
423 void dump(Logger log
) {
424 for (Cell cell
: getCellSet()) {
425 log
.debug(Objects
.toString(cell
));
430 public String
toString() {
431 String res
= "type=" + this.getClass().getSimpleName() + ", ";
432 res
+= "empty=" + (isEmpty()?
"yes": "no") + ", ";
433 res
+= "cellCount=" + getCellsCount() + ", ";
434 res
+= "cellSize=" + getDataSize() + ", ";
435 res
+= "totalHeapSize=" + getHeapSize() + ", ";
436 res
+= "min timestamp=" + timeRangeTracker
.getMin() + ", ";
437 res
+= "max timestamp=" + timeRangeTracker
.getMax();
441 private ReentrantReadWriteLock
getUpdatesLock() {