001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.regionserver;
019
020import java.io.IOException;
021import java.nio.ByteBuffer;
022import org.apache.hadoop.hbase.ByteBufferKeyValue;
023import org.apache.hadoop.hbase.Cell;
024import org.apache.hadoop.hbase.CellComparator;
025import org.apache.hadoop.hbase.CellUtil;
026import org.apache.hadoop.hbase.ExtendedCell;
027import org.apache.hadoop.hbase.KeyValue;
028import org.apache.hadoop.hbase.util.ByteBufferUtils;
029import org.apache.hadoop.hbase.util.ClassSize;
030import org.apache.yetus.audience.InterfaceAudience;
031
032/**
033 * CellChunkImmutableSegment extends the API supported by a {@link Segment}, and
034 * {@link ImmutableSegment}. This immutable segment is working with CellSet with CellChunkMap
035 * delegatee.
036 */
037@InterfaceAudience.Private
038public class CellChunkImmutableSegment extends ImmutableSegment {
039
040  public static final long DEEP_OVERHEAD_CCM =
041    ImmutableSegment.DEEP_OVERHEAD + ClassSize.CELL_CHUNK_MAP;
042  public static final float INDEX_CHUNK_UNUSED_SPACE_PRECENTAGE = 0.1f;
043
044  ///////////////////// CONSTRUCTORS /////////////////////
045  /**
046   * ------------------------------------------------------------------------ C-tor to be used when
047   * new CellChunkImmutableSegment is built as a result of compaction/merge of a list of older
048   * ImmutableSegments. The given iterator returns the Cells that "survived" the compaction.
049   */
050  protected CellChunkImmutableSegment(CellComparator comparator, MemStoreSegmentsIterator iterator,
051    MemStoreLAB memStoreLAB, int numOfCells, MemStoreCompactionStrategy.Action action) {
052    super(null, comparator, memStoreLAB); // initialize the CellSet with NULL
053    long indexOverhead = DEEP_OVERHEAD_CCM;
054    // memStoreLAB cannot be null in this class
055    boolean onHeap = getMemStoreLAB().isOnHeap();
056    // initiate the heapSize with the size of the segment metadata
057    if (onHeap) {
058      incMemStoreSize(0, indexOverhead, 0, 0);
059    } else {
060      incMemStoreSize(0, 0, indexOverhead, 0);
061    }
062    // build the new CellSet based on CellArrayMap and update the CellSet of the new Segment
063    initializeCellSet(numOfCells, iterator, action);
064  }
065
066  /**
067   * ------------------------------------------------------------------------ C-tor to be used when
068   * new CellChunkImmutableSegment is built as a result of flattening of CSLMImmutableSegment The
069   * given iterator returns the Cells that "survived" the compaction.
070   */
071  protected CellChunkImmutableSegment(CSLMImmutableSegment segment, MemStoreSizing memstoreSizing,
072    MemStoreCompactionStrategy.Action action) {
073    super(segment); // initiailize the upper class
074    long indexOverhead = -CSLMImmutableSegment.DEEP_OVERHEAD_CSLM + DEEP_OVERHEAD_CCM;
075    // memStoreLAB cannot be null in this class
076    boolean onHeap = getMemStoreLAB().isOnHeap();
077    // initiate the heapSize with the size of the segment metadata
078    if (onHeap) {
079      incMemStoreSize(0, indexOverhead, 0, 0);
080      memstoreSizing.incMemStoreSize(0, indexOverhead, 0, 0);
081    } else {
082      incMemStoreSize(0, -CSLMImmutableSegment.DEEP_OVERHEAD_CSLM, DEEP_OVERHEAD_CCM, 0);
083      memstoreSizing.incMemStoreSize(0, -CSLMImmutableSegment.DEEP_OVERHEAD_CSLM, DEEP_OVERHEAD_CCM,
084        0);
085    }
086    int numOfCells = segment.getCellsCount();
087    // build the new CellSet based on CellChunkMap
088    reinitializeCellSet(numOfCells, segment.getScanner(Long.MAX_VALUE), segment.getCellSet(),
089      memstoreSizing, action);
090    // arrange the meta-data size, decrease all meta-data sizes related to SkipList;
091    // add sizes of CellChunkMap entry, decrease also Cell object sizes
092    // (reinitializeCellSet doesn't take the care for the sizes)
093    long newSegmentSizeDelta =
094      numOfCells * (indexEntrySize() - ClassSize.CONCURRENT_SKIPLISTMAP_ENTRY);
095    if (onHeap) {
096      incMemStoreSize(0, newSegmentSizeDelta, 0, 0);
097      memstoreSizing.incMemStoreSize(0, newSegmentSizeDelta, 0, 0);
098    } else {
099      incMemStoreSize(0, 0, newSegmentSizeDelta, 0);
100      memstoreSizing.incMemStoreSize(0, 0, newSegmentSizeDelta, 0);
101
102    }
103  }
104
105  @Override
106  protected long indexEntryOnHeapSize(boolean onHeap) {
107    if (onHeap) {
108      return indexEntrySize();
109    }
110    // else the index is allocated off-heap
111    return 0;
112  }
113
114  @Override
115  protected long indexEntryOffHeapSize(boolean offHeap) {
116    if (offHeap) {
117      return indexEntrySize();
118    }
119    // else the index is allocated on-heap
120    return 0;
121  }
122
123  @Override
124  protected long indexEntrySize() {
125    return ((long) ClassSize.CELL_CHUNK_MAP_ENTRY - KeyValue.FIXED_OVERHEAD);
126  }
127
128  @Override
129  protected boolean canBeFlattened() {
130    return false;
131  }
132
133  ///////////////////// PRIVATE METHODS /////////////////////
134  /*------------------------------------------------------------------------*/
135  // Create CellSet based on CellChunkMap from compacting iterator
136  private void initializeCellSet(int numOfCells, MemStoreSegmentsIterator iterator,
137    MemStoreCompactionStrategy.Action action) {
138
139    int numOfCellsAfterCompaction = 0;
140    int currentChunkIdx = 0;
141    int offsetInCurentChunk = ChunkCreator.SIZEOF_CHUNK_HEADER;
142    int numUniqueKeys = 0;
143    Cell prev = null;
144    Chunk[] chunks = allocIndexChunks(numOfCells);
145    while (iterator.hasNext()) { // the iterator hides the elimination logic for compaction
146      boolean alreadyCopied = false;
147      Cell c = iterator.next();
148      numOfCellsAfterCompaction++;
149      assert (c instanceof ExtendedCell);
150      if (((ExtendedCell) c).getChunkId() == ExtendedCell.CELL_NOT_BASED_ON_CHUNK) {
151        // CellChunkMap assumes all cells are allocated on MSLAB.
152        // Therefore, cells which are not allocated on MSLAB initially,
153        // are copied into MSLAB here.
154        c = copyCellIntoMSLAB(c, null); // no memstore sizing object to update
155        alreadyCopied = true;
156      }
157      if (offsetInCurentChunk + ClassSize.CELL_CHUNK_MAP_ENTRY > chunks[currentChunkIdx].size) {
158        currentChunkIdx++; // continue to the next index chunk
159        offsetInCurentChunk = ChunkCreator.SIZEOF_CHUNK_HEADER;
160      }
161      if (action == MemStoreCompactionStrategy.Action.COMPACT && !alreadyCopied) {
162        // for compaction copy cell to the new segment (MSLAB copy)
163        c = maybeCloneWithAllocator(c, false);
164      }
165      offsetInCurentChunk = // add the Cell reference to the index chunk
166        createCellReference((ByteBufferKeyValue) c, chunks[currentChunkIdx].getData(),
167          offsetInCurentChunk);
168      // the sizes still need to be updated in the new segment
169      // second parameter true, because in compaction/merge the addition of the cell to new segment
170      // is always successful
171      updateMetaInfo(c, true, null); // updates the size per cell
172      if (action == MemStoreCompactionStrategy.Action.MERGE_COUNT_UNIQUE_KEYS) {
173        // counting number of unique keys
174        if (prev != null) {
175          if (!CellUtil.matchingRowColumnBytes(prev, c)) {
176            numUniqueKeys++;
177          }
178        } else {
179          numUniqueKeys++;
180        }
181      }
182      prev = c;
183    }
184    if (action == MemStoreCompactionStrategy.Action.COMPACT) {
185      numUniqueKeys = numOfCells;
186    } else if (action != MemStoreCompactionStrategy.Action.MERGE_COUNT_UNIQUE_KEYS) {
187      numUniqueKeys = CellSet.UNKNOWN_NUM_UNIQUES;
188    }
189    // build the immutable CellSet
190    CellChunkMap ccm =
191      new CellChunkMap(getComparator(), chunks, 0, numOfCellsAfterCompaction, false);
192    this.setCellSet(null, new CellSet(ccm, numUniqueKeys)); // update the CellSet of this Segment
193  }
194
195  /*------------------------------------------------------------------------*/
196  // Create CellSet based on CellChunkMap from current ConcurrentSkipListMap based CellSet
197  // (without compacting iterator)
198  // This is a service for not-flat immutable segments
199  private void reinitializeCellSet(int numOfCells, KeyValueScanner segmentScanner,
200    CellSet oldCellSet, MemStoreSizing memstoreSizing, MemStoreCompactionStrategy.Action action) {
201    Cell curCell;
202    Chunk[] chunks = allocIndexChunks(numOfCells);
203
204    int currentChunkIdx = 0;
205    int offsetInCurentChunk = ChunkCreator.SIZEOF_CHUNK_HEADER;
206
207    int numUniqueKeys = 0;
208    Cell prev = null;
209    try {
210      while ((curCell = segmentScanner.next()) != null) {
211        assert (curCell instanceof ExtendedCell);
212        if (((ExtendedCell) curCell).getChunkId() == ExtendedCell.CELL_NOT_BASED_ON_CHUNK) {
213          // CellChunkMap assumes all cells are allocated on MSLAB.
214          // Therefore, cells which are not allocated on MSLAB initially,
215          // are copied into MSLAB here.
216          curCell = copyCellIntoMSLAB(curCell, memstoreSizing);
217        }
218        if (offsetInCurentChunk + ClassSize.CELL_CHUNK_MAP_ENTRY > chunks[currentChunkIdx].size) {
219          // continue to the next metadata chunk
220          currentChunkIdx++;
221          offsetInCurentChunk = ChunkCreator.SIZEOF_CHUNK_HEADER;
222        }
223        offsetInCurentChunk = createCellReference((ByteBufferKeyValue) curCell,
224          chunks[currentChunkIdx].getData(), offsetInCurentChunk);
225        if (action == MemStoreCompactionStrategy.Action.FLATTEN_COUNT_UNIQUE_KEYS) {
226          // counting number of unique keys
227          if (prev != null) {
228            if (!CellUtil.matchingRowColumn(prev, curCell)) {
229              numUniqueKeys++;
230            }
231          } else {
232            numUniqueKeys++;
233          }
234        }
235        prev = curCell;
236      }
237      if (action != MemStoreCompactionStrategy.Action.FLATTEN_COUNT_UNIQUE_KEYS) {
238        numUniqueKeys = CellSet.UNKNOWN_NUM_UNIQUES;
239      }
240    } catch (IOException ie) {
241      throw new IllegalStateException(ie);
242    } finally {
243      segmentScanner.close();
244    }
245
246    CellChunkMap ccm = new CellChunkMap(getComparator(), chunks, 0, numOfCells, false);
247    // update the CellSet of this Segment
248    this.setCellSet(oldCellSet, new CellSet(ccm, numUniqueKeys));
249  }
250
251  /*------------------------------------------------------------------------*/
252  // for a given cell, write the cell representation on the index chunk
253  private int createCellReference(ByteBufferKeyValue cell, ByteBuffer idxBuffer, int idxOffset) {
254    int offset = idxOffset;
255    int dataChunkID = cell.getChunkId();
256
257    offset = ByteBufferUtils.putInt(idxBuffer, offset, dataChunkID); // write data chunk id
258    offset = ByteBufferUtils.putInt(idxBuffer, offset, cell.getOffset()); // offset
259    offset = ByteBufferUtils.putInt(idxBuffer, offset, cell.getSerializedSize()); // length
260    offset = ByteBufferUtils.putLong(idxBuffer, offset, cell.getSequenceId()); // seqId
261
262    return offset;
263  }
264
265  private int calculateNumberOfChunks(int numOfCells, int chunkSize) {
266    int numOfCellsInChunk = calcNumOfCellsInChunk(chunkSize);
267    int numberOfChunks = numOfCells / numOfCellsInChunk;
268    if (numOfCells % numOfCellsInChunk != 0) { // if cells cannot be divided evenly between chunks
269      numberOfChunks++; // add one additional chunk
270    }
271    return numberOfChunks;
272  }
273
274  // Assuming we are going to use regular data chunks as index chunks,
275  // we check here how much free space will remain in the last allocated chunk
276  // (the least occupied one).
277  // If the percentage of its remaining free space is above the INDEX_CHUNK_UNUSED_SPACE
278  // threshold, then we will use index chunks (which are smaller) instead.
279  private ChunkCreator.ChunkType useIndexChunks(int numOfCells) {
280    int dataChunkSize = ChunkCreator.getInstance().getChunkSize();
281    int numOfCellsInChunk = calcNumOfCellsInChunk(dataChunkSize);
282    int cellsInLastChunk = numOfCells % numOfCellsInChunk;
283    if (cellsInLastChunk == 0) { // There is no free space in the last chunk and thus,
284      return ChunkCreator.ChunkType.DATA_CHUNK; // no need to use index chunks.
285    } else {
286      int chunkSpace = dataChunkSize - ChunkCreator.SIZEOF_CHUNK_HEADER;
287      int freeSpaceInLastChunk = chunkSpace - cellsInLastChunk * ClassSize.CELL_CHUNK_MAP_ENTRY;
288      if (freeSpaceInLastChunk > INDEX_CHUNK_UNUSED_SPACE_PRECENTAGE * chunkSpace) {
289        return ChunkCreator.ChunkType.INDEX_CHUNK;
290      }
291      return ChunkCreator.ChunkType.DATA_CHUNK;
292    }
293  }
294
295  private int calcNumOfCellsInChunk(int chunkSize) {
296    int chunkSpace = chunkSize - ChunkCreator.SIZEOF_CHUNK_HEADER;
297    int numOfCellsInChunk = chunkSpace / ClassSize.CELL_CHUNK_MAP_ENTRY;
298    return numOfCellsInChunk;
299  }
300
301  private Chunk[] allocIndexChunks(int numOfCells) {
302    // Decide whether to use regular or small chunks and then
303    // calculate how many chunks we will need for index
304
305    ChunkCreator.ChunkType chunkType = useIndexChunks(numOfCells);
306    int chunkSize = ChunkCreator.getInstance().getChunkSize(chunkType);
307    int numberOfChunks = calculateNumberOfChunks(numOfCells, chunkSize);
308    // all index Chunks are allocated from ChunkCreator
309    Chunk[] chunks = new Chunk[numberOfChunks];
310    // all index Chunks are allocated from ChunkCreator
311    for (int i = 0; i < numberOfChunks; i++) {
312      chunks[i] = this.getMemStoreLAB().getNewExternalChunk(chunkType);
313    }
314    return chunks;
315  }
316
317  private Cell copyCellIntoMSLAB(Cell cell, MemStoreSizing memstoreSizing) {
318    // Take care for a special case when a cell is copied from on-heap to (probably off-heap) MSLAB.
319    // The cell allocated as an on-heap JVM object (byte array) occupies slightly different
320    // amount of memory, than when the cell serialized and allocated on the MSLAB.
321    // Here, we update the heap size of the new segment only for the difference between object and
322    // serialized size. This is a decrease of the size as serialized cell is a bit smaller.
323    // The actual size of the cell is not added yet, and will be added (only in compaction)
324    // in initializeCellSet#updateMetaInfo().
325    long oldHeapSize = heapSizeChange(cell, true);
326    long oldOffHeapSize = offHeapSizeChange(cell, true);
327    long oldCellSize = getCellLength(cell);
328    cell = maybeCloneWithAllocator(cell, true);
329    long newHeapSize = heapSizeChange(cell, true);
330    long newOffHeapSize = offHeapSizeChange(cell, true);
331    long newCellSize = getCellLength(cell);
332    long heapOverhead = newHeapSize - oldHeapSize;
333    long offHeapOverhead = newOffHeapSize - oldOffHeapSize;
334    incMemStoreSize(newCellSize - oldCellSize, heapOverhead, offHeapOverhead, 0);
335    if (memstoreSizing != null) {
336      memstoreSizing.incMemStoreSize(newCellSize - oldCellSize, heapOverhead, offHeapOverhead, 0);
337    }
338    return cell;
339  }
340}