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 /** Returns true if the cell is expired */ 141 private static boolean isCellTTLExpired(final Cell cell, final long oldestTimestamp, 142 final long now) { 143 // Look for a TTL tag first. Use it instead of the family setting if 144 // found. If a cell has multiple TTLs, resolve the conflict by using the 145 // first tag encountered. 146 Iterator<Tag> i = PrivateCellUtil.tagsIterator(cell); 147 while (i.hasNext()) { 148 Tag t = i.next(); 149 if (TagType.TTL_TAG_TYPE == t.getType()) { 150 // Unlike in schema cell TTLs are stored in milliseconds, no need 151 // to convert 152 long ts = cell.getTimestamp(); 153 assert t.getValueLength() == Bytes.SIZEOF_LONG; 154 long ttl = Tag.getValueAsLong(t); 155 if (ts + ttl < now) { 156 return true; 157 } 158 // Per cell TTLs cannot extend lifetime beyond family settings, so 159 // fall through to check that 160 break; 161 } 162 } 163 return false; 164 } 165 166 /** 167 * Check before the delete logic. 168 * @return null means continue. 169 */ 170 protected final MatchCode preCheck(Cell cell) { 171 if (currentRow == null) { 172 // Since the curCell is null it means we are already sure that we have moved over to the next 173 // row 174 return MatchCode.DONE; 175 } 176 // if row key is changed, then we know that we have moved over to the next row 177 if (rowComparator.compareRows(currentRow, cell) != 0) { 178 return MatchCode.DONE; 179 } 180 181 if (this.columns.done()) { 182 return MatchCode.SEEK_NEXT_ROW; 183 } 184 185 long timestamp = cell.getTimestamp(); 186 // check if this is a fake cell. The fake cell is an optimization, we should make the scanner 187 // seek to next column or next row. See StoreFileScanner.requestSeek for more details. 188 // check for early out based on timestamp alone 189 if (timestamp == HConstants.OLDEST_TIMESTAMP || columns.isDone(timestamp)) { 190 return columns.getNextRowOrNextColumn(cell); 191 } 192 // check if the cell is expired by cell TTL 193 if (isCellTTLExpired(cell, this.oldestUnexpiredTS, this.now)) { 194 return MatchCode.SKIP; 195 } 196 return null; 197 } 198 199 protected final MatchCode checkDeleted(DeleteTracker deletes, Cell cell) { 200 if (deletes.isEmpty() && !(deletes instanceof NewVersionBehaviorTracker)) { 201 return null; 202 } 203 // MvccSensitiveTracker always need check all cells to save some infos. 204 DeleteResult deleteResult = deletes.isDeleted(cell); 205 switch (deleteResult) { 206 case FAMILY_DELETED: 207 case COLUMN_DELETED: 208 if (!(deletes instanceof NewVersionBehaviorTracker)) { 209 // MvccSensitive can not seek to next because the Put with lower ts may have higher mvcc 210 return columns.getNextRowOrNextColumn(cell); 211 } 212 case VERSION_DELETED: 213 case FAMILY_VERSION_DELETED: 214 case VERSION_MASKED: 215 return MatchCode.SKIP; 216 case NOT_DELETED: 217 return null; 218 default: 219 throw new RuntimeException("Unexpected delete result: " + deleteResult); 220 } 221 } 222 223 /** 224 * Determines if the caller should do one of several things: 225 * <ul> 226 * <li>seek/skip to the next row (MatchCode.SEEK_NEXT_ROW)</li> 227 * <li>seek/skip to the next column (MatchCode.SEEK_NEXT_COL)</li> 228 * <li>include the current KeyValue (MatchCode.INCLUDE)</li> 229 * <li>ignore the current KeyValue (MatchCode.SKIP)</li> 230 * <li>got to the next row (MatchCode.DONE)</li> 231 * </ul> 232 * @param cell KeyValue to check 233 * @return The match code instance. 234 * @throws IOException in case there is an internal consistency problem caused by a data 235 * corruption. 236 */ 237 public abstract MatchCode match(Cell cell) throws IOException; 238 239 /** Returns the start key */ 240 public Cell getStartKey() { 241 return startKey; 242 } 243 244 /** Returns whether there is an null column in the query */ 245 public abstract boolean hasNullColumnInQuery(); 246 247 /** Returns a cell represent the current row */ 248 public Cell currentRow() { 249 return currentRow; 250 } 251 252 /** 253 * Make {@link #currentRow()} return null. 254 */ 255 public void clearCurrentRow() { 256 currentRow = null; 257 } 258 259 protected abstract void reset(); 260 261 /** 262 * Set the row when there is change in row 263 */ 264 public void setToNewRow(Cell currentRow) { 265 this.currentRow = currentRow; 266 columns.reset(); 267 reset(); 268 } 269 270 public abstract boolean isUserScan(); 271 272 /** 273 * @return Returns false if we know there are no more rows to be scanned (We've reached the 274 * <code>stopRow</code> or we are scanning on row only because this Scan is for a Get, 275 * etc. 276 */ 277 public abstract boolean moreRowsMayExistAfter(Cell cell); 278 279 public Cell getKeyForNextColumn(Cell cell) { 280 // We aren't sure whether any DeleteFamily cells exist, so we can't skip to next column. 281 // TODO: Current way disable us to seek to next column quickly. Is there any better solution? 282 // see HBASE-18471 for more details 283 // see TestFromClientSide3#testScanAfterDeletingSpecifiedRow 284 // see TestFromClientSide3#testScanAfterDeletingSpecifiedRowV2 285 if (cell.getQualifierLength() == 0) { 286 Cell nextKey = PrivateCellUtil.createNextOnRowCol(cell); 287 if (nextKey != cell) { 288 return nextKey; 289 } 290 // The cell is at the end of row/family/qualifier, so it is impossible to find any 291 // DeleteFamily cells. 292 // Let us seek to next column. 293 } 294 ColumnCount nextColumn = columns.getColumnHint(); 295 if (nextColumn == null) { 296 return PrivateCellUtil.createLastOnRowCol(cell); 297 } else { 298 return PrivateCellUtil.createFirstOnRowCol(cell, nextColumn.getBuffer(), 299 nextColumn.getOffset(), nextColumn.getLength()); 300 } 301 } 302 303 /** 304 * @param nextIndexed the key of the next entry in the block index (if any) 305 * @param currentCell The Cell we're using to calculate the seek key 306 * @return result of the compare between the indexed key and the key portion of the passed cell 307 */ 308 public int compareKeyForNextRow(Cell nextIndexed, Cell currentCell) { 309 return PrivateCellUtil.compareKeyBasedOnColHint(rowComparator, nextIndexed, currentCell, 0, 0, 310 null, 0, 0, HConstants.OLDEST_TIMESTAMP, Type.Minimum.getCode()); 311 } 312 313 /** 314 * @param nextIndexed the key of the next entry in the block index (if any) 315 * @param currentCell The Cell we're using to calculate the seek key 316 * @return result of the compare between the indexed key and the key portion of the passed cell 317 */ 318 public int compareKeyForNextColumn(Cell nextIndexed, Cell currentCell) { 319 ColumnCount nextColumn = columns.getColumnHint(); 320 if (nextColumn == null) { 321 return PrivateCellUtil.compareKeyBasedOnColHint(rowComparator, nextIndexed, currentCell, 0, 0, 322 null, 0, 0, HConstants.OLDEST_TIMESTAMP, Type.Minimum.getCode()); 323 } else { 324 return PrivateCellUtil.compareKeyBasedOnColHint(rowComparator, nextIndexed, currentCell, 325 currentCell.getFamilyOffset(), currentCell.getFamilyLength(), nextColumn.getBuffer(), 326 nextColumn.getOffset(), nextColumn.getLength(), HConstants.LATEST_TIMESTAMP, 327 Type.Maximum.getCode()); 328 } 329 } 330 331 /** Returns the Filter */ 332 public abstract Filter getFilter(); 333 334 /** 335 * Delegate to {@link Filter#getNextCellHint(Cell)}. If no filter, return {@code null}. 336 */ 337 public abstract Cell getNextKeyHint(Cell cell) throws IOException; 338 339 @Override 340 public void beforeShipped() throws IOException { 341 if (this.currentRow != null) { 342 this.currentRow = PrivateCellUtil.createFirstOnRow(CellUtil.copyRow(this.currentRow)); 343 } 344 if (columns != null) { 345 columns.beforeShipped(); 346 } 347 } 348 349 protected static Cell createStartKeyFromRow(byte[] startRow, ScanInfo scanInfo) { 350 return PrivateCellUtil.createFirstDeleteFamilyCellOnRow(startRow, scanInfo.getFamily()); 351 } 352 353 protected static Pair<DeleteTracker, ColumnTracker> getTrackers(RegionCoprocessorHost host, 354 NavigableSet<byte[]> columns, ScanInfo scanInfo, long oldestUnexpiredTS, Scan userScan) 355 throws IOException { 356 int resultMaxVersion = scanInfo.getMaxVersions(); 357 int maxVersionToCheck = resultMaxVersion; 358 if (userScan != null) { 359 if (userScan.isRaw()) { 360 resultMaxVersion = userScan.getMaxVersions(); 361 maxVersionToCheck = userScan.hasFilter() ? Integer.MAX_VALUE : resultMaxVersion; 362 } else { 363 resultMaxVersion = Math.min(userScan.getMaxVersions(), scanInfo.getMaxVersions()); 364 maxVersionToCheck = userScan.hasFilter() ? scanInfo.getMaxVersions() : resultMaxVersion; 365 } 366 } 367 368 DeleteTracker deleteTracker; 369 if (scanInfo.isNewVersionBehavior() && (userScan == null || !userScan.isRaw())) { 370 deleteTracker = new NewVersionBehaviorTracker(columns, scanInfo.getComparator(), 371 scanInfo.getMinVersions(), scanInfo.getMaxVersions(), resultMaxVersion, oldestUnexpiredTS); 372 } else { 373 deleteTracker = new ScanDeleteTracker(scanInfo.getComparator()); 374 } 375 if (host != null) { 376 deleteTracker = host.postInstantiateDeleteTracker(deleteTracker); 377 if (deleteTracker instanceof VisibilityScanDeleteTracker && scanInfo.isNewVersionBehavior()) { 378 deleteTracker = new VisibilityNewVersionBehaivorTracker(columns, scanInfo.getComparator(), 379 scanInfo.getMinVersions(), scanInfo.getMaxVersions(), resultMaxVersion, 380 oldestUnexpiredTS); 381 } 382 } 383 384 ColumnTracker columnTracker; 385 386 if (deleteTracker instanceof NewVersionBehaviorTracker) { 387 columnTracker = (NewVersionBehaviorTracker) deleteTracker; 388 } else if (columns == null || columns.size() == 0) { 389 columnTracker = new ScanWildcardColumnTracker(scanInfo.getMinVersions(), maxVersionToCheck, 390 oldestUnexpiredTS, scanInfo.getComparator()); 391 } else { 392 columnTracker = new ExplicitColumnTracker(columns, scanInfo.getMinVersions(), 393 maxVersionToCheck, oldestUnexpiredTS); 394 } 395 return new Pair<>(deleteTracker, columnTracker); 396 } 397 398 // Used only for testing purposes 399 static MatchCode checkColumn(ColumnTracker columnTracker, byte[] bytes, int offset, int length, 400 long ttl, byte type, boolean ignoreCount) throws IOException { 401 KeyValue kv = KeyValueUtil.createFirstOnRow(HConstants.EMPTY_BYTE_ARRAY, 0, 0, 402 HConstants.EMPTY_BYTE_ARRAY, 0, 0, bytes, offset, length); 403 MatchCode matchCode = columnTracker.checkColumn(kv, type); 404 if (matchCode == MatchCode.INCLUDE) { 405 return columnTracker.checkVersions(kv, ttl, type, ignoreCount); 406 } 407 return matchCode; 408 } 409}