001/* 002 * 003 * Licensed to the Apache Software Foundation (ASF) under one 004 * or more contributor license agreements. See the NOTICE file 005 * distributed with this work for additional information 006 * regarding copyright ownership. The ASF licenses this file 007 * to you under the Apache License, Version 2.0 (the 008 * "License"); you may not use this file except in compliance 009 * with the License. You may obtain a copy of the License at 010 * 011 * http://www.apache.org/licenses/LICENSE-2.0 012 * 013 * Unless required by applicable law or agreed to in writing, software 014 * distributed under the License is distributed on an "AS IS" BASIS, 015 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. 016 * See the License for the specific language governing permissions and 017 * limitations under the License. 018 */ 019package org.apache.hadoop.hbase.regionserver; 020 021import java.util.Collections; 022import java.util.Iterator; 023import java.util.List; 024import java.util.Objects; 025import java.util.SortedSet; 026import java.util.concurrent.atomic.AtomicReference; 027import java.util.concurrent.locks.ReentrantReadWriteLock; 028 029import org.apache.hadoop.hbase.Cell; 030import org.apache.hadoop.hbase.CellComparator; 031import org.apache.hadoop.hbase.KeyValue; 032import org.apache.hadoop.hbase.io.TimeRange; 033import org.apache.hadoop.hbase.util.Bytes; 034import org.apache.hadoop.hbase.util.ClassSize; 035import org.apache.yetus.audience.InterfaceAudience; 036import org.slf4j.Logger; 037import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting; 038 039/** 040 * This is an abstraction of a segment maintained in a memstore, e.g., the active 041 * cell set or its snapshot. 042 * 043 * This abstraction facilitates the management of the compaction pipeline and the shifts of these 044 * segments from active set to snapshot set in the default implementation. 045 */ 046@InterfaceAudience.Private 047public abstract class Segment implements MemStoreSizing { 048 049 public final static long FIXED_OVERHEAD = ClassSize.align(ClassSize.OBJECT 050 + 6 * ClassSize.REFERENCE // cellSet, comparator, updatesLock, memStoreLAB, memStoreSizing, 051 // and timeRangeTracker 052 + Bytes.SIZEOF_LONG // minSequenceId 053 + Bytes.SIZEOF_BOOLEAN); // tagsPresent 054 public final static long DEEP_OVERHEAD = FIXED_OVERHEAD + ClassSize.ATOMIC_REFERENCE 055 + ClassSize.CELL_SET + 2 * ClassSize.ATOMIC_LONG 056 + ClassSize.REENTRANT_LOCK; 057 058 private AtomicReference<CellSet> cellSet= new AtomicReference<>(); 059 private final CellComparator comparator; 060 private ReentrantReadWriteLock updatesLock; 061 protected long minSequenceId; 062 private MemStoreLAB memStoreLAB; 063 // Sum of sizes of all Cells added to this Segment. Cell's HeapSize is considered. This is not 064 // including the heap overhead of this class. 065 protected final MemStoreSizing memStoreSizing; 066 protected final TimeRangeTracker timeRangeTracker; 067 protected volatile boolean tagsPresent; 068 069 // Empty constructor to be used when Segment is used as interface, 070 // and there is no need in true Segments state 071 protected Segment(CellComparator comparator, TimeRangeTracker trt) { 072 this.comparator = comparator; 073 // Do we need to be thread safe always? What if ImmutableSegment? 074 // DITTO for the TimeRangeTracker below. 075 this.memStoreSizing = new ThreadSafeMemStoreSizing(); 076 this.timeRangeTracker = trt; 077 } 078 079 protected Segment(CellComparator comparator, List<ImmutableSegment> segments, 080 TimeRangeTracker trt) { 081 long dataSize = 0; 082 long heapSize = 0; 083 long OffHeapSize = 0; 084 int cellsCount = 0; 085 for (Segment segment : segments) { 086 MemStoreSize memStoreSize = segment.getMemStoreSize(); 087 dataSize += memStoreSize.getDataSize(); 088 heapSize += memStoreSize.getHeapSize(); 089 OffHeapSize += memStoreSize.getOffHeapSize(); 090 cellsCount += memStoreSize.getCellsCount(); 091 } 092 this.comparator = comparator; 093 this.updatesLock = new ReentrantReadWriteLock(); 094 // Do we need to be thread safe always? What if ImmutableSegment? 095 // DITTO for the TimeRangeTracker below. 096 this.memStoreSizing = new ThreadSafeMemStoreSizing(dataSize, heapSize, OffHeapSize, cellsCount); 097 this.timeRangeTracker = trt; 098 } 099 100 // This constructor is used to create empty Segments. 101 protected Segment(CellSet cellSet, CellComparator comparator, MemStoreLAB memStoreLAB, TimeRangeTracker trt) { 102 this.cellSet.set(cellSet); 103 this.comparator = comparator; 104 this.updatesLock = new ReentrantReadWriteLock(); 105 this.minSequenceId = Long.MAX_VALUE; 106 this.memStoreLAB = memStoreLAB; 107 // Do we need to be thread safe always? What if ImmutableSegment? 108 // DITTO for the TimeRangeTracker below. 109 this.memStoreSizing = new ThreadSafeMemStoreSizing(); 110 this.tagsPresent = false; 111 this.timeRangeTracker = trt; 112 } 113 114 protected Segment(Segment segment) { 115 this.cellSet.set(segment.getCellSet()); 116 this.comparator = segment.getComparator(); 117 this.updatesLock = segment.getUpdatesLock(); 118 this.minSequenceId = segment.getMinSequenceId(); 119 this.memStoreLAB = segment.getMemStoreLAB(); 120 this.memStoreSizing = segment.memStoreSizing; 121 this.tagsPresent = segment.isTagsPresent(); 122 this.timeRangeTracker = segment.getTimeRangeTracker(); 123 } 124 125 /** 126 * Creates the scanner for the given read point 127 * @return a scanner for the given read point 128 */ 129 protected KeyValueScanner getScanner(long readPoint) { 130 return new SegmentScanner(this, readPoint); 131 } 132 133 public List<KeyValueScanner> getScanners(long readPoint) { 134 return Collections.singletonList(new SegmentScanner(this, readPoint)); 135 } 136 137 /** 138 * @return whether the segment has any cells 139 */ 140 public boolean isEmpty() { 141 return getCellSet().isEmpty(); 142 } 143 144 145 /** 146 * Closing a segment before it is being discarded 147 */ 148 public void close() { 149 if (this.memStoreLAB != null) { 150 this.memStoreLAB.close(); 151 } 152 // do not set MSLab to null as scanners may still be reading the data here and need to decrease 153 // the counter when they finish 154 } 155 156 /** 157 * If the segment has a memory allocator the cell is being cloned to this space, and returned; 158 * otherwise the given cell is returned 159 * 160 * When a cell's size is too big (bigger than maxAlloc), it is not allocated on MSLAB. 161 * Since the process of flattening to CellChunkMap assumes that all cells 162 * are allocated on MSLAB, during this process, the input parameter 163 * forceCloneOfBigCell is set to 'true' and the cell is copied into MSLAB. 164 * 165 * @return either the given cell or its clone 166 */ 167 public Cell maybeCloneWithAllocator(Cell cell, boolean forceCloneOfBigCell) { 168 if (this.memStoreLAB == null) { 169 return cell; 170 } 171 172 Cell cellFromMslab; 173 if (forceCloneOfBigCell) { 174 cellFromMslab = this.memStoreLAB.forceCopyOfBigCellInto(cell); 175 } else { 176 cellFromMslab = this.memStoreLAB.copyCellInto(cell); 177 } 178 return (cellFromMslab != null) ? cellFromMslab : cell; 179 } 180 181 /** 182 * Get cell length after serialized in {@link KeyValue} 183 */ 184 @VisibleForTesting 185 static int getCellLength(Cell cell) { 186 return cell.getSerializedSize(); 187 } 188 189 public boolean shouldSeek(TimeRange tr, long oldestUnexpiredTS) { 190 return !isEmpty() 191 && (tr.isAllTime() || timeRangeTracker.includesTimeRange(tr)) 192 && timeRangeTracker.getMax() >= oldestUnexpiredTS; 193 } 194 195 public boolean isTagsPresent() { 196 return tagsPresent; 197 } 198 199 public void incScannerCount() { 200 if (this.memStoreLAB != null) { 201 this.memStoreLAB.incScannerCount(); 202 } 203 } 204 205 public void decScannerCount() { 206 if (this.memStoreLAB != null) { 207 this.memStoreLAB.decScannerCount(); 208 } 209 } 210 211 /** 212 * Setting the CellSet of the segment - used only for flat immutable segment for setting 213 * immutable CellSet after its creation in immutable segment constructor 214 * @return this object 215 */ 216 217 protected Segment setCellSet(CellSet cellSetOld, CellSet cellSetNew) { 218 this.cellSet.compareAndSet(cellSetOld, cellSetNew); 219 return this; 220 } 221 222 @Override 223 public MemStoreSize getMemStoreSize() { 224 return this.memStoreSizing.getMemStoreSize(); 225 } 226 227 @Override 228 public long getDataSize() { 229 return this.memStoreSizing.getDataSize(); 230 } 231 232 @Override 233 public long getHeapSize() { 234 return this.memStoreSizing.getHeapSize(); 235 } 236 237 @Override 238 public long getOffHeapSize() { 239 return this.memStoreSizing.getOffHeapSize(); 240 } 241 242 @Override 243 public int getCellsCount() { 244 return memStoreSizing.getCellsCount(); 245 } 246 247 @Override 248 public long incMemStoreSize(long delta, long heapOverhead, long offHeapOverhead, int cellsCount) { 249 return this.memStoreSizing.incMemStoreSize(delta, heapOverhead, offHeapOverhead, cellsCount); 250 } 251 252 public boolean sharedLock() { 253 return updatesLock.readLock().tryLock(); 254 } 255 256 public void sharedUnlock() { 257 updatesLock.readLock().unlock(); 258 } 259 260 public void waitForUpdates() { 261 if(!updatesLock.isWriteLocked()) { 262 updatesLock.writeLock().lock(); 263 } 264 } 265 266 @Override 267 public boolean compareAndSetDataSize(long expected, long updated) { 268 return memStoreSizing.compareAndSetDataSize(expected, updated); 269 } 270 271 public long getMinSequenceId() { 272 return minSequenceId; 273 } 274 275 public TimeRangeTracker getTimeRangeTracker() { 276 return this.timeRangeTracker; 277 } 278 279 //*** Methods for SegmentsScanner 280 public Cell last() { 281 return getCellSet().last(); 282 } 283 284 public Iterator<Cell> iterator() { 285 return getCellSet().iterator(); 286 } 287 288 public SortedSet<Cell> headSet(Cell firstKeyOnRow) { 289 return getCellSet().headSet(firstKeyOnRow); 290 } 291 292 public int compare(Cell left, Cell right) { 293 return getComparator().compare(left, right); 294 } 295 296 public int compareRows(Cell left, Cell right) { 297 return getComparator().compareRows(left, right); 298 } 299 300 /** 301 * @return a set of all cells in the segment 302 */ 303 protected CellSet getCellSet() { 304 return cellSet.get(); 305 } 306 307 /** 308 * Returns the Cell comparator used by this segment 309 * @return the Cell comparator used by this segment 310 */ 311 protected CellComparator getComparator() { 312 return comparator; 313 } 314 315 protected void internalAdd(Cell cell, boolean mslabUsed, MemStoreSizing memstoreSizing, 316 boolean sizeAddedPreOperation) { 317 boolean succ = getCellSet().add(cell); 318 updateMetaInfo(cell, succ, mslabUsed, memstoreSizing, sizeAddedPreOperation); 319 } 320 321 protected void updateMetaInfo(Cell cellToAdd, boolean succ, boolean mslabUsed, 322 MemStoreSizing memstoreSizing, boolean sizeAddedPreOperation) { 323 long delta = 0; 324 long cellSize = getCellLength(cellToAdd); 325 int cellsCount = succ ? 1 : 0; 326 // If there's already a same cell in the CellSet and we are using MSLAB, we must count in the 327 // MSLAB allocation size as well, or else there will be memory leak (occupied heap size larger 328 // than the counted number) 329 if (succ || mslabUsed) { 330 delta = cellSize; 331 } 332 if (sizeAddedPreOperation) { 333 delta -= cellSize; 334 } 335 long heapSize = heapSizeChange(cellToAdd, succ || mslabUsed); 336 long offHeapSize = offHeapSizeChange(cellToAdd, succ || mslabUsed); 337 incMemStoreSize(delta, heapSize, offHeapSize, cellsCount); 338 if (memstoreSizing != null) { 339 memstoreSizing.incMemStoreSize(delta, heapSize, offHeapSize, cellsCount); 340 } 341 getTimeRangeTracker().includeTimestamp(cellToAdd); 342 minSequenceId = Math.min(minSequenceId, cellToAdd.getSequenceId()); 343 // In no tags case this NoTagsKeyValue.getTagsLength() is a cheap call. 344 // When we use ACL CP or Visibility CP which deals with Tags during 345 // mutation, the TagRewriteCell.getTagsLength() is a cheaper call. We do not 346 // parse the byte[] to identify the tags length. 347 if (cellToAdd.getTagsLength() > 0) { 348 tagsPresent = true; 349 } 350 } 351 352 protected void updateMetaInfo(Cell cellToAdd, boolean succ, MemStoreSizing memstoreSizing) { 353 updateMetaInfo(cellToAdd, succ, (getMemStoreLAB()!=null), memstoreSizing, false); 354 } 355 356 /** 357 * @return The increase in heap size because of this cell addition. This includes this cell POJO's 358 * heap size itself and additional overhead because of addition on to CSLM. 359 */ 360 protected long heapSizeChange(Cell cell, boolean allocated) { 361 long res = 0; 362 if (allocated) { 363 boolean onHeap = true; 364 MemStoreLAB memStoreLAB = getMemStoreLAB(); 365 if(memStoreLAB != null) { 366 onHeap = memStoreLAB.isOnHeap(); 367 } 368 res += indexEntryOnHeapSize(onHeap); 369 if(onHeap) { 370 res += cell.heapSize(); 371 } 372 res = ClassSize.align(res); 373 } 374 return res; 375 } 376 377 protected long offHeapSizeChange(Cell cell, boolean allocated) { 378 long res = 0; 379 if (allocated) { 380 boolean offHeap = false; 381 MemStoreLAB memStoreLAB = getMemStoreLAB(); 382 if(memStoreLAB != null) { 383 offHeap = memStoreLAB.isOffHeap(); 384 } 385 res += indexEntryOffHeapSize(offHeap); 386 if(offHeap) { 387 res += cell.heapSize(); 388 } 389 res = ClassSize.align(res); 390 } 391 return res; 392 } 393 394 protected long indexEntryOnHeapSize(boolean onHeap) { 395 // in most cases index is allocated on-heap 396 // override this method when it is not always the case, e.g., in CCM 397 return indexEntrySize(); 398 } 399 400 protected long indexEntryOffHeapSize(boolean offHeap) { 401 // in most cases index is allocated on-heap 402 // override this method when it is not always the case, e.g., in CCM 403 return 0; 404 } 405 406 protected abstract long indexEntrySize(); 407 408 /** 409 * Returns a subset of the segment cell set, which starts with the given cell 410 * @param firstCell a cell in the segment 411 * @return a subset of the segment cell set, which starts with the given cell 412 */ 413 protected SortedSet<Cell> tailSet(Cell firstCell) { 414 return getCellSet().tailSet(firstCell); 415 } 416 417 @VisibleForTesting 418 MemStoreLAB getMemStoreLAB() { 419 return memStoreLAB; 420 } 421 422 // Debug methods 423 /** 424 * Dumps all cells of the segment into the given log 425 */ 426 void dump(Logger log) { 427 for (Cell cell: getCellSet()) { 428 log.debug(Objects.toString(cell)); 429 } 430 } 431 432 @Override 433 public String toString() { 434 String res = "type=" + this.getClass().getSimpleName() + ", "; 435 res += "empty=" + (isEmpty()? "yes": "no") + ", "; 436 res += "cellCount=" + getCellsCount() + ", "; 437 res += "cellSize=" + getDataSize() + ", "; 438 res += "totalHeapSize=" + getHeapSize() + ", "; 439 res += "min timestamp=" + timeRangeTracker.getMin() + ", "; 440 res += "max timestamp=" + timeRangeTracker.getMax(); 441 return res; 442 } 443 444 private ReentrantReadWriteLock getUpdatesLock() { 445 return updatesLock; 446 } 447}