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.client;
019
020import java.io.IOException;
021import java.nio.BufferOverflowException;
022import java.nio.ByteBuffer;
023import java.util.ArrayList;
024import java.util.Arrays;
025import java.util.Collections;
026import java.util.Comparator;
027import java.util.Iterator;
028import java.util.List;
029import java.util.Map;
030import java.util.NavigableMap;
031import java.util.NoSuchElementException;
032import java.util.TreeMap;
033import org.apache.hadoop.hbase.Cell;
034import org.apache.hadoop.hbase.CellComparator;
035import org.apache.hadoop.hbase.CellScannable;
036import org.apache.hadoop.hbase.CellScanner;
037import org.apache.hadoop.hbase.CellUtil;
038import org.apache.hadoop.hbase.HConstants;
039import org.apache.hadoop.hbase.KeyValue;
040import org.apache.hadoop.hbase.KeyValueUtil;
041import org.apache.hadoop.hbase.PrivateCellUtil;
042import org.apache.hadoop.hbase.util.Bytes;
043import org.apache.yetus.audience.InterfaceAudience;
044
045/**
046 * Single row result of a {@link Get} or {@link Scan} query.
047 * <p>
048 * This class is <b>NOT THREAD SAFE</b>.
049 * <p>
050 * Convenience methods are available that return various {@link Map} structures and values directly.
051 * <p>
052 * To get a complete mapping of all cells in the Result, which can include multiple families and
053 * multiple versions, use {@link #getMap()}.
054 * <p>
055 * To get a mapping of each family to its columns (qualifiers and values), including only the latest
056 * version of each, use {@link #getNoVersionMap()}. To get a mapping of qualifiers to latest values
057 * for an individual family use {@link #getFamilyMap(byte[])}.
058 * <p>
059 * To get the latest value for a specific family and qualifier use
060 * {@link #getValue(byte[], byte[])}. A Result is backed by an array of {@link Cell} objects, each
061 * representing an HBase cell defined by the row, family, qualifier, timestamp, and value.
062 * <p>
063 * The underlying {@link Cell} objects can be accessed through the method {@link #listCells()}. This
064 * will create a List from the internal Cell []. Better is to exploit the fact that a new Result
065 * instance is a primed {@link CellScanner}; just call {@link #advance()} and {@link #current()} to
066 * iterate over Cells as you would any {@link CellScanner}. Call {@link #cellScanner()} to reset
067 * should you need to iterate the same Result over again ({@link CellScanner}s are one-shot). If you
068 * need to overwrite a Result with another Result instance -- as in the old 'mapred' RecordReader
069 * next invocations -- then create an empty Result with the null constructor and in then use
070 * {@link #copyFrom(Result)}
071 */
072@InterfaceAudience.Public
073public class Result implements CellScannable, CellScanner {
074  private Cell[] cells;
075  private Boolean exists; // if the query was just to check existence.
076  private boolean stale = false;
077
078  /**
079   * See {@link #mayHaveMoreCellsInRow()}.
080   */
081  private boolean mayHaveMoreCellsInRow = false;
082  // We're not using java serialization. Transient here is just a marker to say
083  // that this is where we cache row if we're ever asked for it.
084  private transient byte[] row = null;
085  // Ditto for familyMap. It can be composed on fly from passed in kvs.
086  private transient NavigableMap<byte[],
087    NavigableMap<byte[], NavigableMap<Long, byte[]>>> familyMap = null;
088
089  private static ThreadLocal<byte[]> localBuffer = new ThreadLocal<>();
090  private static final int PAD_WIDTH = 128;
091  public static final Result EMPTY_RESULT = new Result(true);
092
093  private final static int INITIAL_CELLSCANNER_INDEX = -1;
094
095  /**
096   * Index for where we are when Result is acting as a {@link CellScanner}.
097   */
098  private int cellScannerIndex = INITIAL_CELLSCANNER_INDEX;
099  private RegionLoadStats stats;
100
101  private final boolean readonly;
102
103  private Cursor cursor = null;
104
105  /**
106   * Creates an empty Result w/ no KeyValue payload; returns null if you call {@link #rawCells()}.
107   * Use this to represent no results if {@code null} won't do or in old 'mapred' as opposed to
108   * 'mapreduce' package MapReduce where you need to overwrite a Result instance with a
109   * {@link #copyFrom(Result)} call.
110   */
111  public Result() {
112    this(false);
113  }
114
115  /**
116   * Allows to construct special purpose immutable Result objects, such as EMPTY_RESULT.
117   * @param readonly whether this Result instance is readonly
118   */
119  private Result(boolean readonly) {
120    this.readonly = readonly;
121  }
122
123  /**
124   * Instantiate a Result with the specified List of KeyValues. <br>
125   * <strong>Note:</strong> You must ensure that the keyvalues are already sorted.
126   * @param cells List of cells
127   */
128  public static Result create(List<Cell> cells) {
129    return create(cells, null);
130  }
131
132  public static Result create(List<Cell> cells, Boolean exists) {
133    return create(cells, exists, false);
134  }
135
136  public static Result create(List<Cell> cells, Boolean exists, boolean stale) {
137    return create(cells, exists, stale, false);
138  }
139
140  public static Result create(List<Cell> cells, Boolean exists, boolean stale,
141    boolean mayHaveMoreCellsInRow) {
142    if (exists != null) {
143      return new Result(null, exists, stale, mayHaveMoreCellsInRow);
144    }
145    return new Result(cells.toArray(new Cell[cells.size()]), null, stale, mayHaveMoreCellsInRow);
146  }
147
148  /**
149   * Instantiate a Result with the specified array of KeyValues. <br>
150   * <strong>Note:</strong> You must ensure that the keyvalues are already sorted.
151   * @param cells array of cells
152   */
153  public static Result create(Cell[] cells) {
154    return create(cells, null, false);
155  }
156
157  public static Result create(Cell[] cells, Boolean exists, boolean stale) {
158    return create(cells, exists, stale, false);
159  }
160
161  public static Result create(Cell[] cells, Boolean exists, boolean stale,
162    boolean mayHaveMoreCellsInRow) {
163    if (exists != null) {
164      return new Result(null, exists, stale, mayHaveMoreCellsInRow);
165    }
166    return new Result(cells, null, stale, mayHaveMoreCellsInRow);
167  }
168
169  public static Result createCursorResult(Cursor cursor) {
170    return new Result(cursor);
171  }
172
173  private Result(Cursor cursor) {
174    this.cursor = cursor;
175    this.readonly = false;
176  }
177
178  /** Private ctor. Use {@link #create(Cell[])}. */
179  private Result(Cell[] cells, Boolean exists, boolean stale, boolean mayHaveMoreCellsInRow) {
180    this.cells = cells;
181    this.exists = exists;
182    this.stale = stale;
183    this.mayHaveMoreCellsInRow = mayHaveMoreCellsInRow;
184    this.readonly = false;
185  }
186
187  /**
188   * Method for retrieving the row key that corresponds to the row from which this Result was
189   * created. n
190   */
191  public byte[] getRow() {
192    if (this.row == null) {
193      this.row =
194        (this.cells == null || this.cells.length == 0) ? null : CellUtil.cloneRow(this.cells[0]);
195    }
196    return this.row;
197  }
198
199  /**
200   * Return the array of Cells backing this Result instance. The array is sorted from smallest -&gt;
201   * largest using the {@link CellComparator}. The array only contains what your Get or Scan
202   * specifies and no more. For example if you request column "A" 1 version you will have at most 1
203   * Cell in the array. If you request column "A" with 2 version you will have at most 2 Cells, with
204   * the first one being the newer timestamp and the second being the older timestamp (this is the
205   * sort order defined by {@link CellComparator}). If columns don't exist, they won't be present in
206   * the result. Therefore if you ask for 1 version all columns, it is safe to iterate over this
207   * array and expect to see 1 Cell for each column and no more. This API is faster than using
208   * getFamilyMap() and getMap()
209   * @return array of Cells; can be null if nothing in the result
210   */
211  public Cell[] rawCells() {
212    return cells;
213  }
214
215  /**
216   * Create a sorted list of the Cell's in this result. Since HBase 0.20.5 this is equivalent to
217   * raw().
218   * @return sorted List of Cells; can be null if no cells in the result
219   */
220  public List<Cell> listCells() {
221    return isEmpty() ? null : Arrays.asList(rawCells());
222  }
223
224  /**
225   * Return the Cells for the specific column. The Cells are sorted in the {@link CellComparator}
226   * order. That implies the first entry in the list is the most recent column. If the query (Scan
227   * or Get) only requested 1 version the list will contain at most 1 entry. If the column did not
228   * exist in the result set (either the column does not exist or the column was not selected in the
229   * query) the list will be empty. Also see getColumnLatest which returns just a Cell
230   * @param family the family n * @return a list of Cells for this column or empty list if the
231   *               column did not exist in the result set
232   */
233  public List<Cell> getColumnCells(byte[] family, byte[] qualifier) {
234    List<Cell> result = new ArrayList<>();
235
236    Cell[] kvs = rawCells();
237
238    if (kvs == null || kvs.length == 0) {
239      return result;
240    }
241    int pos = binarySearch(kvs, family, qualifier);
242    if (pos == -1) {
243      return result; // cant find it
244    }
245
246    for (int i = pos; i < kvs.length; i++) {
247      if (CellUtil.matchingColumn(kvs[i], family, qualifier)) {
248        result.add(kvs[i]);
249      } else {
250        break;
251      }
252    }
253
254    return result;
255  }
256
257  private byte[] notNullBytes(final byte[] bytes) {
258    if (bytes == null) {
259      return HConstants.EMPTY_BYTE_ARRAY;
260    } else {
261      return bytes;
262    }
263  }
264
265  protected int binarySearch(final Cell[] kvs, final byte[] family, final byte[] qualifier) {
266    byte[] familyNotNull = notNullBytes(family);
267    byte[] qualifierNotNull = notNullBytes(qualifier);
268    Cell searchTerm = PrivateCellUtil.createFirstOnRow(kvs[0].getRowArray(), kvs[0].getRowOffset(),
269      kvs[0].getRowLength(), familyNotNull, 0, (byte) familyNotNull.length, qualifierNotNull, 0,
270      qualifierNotNull.length);
271
272    // pos === ( -(insertion point) - 1)
273    int pos = Arrays.binarySearch(kvs, searchTerm, CellComparator.getInstance());
274    // never will exact match
275    if (pos < 0) {
276      pos = (pos + 1) * -1;
277      // pos is now insertion point
278    }
279    if (pos == kvs.length) {
280      return -1; // doesn't exist
281    }
282    return pos;
283  }
284
285  /**
286   * Searches for the latest value for the specified column.
287   * @param kvs       the array to search
288   * @param family    family name
289   * @param foffset   family offset
290   * @param flength   family length
291   * @param qualifier column qualifier
292   * @param qoffset   qualifier offset
293   * @param qlength   qualifier length
294   * @return the index where the value was found, or -1 otherwise
295   */
296  protected int binarySearch(final Cell[] kvs, final byte[] family, final int foffset,
297    final int flength, final byte[] qualifier, final int qoffset, final int qlength) {
298
299    double keyValueSize =
300      (double) KeyValue.getKeyValueDataStructureSize(kvs[0].getRowLength(), flength, qlength, 0);
301
302    byte[] buffer = localBuffer.get();
303    if (buffer == null || keyValueSize > buffer.length) {
304      // pad to the smallest multiple of the pad width
305      buffer = new byte[(int) Math.ceil(keyValueSize / PAD_WIDTH) * PAD_WIDTH];
306      localBuffer.set(buffer);
307    }
308
309    Cell searchTerm =
310      KeyValueUtil.createFirstOnRow(buffer, 0, kvs[0].getRowArray(), kvs[0].getRowOffset(),
311        kvs[0].getRowLength(), family, foffset, flength, qualifier, qoffset, qlength);
312
313    // pos === ( -(insertion point) - 1)
314    int pos = Arrays.binarySearch(kvs, searchTerm, CellComparator.getInstance());
315    // never will exact match
316    if (pos < 0) {
317      pos = (pos + 1) * -1;
318      // pos is now insertion point
319    }
320    if (pos == kvs.length) {
321      return -1; // doesn't exist
322    }
323    return pos;
324  }
325
326  /**
327   * The Cell for the most recent timestamp for a given column. nn *
328   * @return the Cell for the column, or null if no value exists in the row or none have been
329   *         selected in the query (Get/Scan)
330   */
331  public Cell getColumnLatestCell(byte[] family, byte[] qualifier) {
332    Cell[] kvs = rawCells(); // side effect possibly.
333    if (kvs == null || kvs.length == 0) {
334      return null;
335    }
336    int pos = binarySearch(kvs, family, qualifier);
337    if (pos == -1) {
338      return null;
339    }
340    if (CellUtil.matchingColumn(kvs[pos], family, qualifier)) {
341      return kvs[pos];
342    }
343    return null;
344  }
345
346  /**
347   * The Cell for the most recent timestamp for a given column.
348   * @param family    family name
349   * @param foffset   family offset
350   * @param flength   family length
351   * @param qualifier column qualifier
352   * @param qoffset   qualifier offset
353   * @param qlength   qualifier length
354   * @return the Cell for the column, or null if no value exists in the row or none have been
355   *         selected in the query (Get/Scan)
356   */
357  public Cell getColumnLatestCell(byte[] family, int foffset, int flength, byte[] qualifier,
358    int qoffset, int qlength) {
359
360    Cell[] kvs = rawCells(); // side effect possibly.
361    if (kvs == null || kvs.length == 0) {
362      return null;
363    }
364    int pos = binarySearch(kvs, family, foffset, flength, qualifier, qoffset, qlength);
365    if (pos == -1) {
366      return null;
367    }
368    if (
369      PrivateCellUtil.matchingColumn(kvs[pos], family, foffset, flength, qualifier, qoffset,
370        qlength)
371    ) {
372      return kvs[pos];
373    }
374    return null;
375  }
376
377  /**
378   * Get the latest version of the specified column. Note: this call clones the value content of the
379   * hosting Cell. See {@link #getValueAsByteBuffer(byte[], byte[])}, etc., or {@link #listCells()}
380   * if you would avoid the cloning.
381   * @param family    family name
382   * @param qualifier column qualifier
383   * @return value of latest version of column, null if none found
384   */
385  public byte[] getValue(byte[] family, byte[] qualifier) {
386    Cell kv = getColumnLatestCell(family, qualifier);
387    if (kv == null) {
388      return null;
389    }
390    return CellUtil.cloneValue(kv);
391  }
392
393  /**
394   * Returns the value wrapped in a new <code>ByteBuffer</code>.
395   * @param family    family name
396   * @param qualifier column qualifier
397   * @return the latest version of the column, or <code>null</code> if none found
398   */
399  public ByteBuffer getValueAsByteBuffer(byte[] family, byte[] qualifier) {
400
401    Cell kv = getColumnLatestCell(family, 0, family.length, qualifier, 0, qualifier.length);
402
403    if (kv == null) {
404      return null;
405    }
406    return ByteBuffer.wrap(kv.getValueArray(), kv.getValueOffset(), kv.getValueLength())
407      .asReadOnlyBuffer();
408  }
409
410  /**
411   * Returns the value wrapped in a new <code>ByteBuffer</code>.
412   * @param family    family name
413   * @param foffset   family offset
414   * @param flength   family length
415   * @param qualifier column qualifier
416   * @param qoffset   qualifier offset
417   * @param qlength   qualifier length
418   * @return the latest version of the column, or <code>null</code> if none found
419   */
420  public ByteBuffer getValueAsByteBuffer(byte[] family, int foffset, int flength, byte[] qualifier,
421    int qoffset, int qlength) {
422
423    Cell kv = getColumnLatestCell(family, foffset, flength, qualifier, qoffset, qlength);
424
425    if (kv == null) {
426      return null;
427    }
428    return ByteBuffer.wrap(kv.getValueArray(), kv.getValueOffset(), kv.getValueLength())
429      .asReadOnlyBuffer();
430  }
431
432  /**
433   * Loads the latest version of the specified column into the provided <code>ByteBuffer</code>.
434   * <p>
435   * Does not clear or flip the buffer.
436   * @param family    family name
437   * @param qualifier column qualifier
438   * @param dst       the buffer where to write the value
439   * @return <code>true</code> if a value was found, <code>false</code> otherwise
440   * @throws BufferOverflowException there is insufficient space remaining in the buffer
441   */
442  public boolean loadValue(byte[] family, byte[] qualifier, ByteBuffer dst)
443    throws BufferOverflowException {
444    return loadValue(family, 0, family.length, qualifier, 0, qualifier.length, dst);
445  }
446
447  /**
448   * Loads the latest version of the specified column into the provided <code>ByteBuffer</code>.
449   * <p>
450   * Does not clear or flip the buffer.
451   * @param family    family name
452   * @param foffset   family offset
453   * @param flength   family length
454   * @param qualifier column qualifier
455   * @param qoffset   qualifier offset
456   * @param qlength   qualifier length
457   * @param dst       the buffer where to write the value
458   * @return <code>true</code> if a value was found, <code>false</code> otherwise
459   * @throws BufferOverflowException there is insufficient space remaining in the buffer
460   */
461  public boolean loadValue(byte[] family, int foffset, int flength, byte[] qualifier, int qoffset,
462    int qlength, ByteBuffer dst) throws BufferOverflowException {
463    Cell kv = getColumnLatestCell(family, foffset, flength, qualifier, qoffset, qlength);
464
465    if (kv == null) {
466      return false;
467    }
468    dst.put(kv.getValueArray(), kv.getValueOffset(), kv.getValueLength());
469    return true;
470  }
471
472  /**
473   * Checks if the specified column contains a non-empty value (not a zero-length byte array).
474   * @param family    family name
475   * @param qualifier column qualifier
476   * @return whether or not a latest value exists and is not empty
477   */
478  public boolean containsNonEmptyColumn(byte[] family, byte[] qualifier) {
479
480    return containsNonEmptyColumn(family, 0, family.length, qualifier, 0, qualifier.length);
481  }
482
483  /**
484   * Checks if the specified column contains a non-empty value (not a zero-length byte array).
485   * @param family    family name
486   * @param foffset   family offset
487   * @param flength   family length
488   * @param qualifier column qualifier
489   * @param qoffset   qualifier offset
490   * @param qlength   qualifier length
491   * @return whether or not a latest value exists and is not empty
492   */
493  public boolean containsNonEmptyColumn(byte[] family, int foffset, int flength, byte[] qualifier,
494    int qoffset, int qlength) {
495
496    Cell kv = getColumnLatestCell(family, foffset, flength, qualifier, qoffset, qlength);
497
498    return (kv != null) && (kv.getValueLength() > 0);
499  }
500
501  /**
502   * Checks if the specified column contains an empty value (a zero-length byte array).
503   * @param family    family name
504   * @param qualifier column qualifier
505   * @return whether or not a latest value exists and is empty
506   */
507  public boolean containsEmptyColumn(byte[] family, byte[] qualifier) {
508
509    return containsEmptyColumn(family, 0, family.length, qualifier, 0, qualifier.length);
510  }
511
512  /**
513   * Checks if the specified column contains an empty value (a zero-length byte array).
514   * @param family    family name
515   * @param foffset   family offset
516   * @param flength   family length
517   * @param qualifier column qualifier
518   * @param qoffset   qualifier offset
519   * @param qlength   qualifier length
520   * @return whether or not a latest value exists and is empty
521   */
522  public boolean containsEmptyColumn(byte[] family, int foffset, int flength, byte[] qualifier,
523    int qoffset, int qlength) {
524    Cell kv = getColumnLatestCell(family, foffset, flength, qualifier, qoffset, qlength);
525
526    return (kv != null) && (kv.getValueLength() == 0);
527  }
528
529  /**
530   * Checks for existence of a value for the specified column (empty or not).
531   * @param family    family name
532   * @param qualifier column qualifier
533   * @return true if at least one value exists in the result, false if not
534   */
535  public boolean containsColumn(byte[] family, byte[] qualifier) {
536    Cell kv = getColumnLatestCell(family, qualifier);
537    return kv != null;
538  }
539
540  /**
541   * Checks for existence of a value for the specified column (empty or not).
542   * @param family    family name
543   * @param foffset   family offset
544   * @param flength   family length
545   * @param qualifier column qualifier
546   * @param qoffset   qualifier offset
547   * @param qlength   qualifier length
548   * @return true if at least one value exists in the result, false if not
549   */
550  public boolean containsColumn(byte[] family, int foffset, int flength, byte[] qualifier,
551    int qoffset, int qlength) {
552
553    return getColumnLatestCell(family, foffset, flength, qualifier, qoffset, qlength) != null;
554  }
555
556  /**
557   * Map of families to all versions of its qualifiers and values.
558   * <p>
559   * Returns a three level Map of the form:
560   * <code>Map&amp;family,Map&lt;qualifier,Map&lt;timestamp,value&gt;&gt;&gt;</code>
561   * <p>
562   * Note: All other map returning methods make use of this map internally.
563   * @return map from families to qualifiers to versions
564   */
565  public NavigableMap<byte[], NavigableMap<byte[], NavigableMap<Long, byte[]>>> getMap() {
566    if (this.familyMap != null) {
567      return this.familyMap;
568    }
569    if (isEmpty()) {
570      return null;
571    }
572    this.familyMap = new TreeMap<>(Bytes.BYTES_COMPARATOR);
573    for (Cell kv : this.cells) {
574      byte[] family = CellUtil.cloneFamily(kv);
575      NavigableMap<byte[], NavigableMap<Long, byte[]>> columnMap = familyMap.get(family);
576      if (columnMap == null) {
577        columnMap = new TreeMap<>(Bytes.BYTES_COMPARATOR);
578        familyMap.put(family, columnMap);
579      }
580      byte[] qualifier = CellUtil.cloneQualifier(kv);
581      NavigableMap<Long, byte[]> versionMap = columnMap.get(qualifier);
582      if (versionMap == null) {
583        versionMap = new TreeMap<>(new Comparator<Long>() {
584          @Override
585          public int compare(Long l1, Long l2) {
586            return l2.compareTo(l1);
587          }
588        });
589        columnMap.put(qualifier, versionMap);
590      }
591      Long timestamp = kv.getTimestamp();
592      byte[] value = CellUtil.cloneValue(kv);
593
594      versionMap.put(timestamp, value);
595    }
596    return this.familyMap;
597  }
598
599  /**
600   * Map of families to their most recent qualifiers and values.
601   * <p>
602   * Returns a two level Map of the form: <code>Map&amp;family,Map&lt;qualifier,value&gt;&gt;</code>
603   * <p>
604   * The most recent version of each qualifier will be used.
605   * @return map from families to qualifiers and value
606   */
607  public NavigableMap<byte[], NavigableMap<byte[], byte[]>> getNoVersionMap() {
608    if (this.familyMap == null) {
609      getMap();
610    }
611    if (isEmpty()) {
612      return null;
613    }
614    NavigableMap<byte[], NavigableMap<byte[], byte[]>> returnMap =
615      new TreeMap<>(Bytes.BYTES_COMPARATOR);
616    for (Map.Entry<byte[], NavigableMap<byte[], NavigableMap<Long, byte[]>>> familyEntry : familyMap
617      .entrySet()) {
618      NavigableMap<byte[], byte[]> qualifierMap = new TreeMap<>(Bytes.BYTES_COMPARATOR);
619      for (Map.Entry<byte[], NavigableMap<Long, byte[]>> qualifierEntry : familyEntry.getValue()
620        .entrySet()) {
621        byte[] value = qualifierEntry.getValue().get(qualifierEntry.getValue().firstKey());
622        qualifierMap.put(qualifierEntry.getKey(), value);
623      }
624      returnMap.put(familyEntry.getKey(), qualifierMap);
625    }
626    return returnMap;
627  }
628
629  /**
630   * Map of qualifiers to values.
631   * <p>
632   * Returns a Map of the form: <code>Map&lt;qualifier,value&gt;</code>
633   * @param family column family to get
634   * @return map of qualifiers to values
635   */
636  public NavigableMap<byte[], byte[]> getFamilyMap(byte[] family) {
637    if (this.familyMap == null) {
638      getMap();
639    }
640    if (isEmpty()) {
641      return null;
642    }
643    NavigableMap<byte[], byte[]> returnMap = new TreeMap<>(Bytes.BYTES_COMPARATOR);
644    NavigableMap<byte[], NavigableMap<Long, byte[]>> qualifierMap = familyMap.get(family);
645    if (qualifierMap == null) {
646      return returnMap;
647    }
648    for (Map.Entry<byte[], NavigableMap<Long, byte[]>> entry : qualifierMap.entrySet()) {
649      byte[] value = entry.getValue().get(entry.getValue().firstKey());
650      returnMap.put(entry.getKey(), value);
651    }
652    return returnMap;
653  }
654
655  /**
656   * Returns the value of the first column in the Result.
657   * @return value of the first column
658   */
659  public byte[] value() {
660    if (isEmpty()) {
661      return null;
662    }
663    return CellUtil.cloneValue(cells[0]);
664  }
665
666  /**
667   * Check if the underlying Cell [] is empty or not
668   * @return true if empty
669   */
670  public boolean isEmpty() {
671    return this.cells == null || this.cells.length == 0;
672  }
673
674  /** Returns the size of the underlying Cell [] */
675  public int size() {
676    return this.cells == null ? 0 : this.cells.length;
677  }
678
679  /**
680   * n
681   */
682  @Override
683  public String toString() {
684    StringBuilder sb = new StringBuilder();
685    sb.append("keyvalues=");
686    if (isEmpty()) {
687      sb.append("NONE");
688      return sb.toString();
689    }
690    sb.append("{");
691    boolean moreThanOne = false;
692    for (Cell kv : this.cells) {
693      if (moreThanOne) {
694        sb.append(", ");
695      } else {
696        moreThanOne = true;
697      }
698      sb.append(kv.toString());
699    }
700    sb.append("}");
701    return sb.toString();
702  }
703
704  /**
705   * Does a deep comparison of two Results, down to the byte arrays.
706   * @param res1 first result to compare
707   * @param res2 second result to compare
708   * @throws Exception Every difference is throwing an exception
709   */
710  public static void compareResults(Result res1, Result res2) throws Exception {
711    compareResults(res1, res2, true);
712  }
713
714  /**
715   * Does a deep comparison of two Results, down to the byte arrays.
716   * @param res1    first result to compare
717   * @param res2    second result to compare
718   * @param verbose includes string representation for all cells in the exception if true; otherwise
719   *                include rowkey only
720   * @throws Exception Every difference is throwing an exception
721   */
722  public static void compareResults(Result res1, Result res2, boolean verbose) throws Exception {
723    if (res2 == null) {
724      throw new Exception(
725        "There wasn't enough rows, we stopped at " + Bytes.toStringBinary(res1.getRow()));
726    }
727    if (res1.size() != res2.size()) {
728      if (verbose) {
729        throw new Exception(
730          "This row doesn't have the same number of KVs: " + res1 + " compared to " + res2);
731      } else {
732        throw new Exception(
733          "This row doesn't have the same number of KVs: row=" + Bytes.toStringBinary(res1.getRow())
734            + ", " + res1.size() + " cells are compared to " + res2.size() + " cells");
735      }
736    }
737    Cell[] ourKVs = res1.rawCells();
738    Cell[] replicatedKVs = res2.rawCells();
739    for (int i = 0; i < res1.size(); i++) {
740      if (
741        !ourKVs[i].equals(replicatedKVs[i]) || !CellUtil.matchingValue(ourKVs[i], replicatedKVs[i])
742          || !CellUtil.matchingTags(ourKVs[i], replicatedKVs[i])
743      ) {
744        if (verbose) {
745          throw new Exception("This result was different: " + res1 + " compared to " + res2);
746        } else {
747          throw new Exception(
748            "This result was different: row=" + Bytes.toStringBinary(res1.getRow()));
749        }
750      }
751    }
752  }
753
754  /**
755   * Forms a single result from the partial results in the partialResults list. This method is
756   * useful for reconstructing partial results on the client side.
757   * @param partialResults list of partial results
758   * @return The complete result that is formed by combining all of the partial results together
759   * @throws IOException A complete result cannot be formed because the results in the partial list
760   *                     come from different rows
761   */
762  public static Result createCompleteResult(Iterable<Result> partialResults) throws IOException {
763    if (partialResults == null) {
764      return Result.create(Collections.emptyList(), null, false);
765    }
766    List<Cell> cells = new ArrayList<>();
767    boolean stale = false;
768    byte[] prevRow = null;
769    byte[] currentRow = null;
770    for (Iterator<Result> iter = partialResults.iterator(); iter.hasNext();) {
771      Result r = iter.next();
772      currentRow = r.getRow();
773      if (prevRow != null && !Bytes.equals(prevRow, currentRow)) {
774        throw new IOException("Cannot form complete result. Rows of partial results do not match."
775          + " Partial Results: " + partialResults);
776      }
777      // Ensure that all Results except the last one are marked as partials. The last result
778      // may not be marked as a partial because Results are only marked as partials when
779      // the scan on the server side must be stopped due to reaching the maxResultSize.
780      // Visualizing it makes it easier to understand:
781      // maxResultSize: 2 cells
782      // (-x-) represents cell number x in a row
783      // Example: row1: -1- -2- -3- -4- -5- (5 cells total)
784      // How row1 will be returned by the server as partial Results:
785      // Result1: -1- -2- (2 cells, size limit reached, mark as partial)
786      // Result2: -3- -4- (2 cells, size limit reached, mark as partial)
787      // Result3: -5- (1 cell, size limit NOT reached, NOT marked as partial)
788      if (iter.hasNext() && !r.mayHaveMoreCellsInRow()) {
789        throw new IOException("Cannot form complete result. Result is missing partial flag. "
790          + "Partial Results: " + partialResults);
791      }
792      prevRow = currentRow;
793      stale = stale || r.isStale();
794      for (Cell c : r.rawCells()) {
795        cells.add(c);
796      }
797    }
798
799    return Result.create(cells, null, stale);
800  }
801
802  /**
803   * Get total size of raw cells n * @return Total size.
804   */
805  public static long getTotalSizeOfCells(Result result) {
806    long size = 0;
807    if (result.isEmpty()) {
808      return size;
809    }
810    for (Cell c : result.rawCells()) {
811      size += c.heapSize();
812    }
813    return size;
814  }
815
816  /**
817   * Copy another Result into this one. Needed for the old Mapred framework
818   * @throws UnsupportedOperationException if invoked on instance of EMPTY_RESULT (which is supposed
819   *                                       to be immutable). n
820   */
821  public void copyFrom(Result other) {
822    checkReadonly();
823    this.row = null;
824    this.familyMap = null;
825    this.cells = other.cells;
826  }
827
828  @Override
829  public CellScanner cellScanner() {
830    // Reset
831    this.cellScannerIndex = INITIAL_CELLSCANNER_INDEX;
832    return this;
833  }
834
835  @Override
836  public Cell current() {
837    if (
838      isEmpty() || cellScannerIndex == INITIAL_CELLSCANNER_INDEX || cellScannerIndex >= cells.length
839    ) {
840      return null;
841    }
842    return this.cells[cellScannerIndex];
843  }
844
845  @Override
846  public boolean advance() {
847    if (isEmpty()) {
848      return false;
849    }
850    cellScannerIndex++;
851    if (cellScannerIndex < this.cells.length) {
852      return true;
853    } else if (cellScannerIndex == this.cells.length) {
854      return false;
855    }
856    throw new NoSuchElementException("Cannot advance beyond the last cell");
857  }
858
859  public Boolean getExists() {
860    return exists;
861  }
862
863  public void setExists(Boolean exists) {
864    checkReadonly();
865    this.exists = exists;
866  }
867
868  /**
869   * Whether or not the results are coming from possibly stale data. Stale results might be returned
870   * if {@link Consistency} is not STRONG for the query.
871   * @return Whether or not the results are coming from possibly stale data.
872   */
873  public boolean isStale() {
874    return stale;
875  }
876
877  /**
878   * Whether or not the results are partial.
879   * @deprecated the word 'partial' ambiguous, use {@link #mayHaveMoreCellsInRow()} instead.
880   *             Deprecated since 1.4.0.
881   * @see #mayHaveMoreCellsInRow()
882   */
883  @Deprecated
884  public boolean isPartial() {
885    return mayHaveMoreCellsInRow;
886  }
887
888  /**
889   * For scanning large rows, the RS may choose to return the cells chunk by chunk to prevent OOM or
890   * timeout. This flag is used to tell you if the current Result is the last one of the current
891   * row. False means this Result is the last one. True means there MAY be more cells belonging to
892   * the current row. If you don't use {@link Scan#setAllowPartialResults(boolean)} or
893   * {@link Scan#setBatch(int)}, this method will always return false because the Result must
894   * contains all cells in one Row.
895   */
896  public boolean mayHaveMoreCellsInRow() {
897    return mayHaveMoreCellsInRow;
898  }
899
900  /**
901   * Set load information about the region to the information about the result
902   * @param loadStats statistics about the current region from which this was returned
903   */
904  @InterfaceAudience.Private
905  public void setStatistics(RegionLoadStats loadStats) {
906    this.stats = loadStats;
907  }
908
909  /**
910   * Returns the associated statistics about the region from which this was returned. Can be
911   * <tt>null</tt> if stats are disabled.
912   */
913  public RegionLoadStats getStats() {
914    return stats;
915  }
916
917  /**
918   * All methods modifying state of Result object must call this method to ensure that special
919   * purpose immutable Results can't be accidentally modified.
920   */
921  private void checkReadonly() {
922    if (readonly == true) {
923      throw new UnsupportedOperationException("Attempting to modify readonly EMPTY_RESULT!");
924    }
925  }
926
927  /**
928   * Return true if this Result is a cursor to tell users where the server has scanned. In this
929   * Result the only meaningful method is {@link #getCursor()}. {@code
930   *  while (r = scanner.next() && r != null) {
931   *    if(r.isCursor()){
932   *    // scanning is not end, it is a cursor, save its row key and close scanner if you want, or
933   * // just continue the loop to call next(). } else { // just like before } } // scanning is end }
934   * {@link Scan#setNeedCursorResult(boolean)} {@link Cursor} {@link #getCursor()}
935   */
936  public boolean isCursor() {
937    return cursor != null;
938  }
939
940  /**
941   * Return the cursor if this Result is a cursor result. {@link Scan#setNeedCursorResult(boolean)}
942   * {@link Cursor} {@link #isCursor()}
943   */
944  public Cursor getCursor() {
945    return cursor;
946  }
947}