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.storefiletracker;
019
020import java.io.IOException;
021import java.util.Collection;
022import java.util.List;
023import org.apache.hadoop.hbase.client.TableDescriptorBuilder;
024import org.apache.hadoop.hbase.regionserver.CreateStoreFileWriterParams;
025import org.apache.hadoop.hbase.regionserver.StoreFileInfo;
026import org.apache.hadoop.hbase.regionserver.StoreFileWriter;
027import org.apache.yetus.audience.InterfaceAudience;
028
029/**
030 * An interface to define how we track the store files for a give store.
031 * <p/>
032 * In the old time, we will write store to a tmp directory first, and then rename it to the actual
033 * data file. And once a store file is under data directory, we will consider it as 'committed'. And
034 * we need to do listing when loading store files.
035 * <p/>
036 * When cloud age is coming, now we want to store the store files on object storage, where rename
037 * and list are not as cheap as on HDFS, especially rename. Although introducing a metadata
038 * management layer for object storage could solve the problem, but we still want HBase to run on
039 * pure object storage, so here we introduce this interface to abstract how we track the store
040 * files. For the old implementation, we just persist nothing here, and do listing to load store
041 * files. When running on object storage, we could persist the store file list in a system region,
042 * or in a file on the object storage, to make it possible to write directly into the data directory
043 * to avoid renaming, and also avoid listing when loading store files.
044 * <p/>
045 * The implementation requires to be thread safe as flush and compaction may occur as the same time,
046 * and we could also do multiple compactions at the same time. As the implementation may choose to
047 * persist the store file list to external storage, which could be slow, it is the duty for the
048 * callers to not call it inside a lock which may block normal read/write requests.
049 */
050@InterfaceAudience.Private
051public interface StoreFileTracker {
052  /**
053   * Load the store files list when opening a region.
054   */
055  List<StoreFileInfo> load() throws IOException;
056
057  /**
058   * Add new store files.
059   * <p/>
060   * Used for flush and bulk load.
061   */
062  void add(Collection<StoreFileInfo> newFiles) throws IOException;
063
064  /**
065   * Add new store files and remove compacted store files after compaction.
066   */
067  void replace(Collection<StoreFileInfo> compactedFiles, Collection<StoreFileInfo> newFiles)
068    throws IOException;
069
070  /**
071   * Set the store files.
072   */
073  void set(List<StoreFileInfo> files) throws IOException;
074
075  /**
076   * Create a writer for writing new store files.
077   * @return Writer for a new StoreFile
078   */
079  StoreFileWriter createWriter(CreateStoreFileWriterParams params) throws IOException;
080
081  /**
082   * Adds StoreFileTracker implementations specific configurations into the table descriptor.
083   * <p/>
084   * This is used to avoid accidentally data loss when changing the cluster level store file tracker
085   * implementation, and also possible misconfiguration between master and region servers.
086   * <p/>
087   * See HBASE-26246 for more details.
088   * @param builder The table descriptor builder for the given table.
089   */
090  TableDescriptorBuilder updateWithTrackerConfigs(TableDescriptorBuilder builder);
091
092  /**
093   * Whether the implementation of this tracker requires you to write to temp directory first, i.e,
094   * does not allow broken store files under the actual data directory.
095   */
096  boolean requireWritingToTmpDirFirst();
097}