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.ByteBuffer;
022import java.util.ArrayList;
023import java.util.HashMap;
024import java.util.List;
025import java.util.Map;
026import java.util.NavigableSet;
027import java.util.Set;
028import java.util.TreeMap;
029import java.util.TreeSet;
030import java.util.stream.Collectors;
031import org.apache.hadoop.hbase.HConstants;
032import org.apache.hadoop.hbase.filter.Filter;
033import org.apache.hadoop.hbase.io.TimeRange;
034import org.apache.hadoop.hbase.security.access.Permission;
035import org.apache.hadoop.hbase.security.visibility.Authorizations;
036import org.apache.hadoop.hbase.util.Bytes;
037import org.apache.yetus.audience.InterfaceAudience;
038import org.slf4j.Logger;
039import org.slf4j.LoggerFactory;
040
041/**
042 * Used to perform Get operations on a single row.
043 * <p>
044 * To get everything for a row, instantiate a Get object with the row to get. To further narrow the
045 * scope of what to Get, use the methods below.
046 * <p>
047 * To get all columns from specific families, execute {@link #addFamily(byte[]) addFamily} for each
048 * family to retrieve.
049 * <p>
050 * To get specific columns, execute {@link #addColumn(byte[], byte[]) addColumn} for each column to
051 * retrieve.
052 * <p>
053 * To only retrieve columns within a specific range of version timestamps, execute
054 * {@link #setTimeRange(long, long) setTimeRange}.
055 * <p>
056 * To only retrieve columns with a specific timestamp, execute {@link #setTimestamp(long)
057 * setTimestamp}.
058 * <p>
059 * To limit the number of versions of each column to be returned, execute {@link #readVersions(int)
060 * readVersions}.
061 * <p>
062 * To add a filter, call {@link #setFilter(Filter) setFilter}.
063 */
064@InterfaceAudience.Public
065public class Get extends Query implements Row {
066  private static final Logger LOG = LoggerFactory.getLogger(Get.class);
067
068  private byte[] row = null;
069  private int maxVersions = 1;
070  private boolean cacheBlocks = true;
071  private int storeLimit = -1;
072  private int storeOffset = 0;
073  private TimeRange tr = TimeRange.allTime();
074  private boolean checkExistenceOnly = false;
075  private Map<byte[], NavigableSet<byte[]>> familyMap = new TreeMap<>(Bytes.BYTES_COMPARATOR);
076
077  /**
078   * Create a Get operation for the specified row.
079   * <p>
080   * If no further operations are done, this will get the latest version of all columns in all
081   * families of the specified row.
082   * @param row row key
083   */
084  public Get(byte[] row) {
085    Mutation.checkRow(row);
086    this.row = row;
087  }
088
089  /**
090   * Copy-constructor
091   */
092  public Get(Get get) {
093    this(get.getRow());
094    // from Query
095    this.setFilter(get.getFilter());
096    this.setReplicaId(get.getReplicaId());
097    this.setConsistency(get.getConsistency());
098    // from Get
099    this.cacheBlocks = get.getCacheBlocks();
100    this.maxVersions = get.getMaxVersions();
101    this.storeLimit = get.getMaxResultsPerColumnFamily();
102    this.storeOffset = get.getRowOffsetPerColumnFamily();
103    this.tr = get.getTimeRange();
104    this.checkExistenceOnly = get.isCheckExistenceOnly();
105    this.loadColumnFamiliesOnDemand = get.getLoadColumnFamiliesOnDemandValue();
106    Map<byte[], NavigableSet<byte[]>> fams = get.getFamilyMap();
107    for (Map.Entry<byte[], NavigableSet<byte[]>> entry : fams.entrySet()) {
108      byte[] fam = entry.getKey();
109      NavigableSet<byte[]> cols = entry.getValue();
110      if (cols != null && cols.size() > 0) {
111        for (byte[] col : cols) {
112          addColumn(fam, col);
113        }
114      } else {
115        addFamily(fam);
116      }
117    }
118    for (Map.Entry<String, byte[]> attr : get.getAttributesMap().entrySet()) {
119      setAttribute(attr.getKey(), attr.getValue());
120    }
121    for (Map.Entry<byte[], TimeRange> entry : get.getColumnFamilyTimeRange().entrySet()) {
122      TimeRange tr = entry.getValue();
123      setColumnFamilyTimeRange(entry.getKey(), tr.getMin(), tr.getMax());
124    }
125    super.setPriority(get.getPriority());
126  }
127
128  /**
129   * Create a Get operation for the specified row.
130   */
131  public Get(byte[] row, int rowOffset, int rowLength) {
132    Mutation.checkRow(row, rowOffset, rowLength);
133    this.row = Bytes.copy(row, rowOffset, rowLength);
134  }
135
136  /**
137   * Create a Get operation for the specified row.
138   */
139  public Get(ByteBuffer row) {
140    Mutation.checkRow(row);
141    this.row = new byte[row.remaining()];
142    row.get(this.row);
143  }
144
145  public boolean isCheckExistenceOnly() {
146    return checkExistenceOnly;
147  }
148
149  public Get setCheckExistenceOnly(boolean checkExistenceOnly) {
150    this.checkExistenceOnly = checkExistenceOnly;
151    return this;
152  }
153
154  /**
155   * Get all columns from the specified family.
156   * <p>
157   * Overrides previous calls to addColumn for this family.
158   * @param family family name
159   * @return the Get object
160   */
161  public Get addFamily(byte[] family) {
162    familyMap.remove(family);
163    familyMap.put(family, null);
164    return this;
165  }
166
167  /**
168   * Get the column from the specific family with the specified qualifier.
169   * <p>
170   * Overrides previous calls to addFamily for this family.
171   * @param family    family name
172   * @param qualifier column qualifier
173   * @return the Get objec
174   */
175  public Get addColumn(byte[] family, byte[] qualifier) {
176    NavigableSet<byte[]> set = familyMap.get(family);
177    if (set == null) {
178      set = new TreeSet<>(Bytes.BYTES_COMPARATOR);
179      familyMap.put(family, set);
180    }
181    if (qualifier == null) {
182      qualifier = HConstants.EMPTY_BYTE_ARRAY;
183    }
184    set.add(qualifier);
185    return this;
186  }
187
188  /**
189   * Get versions of columns only within the specified timestamp range, [minStamp, maxStamp).
190   * @param minStamp minimum timestamp value, inclusive
191   * @param maxStamp maximum timestamp value, exclusive
192   * @return this for invocation chaining
193   */
194  public Get setTimeRange(long minStamp, long maxStamp) throws IOException {
195    tr = TimeRange.between(minStamp, maxStamp);
196    return this;
197  }
198
199  /**
200   * Get versions of columns with the specified timestamp.
201   * @param timestamp version timestamp
202   * @return this for invocation chaining
203   */
204  public Get setTimestamp(long timestamp) {
205    try {
206      tr = TimeRange.at(timestamp);
207    } catch (Exception e) {
208      // This should never happen, unless integer overflow or something extremely wrong...
209      LOG.error("TimeRange failed, likely caused by integer overflow. ", e);
210      throw e;
211    }
212    return this;
213  }
214
215  @Override
216  public Get setColumnFamilyTimeRange(byte[] cf, long minStamp, long maxStamp) {
217    super.setColumnFamilyTimeRange(cf, minStamp, maxStamp);
218    return this;
219  }
220
221  /**
222   * Get all available versions.
223   * @return this for invocation chaining
224   */
225  public Get readAllVersions() {
226    this.maxVersions = Integer.MAX_VALUE;
227    return this;
228  }
229
230  /**
231   * Get up to the specified number of versions of each column.
232   * @param versions specified number of versions for each column
233   * @throws IOException if invalid number of versions
234   * @return this for invocation chaining
235   */
236  public Get readVersions(int versions) throws IOException {
237    if (versions <= 0) {
238      throw new IOException("versions must be positive");
239    }
240    this.maxVersions = versions;
241    return this;
242  }
243
244  @Override
245  public Get setLoadColumnFamiliesOnDemand(boolean value) {
246    super.setLoadColumnFamiliesOnDemand(value);
247    return this;
248  }
249
250  /**
251   * Set the maximum number of values to return per row per Column Family
252   * @param limit the maximum number of values returned / row / CF
253   * @return this for invocation chaining
254   */
255  public Get setMaxResultsPerColumnFamily(int limit) {
256    this.storeLimit = limit;
257    return this;
258  }
259
260  /**
261   * Set offset for the row per Column Family. This offset is only within a particular row/CF
262   * combination. It gets reset back to zero when we move to the next row or CF.
263   * @param offset is the number of kvs that will be skipped.
264   * @return this for invocation chaining
265   */
266  public Get setRowOffsetPerColumnFamily(int offset) {
267    this.storeOffset = offset;
268    return this;
269  }
270
271  @Override
272  public Get setFilter(Filter filter) {
273    super.setFilter(filter);
274    return this;
275  }
276
277  /* Accessors */
278
279  /**
280   * Set whether blocks should be cached for this Get.
281   * <p>
282   * This is true by default. When true, default settings of the table and family are used (this
283   * will never override caching blocks if the block cache is disabled for that family or entirely).
284   * @param cacheBlocks if false, default settings are overridden and blocks will not be cached
285   */
286  public Get setCacheBlocks(boolean cacheBlocks) {
287    this.cacheBlocks = cacheBlocks;
288    return this;
289  }
290
291  /**
292   * Get whether blocks should be cached for this Get.
293   * @return true if default caching should be used, false if blocks should not be cached
294   */
295  public boolean getCacheBlocks() {
296    return cacheBlocks;
297  }
298
299  /**
300   * Method for retrieving the get's row
301   */
302  @Override
303  public byte[] getRow() {
304    return this.row;
305  }
306
307  /**
308   * Method for retrieving the get's maximum number of version
309   * @return the maximum number of version to fetch for this get
310   */
311  public int getMaxVersions() {
312    return this.maxVersions;
313  }
314
315  /**
316   * Method for retrieving the get's maximum number of values to return per Column Family
317   * @return the maximum number of values to fetch per CF
318   */
319  public int getMaxResultsPerColumnFamily() {
320    return this.storeLimit;
321  }
322
323  /**
324   * Method for retrieving the get's offset per row per column family (#kvs to be skipped)
325   * @return the row offset
326   */
327  public int getRowOffsetPerColumnFamily() {
328    return this.storeOffset;
329  }
330
331  /**
332   * Method for retrieving the get's TimeRange
333   */
334  public TimeRange getTimeRange() {
335    return this.tr;
336  }
337
338  /**
339   * Method for retrieving the keys in the familyMap
340   * @return keys in the current familyMap
341   */
342  public Set<byte[]> familySet() {
343    return this.familyMap.keySet();
344  }
345
346  /**
347   * Method for retrieving the number of families to get from
348   * @return number of families
349   */
350  public int numFamilies() {
351    return this.familyMap.size();
352  }
353
354  /**
355   * Method for checking if any families have been inserted into this Get
356   * @return true if familyMap is non empty false otherwise
357   */
358  public boolean hasFamilies() {
359    return !this.familyMap.isEmpty();
360  }
361
362  /**
363   * Method for retrieving the get's familyMap
364   */
365  public Map<byte[], NavigableSet<byte[]>> getFamilyMap() {
366    return this.familyMap;
367  }
368
369  /**
370   * Compile the table and column family (i.e. schema) information into a String. Useful for parsing
371   * and aggregation by debugging, logging, and administration tools.
372   */
373  @Override
374  public Map<String, Object> getFingerprint() {
375    Map<String, Object> map = new HashMap<>();
376    List<String> families = new ArrayList<>(this.familyMap.entrySet().size());
377    map.put("families", families);
378    for (Map.Entry<byte[], NavigableSet<byte[]>> entry : this.familyMap.entrySet()) {
379      families.add(Bytes.toStringBinary(entry.getKey()));
380    }
381    return map;
382  }
383
384  /**
385   * Compile the details beyond the scope of getFingerprint (row, columns, timestamps, etc.) into a
386   * Map along with the fingerprinted information. Useful for debugging, logging, and administration
387   * tools.
388   * @param maxCols a limit on the number of columns output prior to truncation
389   */
390  @Override
391  public Map<String, Object> toMap(int maxCols) {
392    // we start with the fingerprint map and build on top of it.
393    Map<String, Object> map = getFingerprint();
394    // replace the fingerprint's simple list of families with a
395    // map from column families to lists of qualifiers and kv details
396    Map<String, List<String>> columns = new HashMap<>();
397    map.put("families", columns);
398    // add scalar information first
399    map.put("row", Bytes.toStringBinary(this.row));
400    map.put("maxVersions", this.maxVersions);
401    map.put("cacheBlocks", this.cacheBlocks);
402    List<Long> timeRange = new ArrayList<>(2);
403    timeRange.add(this.tr.getMin());
404    timeRange.add(this.tr.getMax());
405    map.put("timeRange", timeRange);
406    int colCount = 0;
407    // iterate through affected families and add details
408    for (Map.Entry<byte[], NavigableSet<byte[]>> entry : this.familyMap.entrySet()) {
409      List<String> familyList = new ArrayList<>();
410      columns.put(Bytes.toStringBinary(entry.getKey()), familyList);
411      if (entry.getValue() == null) {
412        colCount++;
413        --maxCols;
414        familyList.add("ALL");
415      } else {
416        colCount += entry.getValue().size();
417        if (maxCols <= 0) {
418          continue;
419        }
420        for (byte[] column : entry.getValue()) {
421          if (--maxCols <= 0) {
422            continue;
423          }
424          familyList.add(Bytes.toStringBinary(column));
425        }
426      }
427    }
428    map.put("totalColumns", colCount);
429    if (this.filter != null) {
430      map.put("filter", this.filter.toString());
431    }
432    // add the id if set
433    if (getId() != null) {
434      map.put("id", getId());
435    }
436    map.put("storeLimit", this.storeLimit);
437    map.put("storeOffset", this.storeOffset);
438    map.put("checkExistenceOnly", this.checkExistenceOnly);
439
440    map.put("targetReplicaId", this.targetReplicaId);
441    map.put("consistency", this.consistency);
442    map.put("loadColumnFamiliesOnDemand", this.loadColumnFamiliesOnDemand);
443    if (!colFamTimeRangeMap.isEmpty()) {
444      Map<String, List<Long>> colFamTimeRangeMapStr = colFamTimeRangeMap.entrySet().stream()
445        .collect(Collectors.toMap((e) -> Bytes.toStringBinary(e.getKey()), e -> {
446          TimeRange value = e.getValue();
447          List<Long> rangeList = new ArrayList<>();
448          rangeList.add(value.getMin());
449          rangeList.add(value.getMax());
450          return rangeList;
451        }));
452
453      map.put("colFamTimeRangeMap", colFamTimeRangeMapStr);
454    }
455    map.put("priority", getPriority());
456    return map;
457  }
458
459  @Override
460  public int hashCode() {
461    // TODO: This is wrong. Can't have two gets the same just because on same row. But it
462    // matches how equals works currently and gets rid of the findbugs warning.
463    return Bytes.hashCode(this.getRow());
464  }
465
466  @Override
467  public boolean equals(Object obj) {
468    if (this == obj) {
469      return true;
470    }
471    if (!(obj instanceof Row)) {
472      return false;
473    }
474    Row other = (Row) obj;
475    // TODO: This is wrong. Can't have two gets the same just because on same row.
476    return Row.COMPARATOR.compare(this, other) == 0;
477  }
478
479  @Override
480  public Get setAttribute(String name, byte[] value) {
481    super.setAttribute(name, value);
482    return this;
483  }
484
485  @Override
486  public Get setId(String id) {
487    super.setId(id);
488    return this;
489  }
490
491  @Override
492  public Get setAuthorizations(Authorizations authorizations) {
493    super.setAuthorizations(authorizations);
494    return this;
495  }
496
497  @Override
498  public Get setACL(Map<String, Permission> perms) {
499    super.setACL(perms);
500    return this;
501  }
502
503  @Override
504  public Get setACL(String user, Permission perms) {
505    super.setACL(user, perms);
506    return this;
507  }
508
509  @Override
510  public Get setConsistency(Consistency consistency) {
511    super.setConsistency(consistency);
512    return this;
513  }
514
515  @Override
516  public Get setReplicaId(int Id) {
517    super.setReplicaId(Id);
518    return this;
519  }
520
521  @Override
522  public Get setIsolationLevel(IsolationLevel level) {
523    super.setIsolationLevel(level);
524    return this;
525  }
526
527  @Override
528  public Get setPriority(int priority) {
529    super.setPriority(priority);
530    return this;
531  }
532}