HBASE-26921 Rewrite the counting cells part in TestMultiVersions (#4316)
[hbase.git] / hbase-common / src / main / java / org / apache / hadoop / hbase / KeyValueUtil.java
blobda843f1f7d6a7fec92a1543cab34a129e26509a6
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.
19 package org.apache.hadoop.hbase;
21 import java.io.DataInput;
22 import java.io.DataOutput;
23 import java.io.EOFException;
24 import java.io.IOException;
25 import java.io.InputStream;
26 import java.io.OutputStream;
27 import java.nio.ByteBuffer;
28 import java.util.ArrayList;
29 import java.util.List;
31 import org.apache.hadoop.hbase.KeyValue.Type;
32 import org.apache.hadoop.hbase.io.util.StreamUtils;
33 import org.apache.hadoop.hbase.util.ByteBufferUtils;
34 import org.apache.hadoop.hbase.util.Bytes;
35 import org.apache.hadoop.io.IOUtils;
36 import org.apache.hadoop.io.WritableUtils;
37 import org.apache.yetus.audience.InterfaceAudience;
38 import org.slf4j.Logger;
39 import org.slf4j.LoggerFactory;
41 import org.apache.hbase.thirdparty.com.google.common.base.Function;
42 import org.apache.hbase.thirdparty.com.google.common.collect.Lists;
43 import org.apache.hbase.thirdparty.org.apache.commons.collections4.IterableUtils;
45 /**
46 * static convenience methods for dealing with KeyValues and collections of KeyValues
48 @InterfaceAudience.Private
49 public class KeyValueUtil {
51 private static final Logger LOG = LoggerFactory.getLogger(KeyValueUtil.class);
53 /**************** length *********************/
55 public static int length(short rlen, byte flen, int qlen, int vlen, int tlen, boolean withTags) {
56 if (withTags) {
57 return (int) (KeyValue.getKeyValueDataStructureSize(rlen, flen, qlen, vlen, tlen));
59 return (int) (KeyValue.getKeyValueDataStructureSize(rlen, flen, qlen, vlen));
62 /**
63 * Returns number of bytes this cell's key part would have been used if serialized as in
64 * {@link KeyValue}. Key includes rowkey, family, qualifier, timestamp and type.
65 * @param cell
66 * @return the key length
68 public static int keyLength(final Cell cell) {
69 return keyLength(cell.getRowLength(), cell.getFamilyLength(), cell.getQualifierLength());
72 private static int keyLength(short rlen, byte flen, int qlen) {
73 return (int) KeyValue.getKeyDataStructureSize(rlen, flen, qlen);
76 public static int lengthWithMvccVersion(final KeyValue kv, final boolean includeMvccVersion) {
77 int length = kv.getLength();
78 if (includeMvccVersion) {
79 length += WritableUtils.getVIntSize(kv.getSequenceId());
81 return length;
84 public static int totalLengthWithMvccVersion(final Iterable<? extends KeyValue> kvs,
85 final boolean includeMvccVersion) {
86 int length = 0;
87 for (KeyValue kv : IterableUtils.emptyIfNull(kvs)) {
88 length += lengthWithMvccVersion(kv, includeMvccVersion);
90 return length;
94 /**************** copy the cell to create a new keyvalue *********************/
96 public static KeyValue copyToNewKeyValue(final Cell cell) {
97 byte[] bytes = copyToNewByteArray(cell);
98 KeyValue kvCell = new KeyValue(bytes, 0, bytes.length);
99 kvCell.setSequenceId(cell.getSequenceId());
100 return kvCell;
104 * The position will be set to the beginning of the new ByteBuffer
105 * @param cell
106 * @return the Bytebuffer containing the key part of the cell
108 public static ByteBuffer copyKeyToNewByteBuffer(final Cell cell) {
109 byte[] bytes = new byte[keyLength(cell)];
110 appendKeyTo(cell, bytes, 0);
111 ByteBuffer buffer = ByteBuffer.wrap(bytes);
112 return buffer;
116 * Copies the key to a new KeyValue
117 * @param cell
118 * @return the KeyValue that consists only the key part of the incoming cell
120 public static KeyValue toNewKeyCell(final Cell cell) {
121 byte[] bytes = new byte[keyLength(cell)];
122 appendKeyTo(cell, bytes, 0);
123 KeyValue kv = new KeyValue.KeyOnlyKeyValue(bytes, 0, bytes.length);
124 // Set the seq id. The new key cell could be used in comparisons so it
125 // is important that it uses the seqid also. If not the comparsion would fail
126 kv.setSequenceId(cell.getSequenceId());
127 return kv;
130 public static byte[] copyToNewByteArray(final Cell cell) {
131 //Cell#getSerializedSize returns the serialized size of the Source cell, which may
132 //not serialize all fields. We are constructing a KeyValue backing array here,
133 //which does include all fields, and must allocate accordingly.
134 //TODO we could probably use Cell#getSerializedSize safely, the errors were
135 //caused by cells corrupted by use-after-free bugs
136 int v1Length = length(cell.getRowLength(), cell.getFamilyLength(),
137 cell.getQualifierLength(), cell.getValueLength(), cell.getTagsLength(), true);
138 byte[] backingBytes = new byte[v1Length];
139 appendToByteArray(cell, backingBytes, 0, true);
140 return backingBytes;
143 public static int appendKeyTo(final Cell cell, final byte[] output,
144 final int offset) {
145 int nextOffset = offset;
146 nextOffset = Bytes.putShort(output, nextOffset, cell.getRowLength());
147 nextOffset = CellUtil.copyRowTo(cell, output, nextOffset);
148 nextOffset = Bytes.putByte(output, nextOffset, cell.getFamilyLength());
149 nextOffset = CellUtil.copyFamilyTo(cell, output, nextOffset);
150 nextOffset = CellUtil.copyQualifierTo(cell, output, nextOffset);
151 nextOffset = Bytes.putLong(output, nextOffset, cell.getTimestamp());
152 nextOffset = Bytes.putByte(output, nextOffset, cell.getTypeByte());
153 return nextOffset;
156 /**************** copy key and value *********************/
158 public static int appendToByteArray(Cell cell, byte[] output, int offset, boolean withTags) {
159 int pos = offset;
160 pos = Bytes.putInt(output, pos, keyLength(cell));
161 pos = Bytes.putInt(output, pos, cell.getValueLength());
162 pos = appendKeyTo(cell, output, pos);
163 pos = CellUtil.copyValueTo(cell, output, pos);
164 if (withTags && (cell.getTagsLength() > 0)) {
165 pos = Bytes.putAsShort(output, pos, cell.getTagsLength());
166 pos = PrivateCellUtil.copyTagsTo(cell, output, pos);
168 return pos;
172 * Copy the Cell content into the passed buf in KeyValue serialization format.
174 public static int appendTo(Cell cell, ByteBuffer buf, int offset, boolean withTags) {
175 offset = ByteBufferUtils.putInt(buf, offset, keyLength(cell));// Key length
176 offset = ByteBufferUtils.putInt(buf, offset, cell.getValueLength());// Value length
177 offset = appendKeyTo(cell, buf, offset);
178 offset = CellUtil.copyValueTo(cell, buf, offset);// Value bytes
179 int tagsLength = cell.getTagsLength();
180 if (withTags && (tagsLength > 0)) {
181 offset = ByteBufferUtils.putAsShort(buf, offset, tagsLength);// Tags length
182 offset = PrivateCellUtil.copyTagsTo(cell, buf, offset);// Tags bytes
184 return offset;
187 public static int appendKeyTo(Cell cell, ByteBuffer buf, int offset) {
188 offset = ByteBufferUtils.putShort(buf, offset, cell.getRowLength());// RK length
189 offset = CellUtil.copyRowTo(cell, buf, offset);// Row bytes
190 offset = ByteBufferUtils.putByte(buf, offset, cell.getFamilyLength());// CF length
191 offset = CellUtil.copyFamilyTo(cell, buf, offset);// CF bytes
192 offset = CellUtil.copyQualifierTo(cell, buf, offset);// Qualifier bytes
193 offset = ByteBufferUtils.putLong(buf, offset, cell.getTimestamp());// TS
194 offset = ByteBufferUtils.putByte(buf, offset, cell.getTypeByte());// Type
195 return offset;
198 public static void appendToByteBuffer(final ByteBuffer bb, final KeyValue kv,
199 final boolean includeMvccVersion) {
200 // keep pushing the limit out. assume enough capacity
201 bb.limit(bb.position() + kv.getLength());
202 bb.put(kv.getBuffer(), kv.getOffset(), kv.getLength());
203 if (includeMvccVersion) {
204 int numMvccVersionBytes = WritableUtils.getVIntSize(kv.getSequenceId());
205 ByteBufferUtils.extendLimit(bb, numMvccVersionBytes);
206 ByteBufferUtils.writeVLong(bb, kv.getSequenceId());
211 /**************** iterating *******************************/
214 * Creates a new KeyValue object positioned in the supplied ByteBuffer and sets the ByteBuffer's
215 * position to the start of the next KeyValue. Does not allocate a new array or copy data.
216 * @param bb
217 * @param includesMvccVersion
218 * @param includesTags
220 public static KeyValue nextShallowCopy(final ByteBuffer bb, final boolean includesMvccVersion,
221 boolean includesTags) {
222 if (bb.isDirect()) {
223 throw new IllegalArgumentException("only supports heap buffers");
225 if (bb.remaining() < 1) {
226 return null;
228 KeyValue keyValue = null;
229 int underlyingArrayOffset = bb.arrayOffset() + bb.position();
230 int keyLength = bb.getInt();
231 int valueLength = bb.getInt();
232 ByteBufferUtils.skip(bb, keyLength + valueLength);
233 int tagsLength = 0;
234 if (includesTags) {
235 // Read short as unsigned, high byte first
236 tagsLength = ((bb.get() & 0xff) << 8) ^ (bb.get() & 0xff);
237 ByteBufferUtils.skip(bb, tagsLength);
239 int kvLength = (int) KeyValue.getKeyValueDataStructureSize(keyLength, valueLength, tagsLength);
240 keyValue = new KeyValue(bb.array(), underlyingArrayOffset, kvLength);
241 if (includesMvccVersion) {
242 long mvccVersion = ByteBufferUtils.readVLong(bb);
243 keyValue.setSequenceId(mvccVersion);
245 return keyValue;
249 /*************** next/previous **********************************/
252 * Decrement the timestamp. For tests (currently wasteful)
254 * Remember timestamps are sorted reverse chronologically.
255 * @param in
256 * @return previous key
258 public static KeyValue previousKey(final KeyValue in) {
259 return createFirstOnRow(CellUtil.cloneRow(in), CellUtil.cloneFamily(in),
260 CellUtil.cloneQualifier(in), in.getTimestamp() - 1);
265 * Create a KeyValue for the specified row, family and qualifier that would be
266 * larger than or equal to all other possible KeyValues that have the same
267 * row, family, qualifier. Used for reseeking. Should NEVER be returned to a client.
269 * @param row
270 * row key
271 * @param roffset
272 * row offset
273 * @param rlength
274 * row length
275 * @param family
276 * family name
277 * @param foffset
278 * family offset
279 * @param flength
280 * family length
281 * @param qualifier
282 * column qualifier
283 * @param qoffset
284 * qualifier offset
285 * @param qlength
286 * qualifier length
287 * @return Last possible key on passed row, family, qualifier.
289 public static KeyValue createLastOnRow(final byte[] row, final int roffset, final int rlength,
290 final byte[] family, final int foffset, final int flength, final byte[] qualifier,
291 final int qoffset, final int qlength) {
292 return new KeyValue(row, roffset, rlength, family, foffset, flength, qualifier, qoffset,
293 qlength, PrivateConstants.OLDEST_TIMESTAMP, Type.Minimum, null, 0, 0);
297 * Create a KeyValue that is smaller than all other possible KeyValues
298 * for the given row. That is any (valid) KeyValue on 'row' would sort
299 * _after_ the result.
301 * @param row - row key (arbitrary byte array)
302 * @return First possible KeyValue on passed <code>row</code>
304 public static KeyValue createFirstOnRow(final byte [] row, int roffset, short rlength) {
305 return new KeyValue(row, roffset, rlength,
306 null, 0, 0, null, 0, 0, HConstants.LATEST_TIMESTAMP, Type.Maximum, null, 0, 0);
310 * Creates a KeyValue that is last on the specified row id. That is,
311 * every other possible KeyValue for the given row would compareTo()
312 * less than the result of this call.
313 * @param row row key
314 * @return Last possible KeyValue on passed <code>row</code>
316 public static KeyValue createLastOnRow(final byte[] row) {
317 return new KeyValue(row, null, null, HConstants.LATEST_TIMESTAMP, Type.Minimum);
321 * Create a KeyValue that is smaller than all other possible KeyValues
322 * for the given row. That is any (valid) KeyValue on 'row' would sort
323 * _after_ the result.
325 * @param row - row key (arbitrary byte array)
326 * @return First possible KeyValue on passed <code>row</code>
328 public static KeyValue createFirstOnRow(final byte [] row) {
329 return createFirstOnRow(row, HConstants.LATEST_TIMESTAMP);
333 * Creates a KeyValue that is smaller than all other KeyValues that
334 * are older than the passed timestamp.
335 * @param row - row key (arbitrary byte array)
336 * @param ts - timestamp
337 * @return First possible key on passed <code>row</code> and timestamp.
339 public static KeyValue createFirstOnRow(final byte [] row,
340 final long ts) {
341 return new KeyValue(row, null, null, ts, Type.Maximum);
345 * Create a KeyValue for the specified row, family and qualifier that would be
346 * smaller than all other possible KeyValues that have the same row,family,qualifier.
347 * Used for seeking.
348 * @param row - row key (arbitrary byte array)
349 * @param family - family name
350 * @param qualifier - column qualifier
351 * @return First possible key on passed <code>row</code>, and column.
353 public static KeyValue createFirstOnRow(final byte [] row, final byte [] family,
354 final byte [] qualifier) {
355 return new KeyValue(row, family, qualifier, HConstants.LATEST_TIMESTAMP, Type.Maximum);
359 * @param row - row key (arbitrary byte array)
360 * @param f - family name
361 * @param q - column qualifier
362 * @param ts - timestamp
363 * @return First possible key on passed <code>row</code>, column and timestamp
365 public static KeyValue createFirstOnRow(final byte [] row, final byte [] f,
366 final byte [] q, final long ts) {
367 return new KeyValue(row, f, q, ts, Type.Maximum);
371 * Create a KeyValue for the specified row, family and qualifier that would be
372 * smaller than all other possible KeyValues that have the same row,
373 * family, qualifier.
374 * Used for seeking.
375 * @param row row key
376 * @param roffset row offset
377 * @param rlength row length
378 * @param family family name
379 * @param foffset family offset
380 * @param flength family length
381 * @param qualifier column qualifier
382 * @param qoffset qualifier offset
383 * @param qlength qualifier length
384 * @return First possible key on passed Row, Family, Qualifier.
386 public static KeyValue createFirstOnRow(final byte [] row,
387 final int roffset, final int rlength, final byte [] family,
388 final int foffset, final int flength, final byte [] qualifier,
389 final int qoffset, final int qlength) {
390 return new KeyValue(row, roffset, rlength, family,
391 foffset, flength, qualifier, qoffset, qlength,
392 HConstants.LATEST_TIMESTAMP, Type.Maximum, null, 0, 0);
396 * Create a KeyValue for the specified row, family and qualifier that would be
397 * smaller than all other possible KeyValues that have the same row,
398 * family, qualifier.
399 * Used for seeking.
401 * @param buffer the buffer to use for the new <code>KeyValue</code> object
402 * @param row the value key
403 * @param family family name
404 * @param qualifier column qualifier
406 * @return First possible key on passed Row, Family, Qualifier.
408 * @throws IllegalArgumentException The resulting <code>KeyValue</code> object would be larger
409 * than the provided buffer or than <code>Integer.MAX_VALUE</code>
411 public static KeyValue createFirstOnRow(byte [] buffer, final byte [] row,
412 final byte [] family, final byte [] qualifier)
413 throws IllegalArgumentException {
414 return createFirstOnRow(buffer, 0, row, 0, row.length,
415 family, 0, family.length,
416 qualifier, 0, qualifier.length);
420 * Create a KeyValue for the specified row, family and qualifier that would be
421 * smaller than all other possible KeyValues that have the same row,
422 * family, qualifier.
423 * Used for seeking.
425 * @param buffer the buffer to use for the new <code>KeyValue</code> object
426 * @param boffset buffer offset
427 * @param row the value key
428 * @param roffset row offset
429 * @param rlength row length
430 * @param family family name
431 * @param foffset family offset
432 * @param flength family length
433 * @param qualifier column qualifier
434 * @param qoffset qualifier offset
435 * @param qlength qualifier length
437 * @return First possible key on passed Row, Family, Qualifier.
439 * @throws IllegalArgumentException The resulting <code>KeyValue</code> object would be larger
440 * than the provided buffer or than <code>Integer.MAX_VALUE</code>
442 public static KeyValue createFirstOnRow(byte[] buffer, final int boffset, final byte[] row,
443 final int roffset, final int rlength, final byte[] family, final int foffset,
444 final int flength, final byte[] qualifier, final int qoffset, final int qlength)
445 throws IllegalArgumentException {
447 long lLength = KeyValue.getKeyValueDataStructureSize(rlength, flength, qlength, 0);
449 if (lLength > Integer.MAX_VALUE) {
450 throw new IllegalArgumentException("KeyValue length " + lLength + " > " + Integer.MAX_VALUE);
452 int iLength = (int) lLength;
453 if (buffer.length - boffset < iLength) {
454 throw new IllegalArgumentException("Buffer size " + (buffer.length - boffset) + " < "
455 + iLength);
458 int len = KeyValue.writeByteArray(buffer, boffset, row, roffset, rlength, family, foffset,
459 flength, qualifier, qoffset, qlength, HConstants.LATEST_TIMESTAMP, KeyValue.Type.Maximum,
460 null, 0, 0, null);
461 return new KeyValue(buffer, boffset, len);
464 /*************** misc **********************************/
466 * @param cell
467 * @return <code>cell</code> if it is an object of class {@link KeyValue} else we will return a
468 * new {@link KeyValue} instance made from <code>cell</code> Note: Even if the cell is an
469 * object of any of the subclass of {@link KeyValue}, we will create a new
470 * {@link KeyValue} object wrapping same buffer. This API is used only with MR based tools
471 * which expect the type to be exactly KeyValue. That is the reason for doing this way.
472 * @deprecated without any replacement.
474 @Deprecated
475 public static KeyValue ensureKeyValue(final Cell cell) {
476 if (cell == null) return null;
477 if (cell instanceof KeyValue) {
478 if (cell.getClass().getName().equals(KeyValue.class.getName())) {
479 return (KeyValue) cell;
481 // Cell is an Object of any of the sub classes of KeyValue. Make a new KeyValue wrapping the
482 // same byte[]
483 KeyValue kv = (KeyValue) cell;
484 KeyValue newKv = new KeyValue(kv.bytes, kv.offset, kv.length);
485 newKv.setSequenceId(kv.getSequenceId());
486 return newKv;
488 return copyToNewKeyValue(cell);
491 @Deprecated
492 public static List<KeyValue> ensureKeyValues(List<Cell> cells) {
493 List<KeyValue> lazyList = Lists.transform(cells, new Function<Cell, KeyValue>() {
494 @Override
495 public KeyValue apply(Cell arg0) {
496 return KeyValueUtil.ensureKeyValue(arg0);
499 return new ArrayList<>(lazyList);
502 * Write out a KeyValue in the manner in which we used to when KeyValue was a
503 * Writable.
505 * @param kv
506 * @param out
507 * @return Length written on stream
508 * @throws IOException
509 * @see #create(DataInput) for the inverse function
511 public static long write(final KeyValue kv, final DataOutput out) throws IOException {
512 // This is how the old Writables write used to serialize KVs. Need to figure
513 // way to make it
514 // work for all implementations.
515 int length = kv.getLength();
516 out.writeInt(length);
517 out.write(kv.getBuffer(), kv.getOffset(), length);
518 return (long) length + Bytes.SIZEOF_INT;
521 static String bytesToHex(byte[] buf, int offset, int length) {
522 String bufferContents = buf != null ? Bytes.toStringBinary(buf, offset, length) : "<null>";
523 return ", KeyValueBytesHex=" + bufferContents + ", offset=" + offset + ", length=" + length;
526 static void checkKeyValueBytes(byte[] buf, int offset, int length, boolean withTags) {
527 if (buf == null) {
528 String msg = "Invalid to have null byte array in KeyValue.";
529 LOG.warn(msg);
530 throw new IllegalArgumentException(msg);
533 int pos = offset, endOffset = offset + length;
534 // check the key
535 if (pos + Bytes.SIZEOF_INT > endOffset) {
536 String msg =
537 "Overflow when reading key length at position=" + pos + bytesToHex(buf, offset, length);
538 LOG.warn(msg);
539 throw new IllegalArgumentException(msg);
541 int keyLen = Bytes.toInt(buf, pos, Bytes.SIZEOF_INT);
542 pos += Bytes.SIZEOF_INT;
543 if (keyLen <= 0 || pos + keyLen > endOffset) {
544 String msg =
545 "Invalid key length in KeyValue. keyLength=" + keyLen + bytesToHex(buf, offset, length);
546 LOG.warn(msg);
547 throw new IllegalArgumentException(msg);
549 // check the value
550 if (pos + Bytes.SIZEOF_INT > endOffset) {
551 String msg =
552 "Overflow when reading value length at position=" + pos + bytesToHex(buf, offset, length);
553 LOG.warn(msg);
554 throw new IllegalArgumentException(msg);
556 int valLen = Bytes.toInt(buf, pos, Bytes.SIZEOF_INT);
557 pos += Bytes.SIZEOF_INT;
558 if (valLen < 0 || pos + valLen > endOffset) {
559 String msg = "Invalid value length in KeyValue, valueLength=" + valLen +
560 bytesToHex(buf, offset, length);
561 LOG.warn(msg);
562 throw new IllegalArgumentException(msg);
564 // check the row
565 if (pos + Bytes.SIZEOF_SHORT > endOffset) {
566 String msg =
567 "Overflow when reading row length at position=" + pos + bytesToHex(buf, offset, length);
568 LOG.warn(msg);
569 throw new IllegalArgumentException(msg);
571 short rowLen = Bytes.toShort(buf, pos, Bytes.SIZEOF_SHORT);
572 pos += Bytes.SIZEOF_SHORT;
573 if (rowLen < 0 || pos + rowLen > endOffset) {
574 String msg =
575 "Invalid row length in KeyValue, rowLength=" + rowLen + bytesToHex(buf, offset, length);
576 LOG.warn(msg);
577 throw new IllegalArgumentException(msg);
579 pos += rowLen;
580 // check the family
581 if (pos + Bytes.SIZEOF_BYTE > endOffset) {
582 String msg = "Overflow when reading family length at position=" + pos +
583 bytesToHex(buf, offset, length);
584 LOG.warn(msg);
585 throw new IllegalArgumentException(msg);
587 int familyLen = buf[pos];
588 pos += Bytes.SIZEOF_BYTE;
589 if (familyLen < 0 || pos + familyLen > endOffset) {
590 String msg = "Invalid family length in KeyValue, familyLength=" + familyLen +
591 bytesToHex(buf, offset, length);
592 LOG.warn(msg);
593 throw new IllegalArgumentException(msg);
595 pos += familyLen;
596 // check the qualifier
597 int qualifierLen = keyLen - Bytes.SIZEOF_SHORT - rowLen - Bytes.SIZEOF_BYTE - familyLen
598 - Bytes.SIZEOF_LONG - Bytes.SIZEOF_BYTE;
599 if (qualifierLen < 0 || pos + qualifierLen > endOffset) {
600 String msg = "Invalid qualifier length in KeyValue, qualifierLen=" + qualifierLen +
601 bytesToHex(buf, offset, length);
602 LOG.warn(msg);
603 throw new IllegalArgumentException(msg);
605 pos += qualifierLen;
606 // check the timestamp
607 if (pos + Bytes.SIZEOF_LONG > endOffset) {
608 String msg =
609 "Overflow when reading timestamp at position=" + pos + bytesToHex(buf, offset, length);
610 LOG.warn(msg);
611 throw new IllegalArgumentException(msg);
613 long timestamp = Bytes.toLong(buf, pos, Bytes.SIZEOF_LONG);
614 if (timestamp < 0) {
615 String msg =
616 "Timestamp cannot be negative, ts=" + timestamp + bytesToHex(buf, offset, length);
617 LOG.warn(msg);
618 throw new IllegalArgumentException(msg);
620 pos += Bytes.SIZEOF_LONG;
621 // check the type
622 if (pos + Bytes.SIZEOF_BYTE > endOffset) {
623 String msg =
624 "Overflow when reading type at position=" + pos + bytesToHex(buf, offset, length);
625 LOG.warn(msg);
626 throw new IllegalArgumentException(msg);
628 byte type = buf[pos];
629 if (!Type.isValidType(type)) {
630 String msg = "Invalid type in KeyValue, type=" + type + bytesToHex(buf, offset, length);
631 LOG.warn(msg);
632 throw new IllegalArgumentException(msg);
634 pos += Bytes.SIZEOF_BYTE;
635 // check the value
636 if (pos + valLen > endOffset) {
637 String msg =
638 "Overflow when reading value part at position=" + pos + bytesToHex(buf, offset, length);
639 LOG.warn(msg);
640 throw new IllegalArgumentException(msg);
642 pos += valLen;
643 // check the tags
644 if (withTags) {
645 if (pos == endOffset) {
646 // withTags is true but no tag in the cell.
647 return;
649 pos = checkKeyValueTagBytes(buf, offset, length, pos, endOffset);
651 if (pos != endOffset) {
652 String msg = "Some redundant bytes in KeyValue's buffer, startOffset=" + pos + ", endOffset="
653 + endOffset + bytesToHex(buf, offset, length);
654 LOG.warn(msg);
655 throw new IllegalArgumentException(msg);
659 private static int checkKeyValueTagBytes(byte[] buf, int offset, int length, int pos,
660 int endOffset) {
661 if (pos + Bytes.SIZEOF_SHORT > endOffset) {
662 String msg = "Overflow when reading tags length at position=" + pos +
663 bytesToHex(buf, offset, length);
664 LOG.warn(msg);
665 throw new IllegalArgumentException(msg);
667 short tagsLen = Bytes.toShort(buf, pos);
668 pos += Bytes.SIZEOF_SHORT;
669 if (tagsLen < 0 || pos + tagsLen > endOffset) {
670 String msg = "Invalid tags length in KeyValue at position=" + (pos - Bytes.SIZEOF_SHORT)
671 + bytesToHex(buf, offset, length);
672 LOG.warn(msg);
673 throw new IllegalArgumentException(msg);
675 int tagsEndOffset = pos + tagsLen;
676 for (; pos < tagsEndOffset;) {
677 if (pos + Tag.TAG_LENGTH_SIZE > endOffset) {
678 String msg = "Overflow when reading tag length at position=" + pos +
679 bytesToHex(buf, offset, length);
680 LOG.warn(msg);
681 throw new IllegalArgumentException(msg);
683 short tagLen = Bytes.toShort(buf, pos);
684 pos += Tag.TAG_LENGTH_SIZE;
685 // tagLen contains one byte tag type, so must be not less than 1.
686 if (tagLen < 1 || pos + tagLen > endOffset) {
687 String msg =
688 "Invalid tag length at position=" + (pos - Tag.TAG_LENGTH_SIZE) + ", tagLength="
689 + tagLen + bytesToHex(buf, offset, length);
690 LOG.warn(msg);
691 throw new IllegalArgumentException(msg);
693 pos += tagLen;
695 return pos;
699 * Create a KeyValue reading from the raw InputStream. Named
700 * <code>createKeyValueFromInputStream</code> so doesn't clash with {@link #create(DataInput)}
701 * @param in inputStream to read.
702 * @param withTags whether the keyvalue should include tags are not
703 * @return Created KeyValue OR if we find a length of zero, we will return null which can be
704 * useful marking a stream as done.
705 * @throws IOException
707 public static KeyValue createKeyValueFromInputStream(InputStream in, boolean withTags)
708 throws IOException {
709 byte[] intBytes = new byte[Bytes.SIZEOF_INT];
710 int bytesRead = 0;
711 while (bytesRead < intBytes.length) {
712 int n = in.read(intBytes, bytesRead, intBytes.length - bytesRead);
713 if (n < 0) {
714 if (bytesRead == 0) {
715 throw new EOFException();
717 throw new IOException("Failed read of int, read " + bytesRead + " bytes");
719 bytesRead += n;
721 byte[] bytes = new byte[Bytes.toInt(intBytes)];
722 IOUtils.readFully(in, bytes, 0, bytes.length);
723 return withTags ? new KeyValue(bytes, 0, bytes.length)
724 : new NoTagsKeyValue(bytes, 0, bytes.length);
728 * @param b
729 * @return A KeyValue made of a byte array that holds the key-only part.
730 * Needed to convert hfile index members to KeyValues.
732 public static KeyValue createKeyValueFromKey(final byte[] b) {
733 return createKeyValueFromKey(b, 0, b.length);
737 * @param bb
738 * @return A KeyValue made of a byte buffer that holds the key-only part.
739 * Needed to convert hfile index members to KeyValues.
741 public static KeyValue createKeyValueFromKey(final ByteBuffer bb) {
742 return createKeyValueFromKey(bb.array(), bb.arrayOffset(), bb.limit());
746 * @param b
747 * @param o
748 * @param l
749 * @return A KeyValue made of a byte array that holds the key-only part.
750 * Needed to convert hfile index members to KeyValues.
752 public static KeyValue createKeyValueFromKey(final byte[] b, final int o, final int l) {
753 byte[] newb = new byte[l + KeyValue.ROW_OFFSET];
754 System.arraycopy(b, o, newb, KeyValue.ROW_OFFSET, l);
755 Bytes.putInt(newb, 0, l);
756 Bytes.putInt(newb, Bytes.SIZEOF_INT, 0);
757 return new KeyValue(newb);
761 * @param in
762 * Where to read bytes from. Creates a byte array to hold the
763 * KeyValue backing bytes copied from the steam.
764 * @return KeyValue created by deserializing from <code>in</code> OR if we
765 * find a length of zero, we will return null which can be useful
766 * marking a stream as done.
767 * @throws IOException
769 public static KeyValue create(final DataInput in) throws IOException {
770 return create(in.readInt(), in);
774 * Create a KeyValue reading <code>length</code> from <code>in</code>
776 * @param length
777 * @param in
778 * @return Created KeyValue OR if we find a length of zero, we will return
779 * null which can be useful marking a stream as done.
780 * @throws IOException
782 public static KeyValue create(int length, final DataInput in) throws IOException {
784 if (length <= 0) {
785 if (length == 0)
786 return null;
787 throw new IOException("Failed read " + length + " bytes, stream corrupt?");
790 // This is how the old Writables.readFrom used to deserialize. Didn't even
791 // vint.
792 byte[] bytes = new byte[length];
793 in.readFully(bytes);
794 return new KeyValue(bytes, 0, length);
797 public static int getSerializedSize(Cell cell, boolean withTags) {
798 if (withTags) {
799 return cell.getSerializedSize();
801 if (cell instanceof ExtendedCell) {
802 return ((ExtendedCell) cell).getSerializedSize(withTags);
804 return length(cell.getRowLength(), cell.getFamilyLength(), cell.getQualifierLength(),
805 cell.getValueLength(), cell.getTagsLength(), withTags);
808 public static int oswrite(final Cell cell, final OutputStream out, final boolean withTags)
809 throws IOException {
810 if (cell instanceof ExtendedCell) {
811 return ((ExtendedCell)cell).write(out, withTags);
812 } else {
813 short rlen = cell.getRowLength();
814 byte flen = cell.getFamilyLength();
815 int qlen = cell.getQualifierLength();
816 int vlen = cell.getValueLength();
817 int tlen = cell.getTagsLength();
818 int size = 0;
819 // write key length
820 int klen = keyLength(rlen, flen, qlen);
821 ByteBufferUtils.putInt(out, klen);
822 // write value length
823 ByteBufferUtils.putInt(out, vlen);
824 // Write rowkey - 2 bytes rk length followed by rowkey bytes
825 StreamUtils.writeShort(out, rlen);
826 out.write(cell.getRowArray(), cell.getRowOffset(), rlen);
827 // Write cf - 1 byte of cf length followed by the family bytes
828 out.write(flen);
829 out.write(cell.getFamilyArray(), cell.getFamilyOffset(), flen);
830 // write qualifier
831 out.write(cell.getQualifierArray(), cell.getQualifierOffset(), qlen);
832 // write timestamp
833 StreamUtils.writeLong(out, cell.getTimestamp());
834 // write the type
835 out.write(cell.getTypeByte());
836 // write value
837 out.write(cell.getValueArray(), cell.getValueOffset(), vlen);
838 size = klen + vlen + KeyValue.KEYVALUE_INFRASTRUCTURE_SIZE;
839 // write tags if we have to
840 if (withTags && tlen > 0) {
841 // 2 bytes tags length followed by tags bytes
842 // tags length is serialized with 2 bytes only(short way) even if the
843 // type is int. As this
844 // is non -ve numbers, we save the sign bit. See HBASE-11437
845 out.write((byte) (0xff & (tlen >> 8)));
846 out.write((byte) (0xff & tlen));
847 out.write(cell.getTagsArray(), cell.getTagsOffset(), tlen);
848 size += tlen + KeyValue.TAGS_LENGTH_SIZE;
850 return size;