001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.regionserver.querymatcher;
019
020import java.io.IOException;
021import java.util.Iterator;
022import java.util.NavigableSet;
023import org.apache.hadoop.hbase.Cell;
024import org.apache.hadoop.hbase.CellComparator;
025import org.apache.hadoop.hbase.CellUtil;
026import org.apache.hadoop.hbase.HConstants;
027import org.apache.hadoop.hbase.KeyValue;
028import org.apache.hadoop.hbase.KeyValue.Type;
029import org.apache.hadoop.hbase.KeyValueUtil;
030import org.apache.hadoop.hbase.PrivateCellUtil;
031import org.apache.hadoop.hbase.Tag;
032import org.apache.hadoop.hbase.TagType;
033import org.apache.hadoop.hbase.client.Scan;
034import org.apache.hadoop.hbase.filter.Filter;
035import org.apache.hadoop.hbase.regionserver.RegionCoprocessorHost;
036import org.apache.hadoop.hbase.regionserver.ScanInfo;
037import org.apache.hadoop.hbase.regionserver.ShipperListener;
038import org.apache.hadoop.hbase.regionserver.querymatcher.DeleteTracker.DeleteResult;
039import org.apache.hadoop.hbase.security.visibility.VisibilityNewVersionBehaivorTracker;
040import org.apache.hadoop.hbase.security.visibility.VisibilityScanDeleteTracker;
041import org.apache.hadoop.hbase.util.Bytes;
042import org.apache.hadoop.hbase.util.Pair;
043import org.apache.yetus.audience.InterfaceAudience;
044
045/**
046 * A query matcher that is specifically designed for the scan case.
047 */
048@InterfaceAudience.Private
049public abstract class ScanQueryMatcher implements ShipperListener {
050
051  /**
052   * {@link #match} return codes. These instruct the scanner moving through memstores and StoreFiles
053   * what to do with the current KeyValue.
054   * <p>
055   * Additionally, this contains "early-out" language to tell the scanner to move on to the next
056   * File (memstore or Storefile), or to return immediately.
057   */
058  public static enum MatchCode {
059    /**
060     * Include KeyValue in the returned result
061     */
062    INCLUDE,
063
064    /**
065     * Do not include KeyValue in the returned result
066     */
067    SKIP,
068
069    /**
070     * Do not include, jump to next StoreFile or memstore (in time order)
071     */
072    NEXT,
073
074    /**
075     * Do not include, return current result
076     */
077    DONE,
078
079    /**
080     * These codes are used by the ScanQueryMatcher
081     */
082
083    /**
084     * Done with the row, seek there.
085     */
086    SEEK_NEXT_ROW,
087
088    /**
089     * Done with column, seek to next.
090     */
091    SEEK_NEXT_COL,
092
093    /**
094     * Done with scan, thanks to the row filter.
095     */
096    DONE_SCAN,
097
098    /**
099     * Seek to next key which is given as hint.
100     */
101    SEEK_NEXT_USING_HINT,
102
103    /**
104     * Include KeyValue and done with column, seek to next.
105     */
106    INCLUDE_AND_SEEK_NEXT_COL,
107
108    /**
109     * Include KeyValue and done with row, seek to next.
110     */
111    INCLUDE_AND_SEEK_NEXT_ROW,
112  }
113
114  /** Row comparator for the region this query is for */
115  protected final CellComparator rowComparator;
116
117  /** Key to seek to in memstore and StoreFiles */
118  protected final Cell startKey;
119
120  /** Keeps track of columns and versions */
121  protected final ColumnTracker columns;
122
123  /** The oldest timestamp we are interested in, based on TTL */
124  protected final long oldestUnexpiredTS;
125
126  protected final long now;
127
128  /** Row the query is on */
129  protected Cell currentRow;
130
131  protected ScanQueryMatcher(Cell startKey, ScanInfo scanInfo, ColumnTracker columns,
132    long oldestUnexpiredTS, long now) {
133    this.rowComparator = scanInfo.getComparator();
134    this.startKey = startKey;
135    this.oldestUnexpiredTS = oldestUnexpiredTS;
136    this.now = now;
137    this.columns = columns;
138  }
139
140  /**
141   * nn * @return true if the cell is expired
142   */
143  private static boolean isCellTTLExpired(final Cell cell, final long oldestTimestamp,
144    final long now) {
145    // Look for a TTL tag first. Use it instead of the family setting if
146    // found. If a cell has multiple TTLs, resolve the conflict by using the
147    // first tag encountered.
148    Iterator<Tag> i = PrivateCellUtil.tagsIterator(cell);
149    while (i.hasNext()) {
150      Tag t = i.next();
151      if (TagType.TTL_TAG_TYPE == t.getType()) {
152        // Unlike in schema cell TTLs are stored in milliseconds, no need
153        // to convert
154        long ts = cell.getTimestamp();
155        assert t.getValueLength() == Bytes.SIZEOF_LONG;
156        long ttl = Tag.getValueAsLong(t);
157        if (ts + ttl < now) {
158          return true;
159        }
160        // Per cell TTLs cannot extend lifetime beyond family settings, so
161        // fall through to check that
162        break;
163      }
164    }
165    return false;
166  }
167
168  /**
169   * Check before the delete logic.
170   * @return null means continue.
171   */
172  protected final MatchCode preCheck(Cell cell) {
173    if (currentRow == null) {
174      // Since the curCell is null it means we are already sure that we have moved over to the next
175      // row
176      return MatchCode.DONE;
177    }
178    // if row key is changed, then we know that we have moved over to the next row
179    if (rowComparator.compareRows(currentRow, cell) != 0) {
180      return MatchCode.DONE;
181    }
182
183    if (this.columns.done()) {
184      return MatchCode.SEEK_NEXT_ROW;
185    }
186
187    long timestamp = cell.getTimestamp();
188    // check if this is a fake cell. The fake cell is an optimization, we should make the scanner
189    // seek to next column or next row. See StoreFileScanner.requestSeek for more details.
190    // check for early out based on timestamp alone
191    if (timestamp == HConstants.OLDEST_TIMESTAMP || columns.isDone(timestamp)) {
192      return columns.getNextRowOrNextColumn(cell);
193    }
194    // check if the cell is expired by cell TTL
195    if (isCellTTLExpired(cell, this.oldestUnexpiredTS, this.now)) {
196      return MatchCode.SKIP;
197    }
198    return null;
199  }
200
201  protected final MatchCode checkDeleted(DeleteTracker deletes, Cell cell) {
202    if (deletes.isEmpty() && !(deletes instanceof NewVersionBehaviorTracker)) {
203      return null;
204    }
205    // MvccSensitiveTracker always need check all cells to save some infos.
206    DeleteResult deleteResult = deletes.isDeleted(cell);
207    switch (deleteResult) {
208      case FAMILY_DELETED:
209      case COLUMN_DELETED:
210        if (!(deletes instanceof NewVersionBehaviorTracker)) {
211          // MvccSensitive can not seek to next because the Put with lower ts may have higher mvcc
212          return columns.getNextRowOrNextColumn(cell);
213        }
214      case VERSION_DELETED:
215      case FAMILY_VERSION_DELETED:
216      case VERSION_MASKED:
217        return MatchCode.SKIP;
218      case NOT_DELETED:
219        return null;
220      default:
221        throw new RuntimeException("Unexpected delete result: " + deleteResult);
222    }
223  }
224
225  /**
226   * Determines if the caller should do one of several things:
227   * <ul>
228   * <li>seek/skip to the next row (MatchCode.SEEK_NEXT_ROW)</li>
229   * <li>seek/skip to the next column (MatchCode.SEEK_NEXT_COL)</li>
230   * <li>include the current KeyValue (MatchCode.INCLUDE)</li>
231   * <li>ignore the current KeyValue (MatchCode.SKIP)</li>
232   * <li>got to the next row (MatchCode.DONE)</li>
233   * </ul>
234   * @param cell KeyValue to check
235   * @return The match code instance.
236   * @throws IOException in case there is an internal consistency problem caused by a data
237   *                     corruption.
238   */
239  public abstract MatchCode match(Cell cell) throws IOException;
240
241  /** Returns the start key */
242  public Cell getStartKey() {
243    return startKey;
244  }
245
246  /** Returns whether there is an null column in the query */
247  public abstract boolean hasNullColumnInQuery();
248
249  /** Returns a cell represent the current row */
250  public Cell currentRow() {
251    return currentRow;
252  }
253
254  /**
255   * Make {@link #currentRow()} return null.
256   */
257  public void clearCurrentRow() {
258    currentRow = null;
259  }
260
261  protected abstract void reset();
262
263  /**
264   * Set the row when there is change in row n
265   */
266  public void setToNewRow(Cell currentRow) {
267    this.currentRow = currentRow;
268    columns.reset();
269    reset();
270  }
271
272  public abstract boolean isUserScan();
273
274  /**
275   * @return Returns false if we know there are no more rows to be scanned (We've reached the
276   *         <code>stopRow</code> or we are scanning on row only because this Scan is for a Get,
277   *         etc.
278   */
279  public abstract boolean moreRowsMayExistAfter(Cell cell);
280
281  public Cell getKeyForNextColumn(Cell cell) {
282    // We aren't sure whether any DeleteFamily cells exist, so we can't skip to next column.
283    // TODO: Current way disable us to seek to next column quickly. Is there any better solution?
284    // see HBASE-18471 for more details
285    // see TestFromClientSide3#testScanAfterDeletingSpecifiedRow
286    // see TestFromClientSide3#testScanAfterDeletingSpecifiedRowV2
287    if (cell.getQualifierLength() == 0) {
288      Cell nextKey = PrivateCellUtil.createNextOnRowCol(cell);
289      if (nextKey != cell) {
290        return nextKey;
291      }
292      // The cell is at the end of row/family/qualifier, so it is impossible to find any
293      // DeleteFamily cells.
294      // Let us seek to next column.
295    }
296    ColumnCount nextColumn = columns.getColumnHint();
297    if (nextColumn == null) {
298      return PrivateCellUtil.createLastOnRowCol(cell);
299    } else {
300      return PrivateCellUtil.createFirstOnRowCol(cell, nextColumn.getBuffer(),
301        nextColumn.getOffset(), nextColumn.getLength());
302    }
303  }
304
305  /**
306   * @param nextIndexed the key of the next entry in the block index (if any)
307   * @param currentCell The Cell we're using to calculate the seek key
308   * @return result of the compare between the indexed key and the key portion of the passed cell
309   */
310  public int compareKeyForNextRow(Cell nextIndexed, Cell currentCell) {
311    return PrivateCellUtil.compareKeyBasedOnColHint(rowComparator, nextIndexed, currentCell, 0, 0,
312      null, 0, 0, HConstants.OLDEST_TIMESTAMP, Type.Minimum.getCode());
313  }
314
315  /**
316   * @param nextIndexed the key of the next entry in the block index (if any)
317   * @param currentCell The Cell we're using to calculate the seek key
318   * @return result of the compare between the indexed key and the key portion of the passed cell
319   */
320  public int compareKeyForNextColumn(Cell nextIndexed, Cell currentCell) {
321    ColumnCount nextColumn = columns.getColumnHint();
322    if (nextColumn == null) {
323      return PrivateCellUtil.compareKeyBasedOnColHint(rowComparator, nextIndexed, currentCell, 0, 0,
324        null, 0, 0, HConstants.OLDEST_TIMESTAMP, Type.Minimum.getCode());
325    } else {
326      return PrivateCellUtil.compareKeyBasedOnColHint(rowComparator, nextIndexed, currentCell,
327        currentCell.getFamilyOffset(), currentCell.getFamilyLength(), nextColumn.getBuffer(),
328        nextColumn.getOffset(), nextColumn.getLength(), HConstants.LATEST_TIMESTAMP,
329        Type.Maximum.getCode());
330    }
331  }
332
333  /** Returns the Filter */
334  public abstract Filter getFilter();
335
336  /**
337   * Delegate to {@link Filter#getNextCellHint(Cell)}. If no filter, return {@code null}.
338   */
339  public abstract Cell getNextKeyHint(Cell cell) throws IOException;
340
341  @Override
342  public void beforeShipped() throws IOException {
343    if (this.currentRow != null) {
344      this.currentRow = PrivateCellUtil.createFirstOnRow(CellUtil.copyRow(this.currentRow));
345    }
346    if (columns != null) {
347      columns.beforeShipped();
348    }
349  }
350
351  protected static Cell createStartKeyFromRow(byte[] startRow, ScanInfo scanInfo) {
352    return PrivateCellUtil.createFirstDeleteFamilyCellOnRow(startRow, scanInfo.getFamily());
353  }
354
355  protected static Pair<DeleteTracker, ColumnTracker> getTrackers(RegionCoprocessorHost host,
356    NavigableSet<byte[]> columns, ScanInfo scanInfo, long oldestUnexpiredTS, Scan userScan)
357    throws IOException {
358    int resultMaxVersion = scanInfo.getMaxVersions();
359    int maxVersionToCheck = resultMaxVersion;
360    if (userScan != null) {
361      if (userScan.isRaw()) {
362        resultMaxVersion = userScan.getMaxVersions();
363        maxVersionToCheck = userScan.hasFilter() ? Integer.MAX_VALUE : resultMaxVersion;
364      } else {
365        resultMaxVersion = Math.min(userScan.getMaxVersions(), scanInfo.getMaxVersions());
366        maxVersionToCheck = userScan.hasFilter() ? scanInfo.getMaxVersions() : resultMaxVersion;
367      }
368    }
369
370    DeleteTracker deleteTracker;
371    if (scanInfo.isNewVersionBehavior() && (userScan == null || !userScan.isRaw())) {
372      deleteTracker = new NewVersionBehaviorTracker(columns, scanInfo.getComparator(),
373        scanInfo.getMinVersions(), scanInfo.getMaxVersions(), resultMaxVersion, oldestUnexpiredTS);
374    } else {
375      deleteTracker = new ScanDeleteTracker(scanInfo.getComparator());
376    }
377    if (host != null) {
378      deleteTracker = host.postInstantiateDeleteTracker(deleteTracker);
379      if (deleteTracker instanceof VisibilityScanDeleteTracker && scanInfo.isNewVersionBehavior()) {
380        deleteTracker = new VisibilityNewVersionBehaivorTracker(columns, scanInfo.getComparator(),
381          scanInfo.getMinVersions(), scanInfo.getMaxVersions(), resultMaxVersion,
382          oldestUnexpiredTS);
383      }
384    }
385
386    ColumnTracker columnTracker;
387
388    if (deleteTracker instanceof NewVersionBehaviorTracker) {
389      columnTracker = (NewVersionBehaviorTracker) deleteTracker;
390    } else if (columns == null || columns.size() == 0) {
391      columnTracker = new ScanWildcardColumnTracker(scanInfo.getMinVersions(), maxVersionToCheck,
392        oldestUnexpiredTS, scanInfo.getComparator());
393    } else {
394      columnTracker = new ExplicitColumnTracker(columns, scanInfo.getMinVersions(),
395        maxVersionToCheck, oldestUnexpiredTS);
396    }
397    return new Pair<>(deleteTracker, columnTracker);
398  }
399
400  // Used only for testing purposes
401  static MatchCode checkColumn(ColumnTracker columnTracker, byte[] bytes, int offset, int length,
402    long ttl, byte type, boolean ignoreCount) throws IOException {
403    KeyValue kv = KeyValueUtil.createFirstOnRow(HConstants.EMPTY_BYTE_ARRAY, 0, 0,
404      HConstants.EMPTY_BYTE_ARRAY, 0, 0, bytes, offset, length);
405    MatchCode matchCode = columnTracker.checkColumn(kv, type);
406    if (matchCode == MatchCode.INCLUDE) {
407      return columnTracker.checkVersions(kv, ttl, type, ignoreCount);
408    }
409    return matchCode;
410  }
411}