HBASE-26921 Rewrite the counting cells part in TestMultiVersions (#4316)
[hbase.git] / hbase-common / src / main / java / org / apache / hadoop / hbase / nio / ByteBuff.java
blob3e0a830d357a56562d47ae987d687e3c2b9d5437
1 /**
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org.apache.hadoop.hbase.nio;
20 import java.io.IOException;
21 import java.nio.ByteBuffer;
22 import java.nio.channels.FileChannel;
23 import java.nio.channels.ReadableByteChannel;
24 import java.util.List;
26 import org.apache.hadoop.hbase.io.ByteBuffAllocator.Recycler;
27 import org.apache.hadoop.hbase.util.ByteBufferUtils;
28 import org.apache.hadoop.hbase.util.Bytes;
29 import org.apache.hadoop.hbase.util.ObjectIntPair;
30 import org.apache.yetus.audience.InterfaceAudience;
32 import org.apache.hbase.thirdparty.io.netty.util.internal.ObjectUtil;
35 /**
36 * An abstract class that abstracts out as to how the byte buffers are used, either single or
37 * multiple. We have this interface because the java's ByteBuffers cannot be sub-classed. This class
38 * provides APIs similar to the ones provided in java's nio ByteBuffers and allows you to do
39 * positional reads/writes and relative reads and writes on the underlying BB. In addition to it, we
40 * have some additional APIs which helps us in the read path. <br/>
41 * The ByteBuff implement {@link HBaseReferenceCounted} interface which mean need to maintains a
42 * {@link RefCnt} inside, if ensure that the ByteBuff won't be used any more, we must do a
43 * {@link ByteBuff#release()} to recycle its NIO ByteBuffers. when considering the
44 * {@link ByteBuff#duplicate()} or {@link ByteBuff#slice()}, releasing either the duplicated one or
45 * the original one will free its memory, because they share the same NIO ByteBuffers. when you want
46 * to retain the NIO ByteBuffers even if the origin one called {@link ByteBuff#release()}, you can
47 * do like this:
49 * <pre>
50 * ByteBuff original = ...;
51 * ByteBuff dup = original.duplicate();
52 * dup.retain();
53 * original.release();
54 * // The NIO buffers can still be accessed unless you release the duplicated one
55 * dup.get(...);
56 * dup.release();
57 * // Both the original and dup can not access the NIO buffers any more.
58 * </pre>
60 @InterfaceAudience.Private
61 public abstract class ByteBuff implements HBaseReferenceCounted {
62 private static final String REFERENCE_COUNT_NAME = "ReferenceCount";
63 private static final int NIO_BUFFER_LIMIT = 64 * 1024; // should not be more than 64KB.
65 protected RefCnt refCnt;
67 /*************************** Methods for reference count **********************************/
69 protected void checkRefCount() {
70 ObjectUtil.checkPositive(refCnt(), REFERENCE_COUNT_NAME);
73 public int refCnt() {
74 return refCnt.refCnt();
77 @Override
78 public boolean release() {
79 return refCnt.release();
82 /******************************* Methods for ByteBuff **************************************/
84 /**
85 * @return this ByteBuff's current position
87 public abstract int position();
89 /**
90 * Sets this ByteBuff's position to the given value.
91 * @param position
92 * @return this object
94 public abstract ByteBuff position(int position);
96 /**
97 * Jumps the current position of this ByteBuff by specified length.
98 * @param len the length to be skipped
100 public abstract ByteBuff skip(int len);
103 * Jumps back the current position of this ByteBuff by specified length.
104 * @param len the length to move back
106 public abstract ByteBuff moveBack(int len);
109 * @return the total capacity of this ByteBuff.
111 public abstract int capacity();
114 * Returns the limit of this ByteBuff
115 * @return limit of the ByteBuff
117 public abstract int limit();
120 * Marks the limit of this ByteBuff.
121 * @param limit
122 * @return This ByteBuff
124 public abstract ByteBuff limit(int limit);
127 * Rewinds this ByteBuff and the position is set to 0
128 * @return this object
130 public abstract ByteBuff rewind();
133 * Marks the current position of the ByteBuff
134 * @return this object
136 public abstract ByteBuff mark();
139 * Returns bytes from current position till length specified, as a single ByteBuffer. When all
140 * these bytes happen to be in a single ByteBuffer, which this object wraps, that ByteBuffer item
141 * as such will be returned. So users are warned not to change the position or limit of this
142 * returned ByteBuffer. The position of the returned byte buffer is at the begin of the required
143 * bytes. When the required bytes happen to span across multiple ByteBuffers, this API will copy
144 * the bytes to a newly created ByteBuffer of required size and return that.
146 * @param length number of bytes required.
147 * @return bytes from current position till length specified, as a single ByteButter.
149 public abstract ByteBuffer asSubByteBuffer(int length);
152 * Returns bytes from given offset till length specified, as a single ByteBuffer. When all these
153 * bytes happen to be in a single ByteBuffer, which this object wraps, that ByteBuffer item as
154 * such will be returned (with offset in this ByteBuffer where the bytes starts). So users are
155 * warned not to change the position or limit of this returned ByteBuffer. When the required bytes
156 * happen to span across multiple ByteBuffers, this API will copy the bytes to a newly created
157 * ByteBuffer of required size and return that.
159 * @param offset the offset in this ByteBuff from where the subBuffer should be created
160 * @param length the length of the subBuffer
161 * @param pair a pair that will have the bytes from the current position till length specified,
162 * as a single ByteBuffer and offset in that Buffer where the bytes starts.
163 * Since this API gets called in a loop we are passing a pair to it which could be created
164 * outside the loop and the method would set the values on the pair that is passed in by
165 * the caller. Thus it avoids more object creations that would happen if the pair that is
166 * returned is created by this method every time.
168 public abstract void asSubByteBuffer(int offset, int length, ObjectIntPair<ByteBuffer> pair);
171 * Returns the number of elements between the current position and the
172 * limit.
173 * @return the remaining elements in this ByteBuff
175 public abstract int remaining();
178 * Returns true if there are elements between the current position and the limt
179 * @return true if there are elements, false otherwise
181 public abstract boolean hasRemaining();
184 * Similar to {@link ByteBuffer}.reset(), ensures that this ByteBuff
185 * is reset back to last marked position.
186 * @return This ByteBuff
188 public abstract ByteBuff reset();
191 * Returns an ByteBuff which is a sliced version of this ByteBuff. The position, limit and mark
192 * of the new ByteBuff will be independent than that of the original ByteBuff.
193 * The content of the new ByteBuff will start at this ByteBuff's current position
194 * @return a sliced ByteBuff
196 public abstract ByteBuff slice();
199 * Returns an ByteBuff which is a duplicate version of this ByteBuff. The
200 * position, limit and mark of the new ByteBuff will be independent than that
201 * of the original ByteBuff. The content of the new ByteBuff will start at
202 * this ByteBuff's current position The position, limit and mark of the new
203 * ByteBuff would be identical to this ByteBuff in terms of values.
205 * @return a sliced ByteBuff
207 public abstract ByteBuff duplicate();
210 * A relative method that returns byte at the current position. Increments the
211 * current position by the size of a byte.
212 * @return the byte at the current position
214 public abstract byte get();
217 * Fetches the byte at the given index. Does not change position of the underlying ByteBuffers
218 * @param index
219 * @return the byte at the given index
221 public abstract byte get(int index);
224 * Fetches the byte at the given offset from current position. Does not change position
225 * of the underlying ByteBuffers.
227 * @param offset
228 * @return the byte value at the given index.
230 public abstract byte getByteAfterPosition(int offset);
233 * Writes a byte to this ByteBuff at the current position and increments the position
234 * @param b
235 * @return this object
237 public abstract ByteBuff put(byte b);
240 * Writes a byte to this ByteBuff at the given index
241 * @param index
242 * @param b
243 * @return this object
245 public abstract ByteBuff put(int index, byte b);
248 * Copies the specified number of bytes from this ByteBuff's current position to
249 * the byte[]'s offset. Also advances the position of the ByteBuff by the given length.
250 * @param dst
251 * @param offset within the current array
252 * @param length upto which the bytes to be copied
254 public abstract void get(byte[] dst, int offset, int length);
257 * Copies the specified number of bytes from this ByteBuff's given position to
258 * the byte[]'s offset. The position of the ByteBuff remains in the current position only
259 * @param sourceOffset the offset in this ByteBuff from where the copy should happen
260 * @param dst the byte[] to which the ByteBuff's content is to be copied
261 * @param offset within the current array
262 * @param length upto which the bytes to be copied
264 public abstract void get(int sourceOffset, byte[] dst, int offset, int length);
267 * Copies the content from this ByteBuff's current position to the byte array and fills it. Also
268 * advances the position of the ByteBuff by the length of the byte[].
269 * @param dst
271 public abstract void get(byte[] dst);
274 * Copies from the given byte[] to this ByteBuff
275 * @param src
276 * @param offset the position in the byte array from which the copy should be done
277 * @param length the length upto which the copy should happen
278 * @return this ByteBuff
280 public abstract ByteBuff put(byte[] src, int offset, int length);
283 * Copies from the given byte[] to this ByteBuff
284 * @param src
285 * @return this ByteBuff
287 public abstract ByteBuff put(byte[] src);
290 * @return true or false if the underlying BB support hasArray
292 public abstract boolean hasArray();
295 * @return the byte[] if the underlying BB has single BB and hasArray true
297 public abstract byte[] array();
300 * @return the arrayOffset of the byte[] incase of a single BB backed ByteBuff
302 public abstract int arrayOffset();
305 * Returns the short value at the current position. Also advances the position by the size
306 * of short
308 * @return the short value at the current position
310 public abstract short getShort();
313 * Fetches the short value at the given index. Does not change position of the
314 * underlying ByteBuffers. The caller is sure that the index will be after
315 * the current position of this ByteBuff. So even if the current short does not fit in the
316 * current item we can safely move to the next item and fetch the remaining bytes forming
317 * the short
319 * @param index
320 * @return the short value at the given index
322 public abstract short getShort(int index);
325 * Fetches the short value at the given offset from current position. Does not change position
326 * of the underlying ByteBuffers.
328 * @param offset
329 * @return the short value at the given index.
331 public abstract short getShortAfterPosition(int offset);
334 * Returns the int value at the current position. Also advances the position by the size of int
336 * @return the int value at the current position
338 public abstract int getInt();
341 * Writes an int to this ByteBuff at its current position. Also advances the position
342 * by size of int
343 * @param value Int value to write
344 * @return this object
346 public abstract ByteBuff putInt(int value);
349 * Fetches the int at the given index. Does not change position of the underlying ByteBuffers.
350 * Even if the current int does not fit in the
351 * current item we can safely move to the next item and fetch the remaining bytes forming
352 * the int
354 * @param index
355 * @return the int value at the given index
357 public abstract int getInt(int index);
360 * Fetches the int value at the given offset from current position. Does not change position
361 * of the underlying ByteBuffers.
363 * @param offset
364 * @return the int value at the given index.
366 public abstract int getIntAfterPosition(int offset);
369 * Returns the long value at the current position. Also advances the position by the size of long
371 * @return the long value at the current position
373 public abstract long getLong();
376 * Writes a long to this ByteBuff at its current position.
377 * Also advances the position by size of long
378 * @param value Long value to write
379 * @return this object
381 public abstract ByteBuff putLong(long value);
384 * Fetches the long at the given index. Does not change position of the
385 * underlying ByteBuffers. The caller is sure that the index will be after
386 * the current position of this ByteBuff. So even if the current long does not fit in the
387 * current item we can safely move to the next item and fetch the remaining bytes forming
388 * the long
390 * @param index
391 * @return the long value at the given index
393 public abstract long getLong(int index);
396 * Fetches the long value at the given offset from current position. Does not change position
397 * of the underlying ByteBuffers.
399 * @param offset
400 * @return the long value at the given index.
402 public abstract long getLongAfterPosition(int offset);
405 * Copy the content from this ByteBuff to a byte[].
406 * @return byte[] with the copied contents from this ByteBuff.
408 public byte[] toBytes() {
409 return toBytes(0, this.limit());
413 * Copy the content from this ByteBuff to a byte[] based on the given offset and
414 * length
416 * @param offset
417 * the position from where the copy should start
418 * @param length
419 * the length upto which the copy has to be done
420 * @return byte[] with the copied contents from this ByteBuff.
422 public abstract byte[] toBytes(int offset, int length);
425 * Copies the content from this ByteBuff to a ByteBuffer
426 * Note : This will advance the position marker of {@code out} but not change the position maker
427 * for this ByteBuff
428 * @param out the ByteBuffer to which the copy has to happen
429 * @param sourceOffset the offset in the ByteBuff from which the elements has
430 * to be copied
431 * @param length the length in this ByteBuff upto which the elements has to be copied
433 public abstract void get(ByteBuffer out, int sourceOffset, int length);
436 * Copies the contents from the src ByteBuff to this ByteBuff. This will be
437 * absolute positional copying and
438 * won't affect the position of any of the buffers.
439 * @param offset the position in this ByteBuff to which the copy should happen
440 * @param src the src ByteBuff
441 * @param srcOffset the offset in the src ByteBuff from where the elements should be read
442 * @param length the length up to which the copy should happen
444 public abstract ByteBuff put(int offset, ByteBuff src, int srcOffset, int length);
447 * Reads bytes from the given channel into this ByteBuff
448 * @param channel
449 * @return The number of bytes read from the channel
450 * @throws IOException
452 public abstract int read(ReadableByteChannel channel) throws IOException;
455 * Reads bytes from FileChannel into this ByteBuff
457 public abstract int read(FileChannel channel, long offset) throws IOException;
460 * Write this ByteBuff's data into target file
462 public abstract int write(FileChannel channel, long offset) throws IOException;
465 * function interface for Channel read
467 @FunctionalInterface
468 interface ChannelReader {
469 int read(ReadableByteChannel channel, ByteBuffer buf, long offset) throws IOException;
472 static final ChannelReader CHANNEL_READER = (channel, buf, offset) -> {
473 return channel.read(buf);
476 static final ChannelReader FILE_READER = (channel, buf, offset) -> {
477 return ((FileChannel)channel).read(buf, offset);
480 // static helper methods
481 public static int read(ReadableByteChannel channel, ByteBuffer buf, long offset,
482 ChannelReader reader) throws IOException {
483 if (buf.remaining() <= NIO_BUFFER_LIMIT) {
484 return reader.read(channel, buf, offset);
486 int originalLimit = buf.limit();
487 int initialRemaining = buf.remaining();
488 int ret = 0;
490 while (buf.remaining() > 0) {
491 try {
492 int ioSize = Math.min(buf.remaining(), NIO_BUFFER_LIMIT);
493 buf.limit(buf.position() + ioSize);
494 offset += ret;
495 ret = reader.read(channel, buf, offset);
496 if (ret < ioSize) {
497 break;
499 } finally {
500 buf.limit(originalLimit);
503 int nBytes = initialRemaining - buf.remaining();
504 return (nBytes > 0) ? nBytes : ret;
508 * Read integer from ByteBuff coded in 7 bits and increment position.
509 * @return Read integer.
511 public static int readCompressedInt(ByteBuff buf) {
512 byte b = buf.get();
513 if ((b & ByteBufferUtils.NEXT_BIT_MASK) != 0) {
514 return (b & ByteBufferUtils.VALUE_MASK)
515 + (readCompressedInt(buf) << ByteBufferUtils.NEXT_BIT_SHIFT);
517 return b & ByteBufferUtils.VALUE_MASK;
521 * Compares two ByteBuffs
523 * @param buf1 the first ByteBuff
524 * @param o1 the offset in the first ByteBuff from where the compare has to happen
525 * @param len1 the length in the first ByteBuff upto which the compare has to happen
526 * @param buf2 the second ByteBuff
527 * @param o2 the offset in the second ByteBuff from where the compare has to happen
528 * @param len2 the length in the second ByteBuff upto which the compare has to happen
529 * @return Positive if buf1 is bigger than buf2, 0 if they are equal, and negative if buf1 is
530 * smaller than buf2.
532 public static int compareTo(ByteBuff buf1, int o1, int len1, ByteBuff buf2,
533 int o2, int len2) {
534 if (buf1.hasArray() && buf2.hasArray()) {
535 return Bytes.compareTo(buf1.array(), buf1.arrayOffset() + o1, len1, buf2.array(),
536 buf2.arrayOffset() + o2, len2);
538 int end1 = o1 + len1;
539 int end2 = o2 + len2;
540 for (int i = o1, j = o2; i < end1 && j < end2; i++, j++) {
541 int a = buf1.get(i) & 0xFF;
542 int b = buf2.get(j) & 0xFF;
543 if (a != b) {
544 return a - b;
547 return len1 - len2;
551 * Read long which was written to fitInBytes bytes and increment position.
552 * @param fitInBytes In how many bytes given long is stored.
553 * @return The value of parsed long.
555 public static long readLong(ByteBuff in, final int fitInBytes) {
556 long tmpLength = 0;
557 for (int i = 0; i < fitInBytes; ++i) {
558 tmpLength |= (in.get() & 0xffl) << (8l * i);
560 return tmpLength;
563 public abstract ByteBuffer[] nioByteBuffers();
565 @Override
566 public String toString() {
567 return this.getClass().getSimpleName() + "[pos=" + position() + ", lim=" + limit() +
568 ", cap= " + capacity() + "]";
571 /********************************* ByteBuff wrapper methods ***********************************/
574 * In theory, the upstream should never construct an ByteBuff by passing an given refCnt, so
575 * please don't use this public method in other place. Make the method public here because the
576 * BucketEntry#wrapAsCacheable in hbase-server module will use its own refCnt and ByteBuffers from
577 * IOEngine to composite an HFileBlock's ByteBuff, we didn't find a better way so keep the public
578 * way here.
580 public static ByteBuff wrap(ByteBuffer[] buffers, RefCnt refCnt) {
581 if (buffers == null || buffers.length == 0) {
582 throw new IllegalArgumentException("buffers shouldn't be null or empty");
584 return buffers.length == 1 ? new SingleByteBuff(refCnt, buffers[0])
585 : new MultiByteBuff(refCnt, buffers);
588 public static ByteBuff wrap(ByteBuffer[] buffers, Recycler recycler) {
589 return wrap(buffers, RefCnt.create(recycler));
592 public static ByteBuff wrap(ByteBuffer[] buffers) {
593 return wrap(buffers, RefCnt.create());
596 public static ByteBuff wrap(List<ByteBuffer> buffers, Recycler recycler) {
597 return wrap(buffers, RefCnt.create(recycler));
600 public static ByteBuff wrap(List<ByteBuffer> buffers) {
601 return wrap(buffers, RefCnt.create());
604 public static ByteBuff wrap(ByteBuffer buffer) {
605 return wrap(buffer, RefCnt.create());
609 * Make this private because we don't want to expose the refCnt related wrap method to upstream.
611 private static ByteBuff wrap(List<ByteBuffer> buffers, RefCnt refCnt) {
612 if (buffers == null || buffers.size() == 0) {
613 throw new IllegalArgumentException("buffers shouldn't be null or empty");
615 return buffers.size() == 1 ? new SingleByteBuff(refCnt, buffers.get(0))
616 : new MultiByteBuff(refCnt, buffers.toArray(new ByteBuffer[0]));
620 * Make this private because we don't want to expose the refCnt related wrap method to upstream.
622 private static ByteBuff wrap(ByteBuffer buffer, RefCnt refCnt) {
623 return new SingleByteBuff(refCnt, buffer);