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.fs.Path;
024import org.apache.hadoop.hbase.util.Pair;
025import org.apache.yetus.audience.InterfaceAudience;
026
027/**
028 * Block cache interface. Anything that implements the {@link Cacheable} interface can be put in the
029 * cache.
030 */
031@InterfaceAudience.Private
032public interface BlockCache extends Iterable<CachedBlock> {
033  /**
034   * Add block to cache.
035   * @param cacheKey The block's cache key.
036   * @param buf      The block contents wrapped in a ByteBuffer.
037   * @param inMemory Whether block should be treated as in-memory
038   */
039  void cacheBlock(BlockCacheKey cacheKey, Cacheable buf, boolean inMemory);
040
041  /**
042   * Add block to cache.
043   * @param cacheKey      The block's cache key.
044   * @param buf           The block contents wrapped in a ByteBuffer.
045   * @param inMemory      Whether block should be treated as in-memory
046   * @param waitWhenCache Whether to wait for the cache to be flushed mainly when BucketCache is
047   *                      configured.
048   */
049  default void cacheBlock(BlockCacheKey cacheKey, Cacheable buf, boolean inMemory,
050    boolean waitWhenCache) {
051    cacheBlock(cacheKey, buf, inMemory);
052  }
053
054  /**
055   * Add block to cache (defaults to not in-memory).
056   * @param cacheKey The block's cache key.
057   * @param buf      The object to cache.
058   */
059  void cacheBlock(BlockCacheKey cacheKey, Cacheable buf);
060
061  /**
062   * Fetch block from cache.
063   * @param cacheKey           Block to fetch.
064   * @param caching            Whether this request has caching enabled (used for stats)
065   * @param repeat             Whether this is a repeat lookup for the same block (used to avoid
066   *                           double counting cache misses when doing double-check locking)
067   * @param updateCacheMetrics Whether to update cache metrics or not
068   * @return Block or null if block is not in 2 cache.
069   */
070  Cacheable getBlock(BlockCacheKey cacheKey, boolean caching, boolean repeat,
071    boolean updateCacheMetrics);
072
073  /**
074   * Fetch block from cache.
075   * @param cacheKey           Block to fetch.
076   * @param caching            Whether this request has caching enabled (used for stats)
077   * @param repeat             Whether this is a repeat lookup for the same block (used to avoid
078   *                           double counting cache misses when doing double-check locking)
079   * @param updateCacheMetrics Whether to update cache metrics or not
080   * @param blockType          BlockType
081   * @return Block or null if block is not in 2 cache.
082   */
083  default Cacheable getBlock(BlockCacheKey cacheKey, boolean caching, boolean repeat,
084    boolean updateCacheMetrics, BlockType blockType) {
085    return getBlock(cacheKey, caching, repeat, updateCacheMetrics);
086  }
087
088  /**
089   * Evict block from cache.
090   * @param cacheKey Block to evict
091   * @return true if block existed and was evicted, false if not
092   */
093  boolean evictBlock(BlockCacheKey cacheKey);
094
095  /**
096   * Evicts all blocks for the given HFile.
097   * @return the number of blocks evicted
098   */
099  int evictBlocksByHfileName(String hfileName);
100
101  /**
102   * Get the statistics for this block cache.
103   */
104  CacheStats getStats();
105
106  /**
107   * Shutdown the cache.
108   */
109  void shutdown();
110
111  /**
112   * Returns the total size of the block cache, in bytes.
113   * @return size of cache, in bytes
114   */
115  long size();
116
117  /**
118   * Returns the Max size of the block cache, in bytes.
119   * @return size of cache, in bytes
120   */
121  long getMaxSize();
122
123  /**
124   * Returns the free size of the block cache, in bytes.
125   * @return free space in cache, in bytes
126   */
127  long getFreeSize();
128
129  /**
130   * Returns the occupied size of the block cache, in bytes.
131   * @return occupied space in cache, in bytes
132   */
133  long getCurrentSize();
134
135  /**
136   * Returns the occupied size of data blocks, in bytes.
137   * @return occupied space in cache, in bytes
138   */
139  long getCurrentDataSize();
140
141  /**
142   * Returns the number of blocks currently cached in the block cache.
143   * @return number of blocks in the cache
144   */
145  long getBlockCount();
146
147  /**
148   * Returns the number of data blocks currently cached in the block cache.
149   * @return number of blocks in the cache
150   */
151  long getDataBlockCount();
152
153  /** Returns Iterator over the blocks in the cache. */
154  @Override
155  Iterator<CachedBlock> iterator();
156
157  /** Returns The list of sub blockcaches that make up this one; returns null if no sub caches. */
158  BlockCache[] getBlockCaches();
159
160  /**
161   * Check if block type is meta or index block
162   * @param blockType block type of a given HFile block
163   * @return true if block type is non-data block
164   */
165  default boolean isMetaBlock(BlockType blockType) {
166    return blockType != null && blockType.getCategory() != BlockType.BlockCategory.DATA;
167  }
168
169  /**
170   * Notifies the cache implementation that the given file has been fully cached (all its blocks
171   * made into the cache).
172   * @param fileName        the file that has been completely cached.
173   * @param totalBlockCount the total of blocks cached for this file.
174   * @param dataBlockCount  number of DATA block type cached.
175   * @param size            the size, in bytes, cached.
176   */
177  default void notifyFileCachingCompleted(Path fileName, int totalBlockCount, int dataBlockCount,
178    long size) {
179    // noop
180  }
181
182  /**
183   * Checks whether there's enough space left in the cache to accommodate the passed block. This
184   * method may not be overridden by all implementing classes. In such cases, the returned Optional
185   * will be empty. For subclasses implementing this logic, the returned Optional would contain the
186   * boolean value reflecting if the passed block fits into the remaining cache space available.
187   * @param block the block we want to check if fits into the cache.
188   * @return empty optional if this method is not supported, otherwise the returned optional
189   *         contains the boolean value informing if the block fits into the cache available space.
190   */
191  default Optional<Boolean> blockFitsIntoTheCache(HFileBlock block) {
192    return Optional.empty();
193  }
194
195  /**
196   * Checks whether blocks for the passed file should be cached or not. This method may not be
197   * overridden by all implementing classes. In such cases, the returned Optional will be empty. For
198   * subclasses implementing this logic, the returned Optional would contain the boolean value
199   * reflecting if the passed file should indeed be cached.
200   * @param fileName to check if it should be cached.
201   * @return empty optional if this method is not supported, otherwise the returned optional
202   *         contains the boolean value informing if the file should be cached.
203   */
204  default Optional<Boolean> shouldCacheFile(String fileName) {
205    return Optional.empty();
206  }
207
208  /**
209   * Checks whether the block for the passed key is already cached. This method may not be
210   * overridden by all implementing classes. In such cases, the returned Optional will be empty. For
211   * subclasses implementing this logic, the returned Optional would contain the boolean value
212   * reflecting if the block for the passed key is already cached or not.
213   * @param key for the block we want to check if it's already in the cache.
214   * @return empty optional if this method is not supported, otherwise the returned optional
215   *         contains the boolean value informing if the block is already cached.
216   */
217  default Optional<Boolean> isAlreadyCached(BlockCacheKey key) {
218    return Optional.empty();
219  }
220
221  /**
222   * Returns an Optional containing the size of the block related to the passed key. If the block is
223   * not in the cache, returned optional will be empty. Also, this method may not be overridden by
224   * all implementing classes. In such cases, the returned Optional will be empty.
225   * @param key for the block we want to check if it's already in the cache.
226   * @return empty optional if this method is not supported, otherwise the returned optional
227   *         contains the boolean value informing if the block is already cached.
228   */
229  default Optional<Integer> getBlockSize(BlockCacheKey key) {
230    return Optional.empty();
231  }
232
233  /**
234   * Returns an Optional containing the map of files that have been fully cached (all its blocks are
235   * present in the cache. This method may not be overridden by all implementing classes. In such
236   * cases, the returned Optional will be empty.
237   * @return empty optional if this method is not supported, otherwise the returned optional
238   *         contains a map of all files that have been fully cached.
239   */
240  default Optional<Map<String, Pair<String, Long>>> getFullyCachedFiles() {
241    return Optional.empty();
242  }
243
244  /**
245   * Returns an Optional containing a map of regions and the percentage of how much of it has been
246   * cached so far.
247   * @return empty optional if this method is not supported, otherwise the returned optional
248   *         contains a map of current regions caching percentage.
249   */
250  default Optional<Map<String, Long>> getRegionCachedInfo() {
251    return Optional.empty();
252  }
253}