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.io.hfile;
019
020import java.util.Iterator;
021import java.util.Map;
022import java.util.Optional;
023import org.apache.hadoop.conf.Configuration;
024import org.apache.hadoop.fs.Path;
025import org.apache.hadoop.hbase.conf.ConfigurationObserver;
026import org.apache.hadoop.hbase.util.Pair;
027import org.apache.yetus.audience.InterfaceAudience;
028
029/**
030 * Block cache interface. Anything that implements the {@link Cacheable} interface can be put in the
031 * cache.
032 */
033@InterfaceAudience.Private
034public interface BlockCache extends Iterable<CachedBlock>, ConfigurationObserver {
035  /**
036   * Add block to cache.
037   * @param cacheKey The block's cache key.
038   * @param buf      The block contents wrapped in a ByteBuffer.
039   * @param inMemory Whether block should be treated as in-memory
040   */
041  void cacheBlock(BlockCacheKey cacheKey, Cacheable buf, boolean inMemory);
042
043  /**
044   * Add block to cache.
045   * @param cacheKey      The block's cache key.
046   * @param buf           The block contents wrapped in a ByteBuffer.
047   * @param inMemory      Whether block should be treated as in-memory
048   * @param waitWhenCache Whether to wait for the cache to be flushed mainly when BucketCache is
049   *                      configured.
050   */
051  default void cacheBlock(BlockCacheKey cacheKey, Cacheable buf, boolean inMemory,
052    boolean waitWhenCache) {
053    cacheBlock(cacheKey, buf, inMemory);
054  }
055
056  /**
057   * Add block to cache (defaults to not in-memory).
058   * @param cacheKey The block's cache key.
059   * @param buf      The object to cache.
060   */
061  void cacheBlock(BlockCacheKey cacheKey, Cacheable buf);
062
063  /**
064   * Fetch block from cache.
065   * @param cacheKey           Block to fetch.
066   * @param caching            Whether this request has caching enabled (used for stats)
067   * @param repeat             Whether this is a repeat lookup for the same block (used to avoid
068   *                           double counting cache misses when doing double-check locking)
069   * @param updateCacheMetrics Whether to update cache metrics or not
070   * @return Block or null if block is not in 2 cache.
071   */
072  Cacheable getBlock(BlockCacheKey cacheKey, boolean caching, boolean repeat,
073    boolean updateCacheMetrics);
074
075  /**
076   * Fetch block from cache.
077   * @param cacheKey           Block to fetch.
078   * @param caching            Whether this request has caching enabled (used for stats)
079   * @param repeat             Whether this is a repeat lookup for the same block (used to avoid
080   *                           double counting cache misses when doing double-check locking)
081   * @param updateCacheMetrics Whether to update cache metrics or not
082   * @param blockType          BlockType
083   * @return Block or null if block is not in 2 cache.
084   */
085  default Cacheable getBlock(BlockCacheKey cacheKey, boolean caching, boolean repeat,
086    boolean updateCacheMetrics, BlockType blockType) {
087    return getBlock(cacheKey, caching, repeat, updateCacheMetrics);
088  }
089
090  /**
091   * Evict block from cache.
092   * @param cacheKey Block to evict
093   * @return true if block existed and was evicted, false if not
094   */
095  boolean evictBlock(BlockCacheKey cacheKey);
096
097  /**
098   * Evicts all blocks for the given HFile.
099   * @return the number of blocks evicted
100   */
101  int evictBlocksByHfileName(String hfileName);
102
103  /**
104   * Evicts all blocks for the given HFile by path.
105   * @return the number of blocks evicted
106   */
107  default int evictBlocksByHfilePath(Path hfilePath) {
108    return evictBlocksByHfileName(hfilePath.getName());
109  }
110
111  /**
112   * Get the statistics for this block cache.
113   */
114  CacheStats getStats();
115
116  /**
117   * Shutdown the cache.
118   */
119  void shutdown();
120
121  /**
122   * Returns the total size of the block cache, in bytes.
123   * @return size of cache, in bytes
124   */
125  long size();
126
127  /**
128   * Returns the Max size of the block cache, in bytes.
129   * @return size of cache, in bytes
130   */
131  long getMaxSize();
132
133  /**
134   * Returns the free size of the block cache, in bytes.
135   * @return free space in cache, in bytes
136   */
137  long getFreeSize();
138
139  /**
140   * Returns the occupied size of the block cache, in bytes.
141   * @return occupied space in cache, in bytes
142   */
143  long getCurrentSize();
144
145  /**
146   * Returns the occupied size of data blocks, in bytes.
147   * @return occupied space in cache, in bytes
148   */
149  long getCurrentDataSize();
150
151  /**
152   * Returns the number of blocks currently cached in the block cache.
153   * @return number of blocks in the cache
154   */
155  long getBlockCount();
156
157  /**
158   * Returns the number of data blocks currently cached in the block cache.
159   * @return number of blocks in the cache
160   */
161  long getDataBlockCount();
162
163  /** Returns Iterator over the blocks in the cache. */
164  @Override
165  Iterator<CachedBlock> iterator();
166
167  /** Returns The list of sub blockcaches that make up this one; returns null if no sub caches. */
168  BlockCache[] getBlockCaches();
169
170  /**
171   * Check if block type is meta or index block
172   * @param blockType block type of a given HFile block
173   * @return true if block type is non-data block
174   */
175  default boolean isMetaBlock(BlockType blockType) {
176    return blockType != null && blockType.getCategory() != BlockType.BlockCategory.DATA;
177  }
178
179  /**
180   * Notifies the cache implementation that the given file has been fully cached (all its blocks
181   * made into the cache).
182   * @param fileName        the file that has been completely cached.
183   * @param totalBlockCount the total of blocks cached for this file.
184   * @param dataBlockCount  number of DATA block type cached.
185   * @param size            the size, in bytes, cached.
186   */
187  default void notifyFileCachingCompleted(Path fileName, int totalBlockCount, int dataBlockCount,
188    long size) {
189    // noop
190  }
191
192  /**
193   * Checks whether there's enough space left in the cache to accommodate the passed block. This
194   * method may not be overridden by all implementing classes. In such cases, the returned Optional
195   * will be empty. For subclasses implementing this logic, the returned Optional would contain the
196   * boolean value reflecting if the passed block fits into the remaining cache space available.
197   * @param block the block we want to check if fits into the cache.
198   * @return empty optional if this method is not supported, otherwise the returned optional
199   *         contains the boolean value informing if the block fits into the cache available space.
200   */
201  default Optional<Boolean> blockFitsIntoTheCache(HFileBlock block) {
202    return Optional.empty();
203  }
204
205  /**
206   * Checks whether blocks for the passed file should be cached or not. This method may not be
207   * overridden by all implementing classes. In such cases, the returned Optional will be empty. For
208   * subclasses implementing this logic, the returned Optional would contain the boolean value
209   * reflecting if the passed file should indeed be cached.
210   * @param hFileInfo Information about the file to check if it should be cached.
211   * @param conf      The configuration object to use for determining caching behavior.
212   * @return empty optional if this method is not supported, otherwise the returned optional
213   *         contains the boolean value informing if the file should be cached.
214   */
215  default Optional<Boolean> shouldCacheFile(HFileInfo hFileInfo, Configuration conf) {
216    return Optional.empty();
217  }
218
219  /**
220   * Checks whether the block represented by the given key should be cached or not. This method may
221   * not be overridden by all implementing classes. In such cases, the returned Optional will be
222   * empty. For subclasses implementing this logic, the returned Optional would contain the boolean
223   * value reflecting if the passed block should indeed be cached.
224   * @param key          The key representing the block to check if it should be cached.
225   * @param maxTimeStamp The maximum timestamp for the block to check if it should be cached.
226   * @param conf         The configuration object to use for determining caching behavior.
227   * @return An empty Optional if this method is not supported; otherwise, the returned Optional
228   *         contains the boolean value indicating if the block should be cached.
229   */
230  default Optional<Boolean> shouldCacheBlock(BlockCacheKey key, long maxTimeStamp,
231    Configuration conf) {
232    return Optional.empty();
233  }
234
235  /**
236   * Checks whether the block for the passed key is already cached. This method may not be
237   * overridden by all implementing classes. In such cases, the returned Optional will be empty. For
238   * subclasses implementing this logic, the returned Optional would contain the boolean value
239   * reflecting if the block for the passed key is already cached or not.
240   * @param key for the block we want to check if it's already in the cache.
241   * @return empty optional if this method is not supported, otherwise the returned optional
242   *         contains the boolean value informing if the block is already cached.
243   */
244  default Optional<Boolean> isAlreadyCached(BlockCacheKey key) {
245    return Optional.empty();
246  }
247
248  /**
249   * Returns an Optional containing the size of the block related to the passed key. If the block is
250   * not in the cache, returned optional will be empty. Also, this method may not be overridden by
251   * all implementing classes. In such cases, the returned Optional will be empty.
252   * @param key for the block we want to check if it's already in the cache.
253   * @return empty optional if this method is not supported, otherwise the returned optional
254   *         contains the boolean value informing if the block is already cached.
255   */
256  default Optional<Integer> getBlockSize(BlockCacheKey key) {
257    return Optional.empty();
258  }
259
260  /**
261   * Returns an Optional containing the map of files that have been fully cached (all its blocks are
262   * present in the cache. This method may not be overridden by all implementing classes. In such
263   * cases, the returned Optional will be empty.
264   * @return empty optional if this method is not supported, otherwise the returned optional
265   *         contains a map of all files that have been fully cached.
266   */
267  default Optional<Map<String, Pair<String, Long>>> getFullyCachedFiles() {
268    return Optional.empty();
269  }
270
271  /**
272   * Returns an Optional containing a map of regions and the percentage of how much of it has been
273   * cached so far.
274   * @return empty optional if this method is not supported, otherwise the returned optional
275   *         contains a map of current regions caching percentage.
276   */
277  default Optional<Map<String, Long>> getRegionCachedInfo() {
278    return Optional.empty();
279  }
280
281  /**
282   * Evict all blocks for the given file name between the passed offset values.
283   * @param hfileName  The file for which blocks should be evicted.
284   * @param initOffset the initial offset for the range of blocks to be evicted.
285   * @param endOffset  the end offset for the range of blocks to be evicted.
286   * @return number of blocks evicted.
287   */
288  default int evictBlocksRangeByHfileName(String hfileName, long initOffset, long endOffset) {
289    return 0;
290  }
291
292  /**
293   * API to check whether or not, the cache is enabled.
294   * @return returns true if the cache is enabled, false otherwise.
295   */
296  default boolean isCacheEnabled() {
297    return true;
298  }
299
300  /**
301   * Wait for the block cache implementation to be completely enabled. Some block cache
302   * implementations may take longer to initialise, and this initialisation may be asynchronous.
303   * @param timeout time to wait for the cache to become enabled.
304   * @return boolean true if the cache is enabled, false otherwise.
305   */
306  default boolean waitForCacheInitialization(long timeout) {
307    return true;
308  }
309
310  /**
311   * Allows for BlockCache implementations to provide a mean to refresh their configurations. Since
312   * HBASE-29249, CacheConfig implements PropagatingConfigurationObserver and registers itself
313   * together with the used BlockCache implementation for notifications of dynamic configuration
314   * changes. The default is a noop.
315   * @param config the new configuration to be updated.
316   */
317  default void onConfigurationChange(Configuration config) {
318    // noop
319  }
320}