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