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