HBASE-21843 RegionGroupingProvider breaks the meta wal file name pattern which may...
[hbase.git] / hbase-server / src / main / java / org / apache / hadoop / hbase / wal / WAL.java
blobcf367cde159df2b542f138c3a0a912eb5ad368c1
1 /**
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org.apache.hadoop.hbase.wal;
20 import java.io.Closeable;
21 import java.io.IOException;
22 import java.util.Map;
23 import java.util.Set;
24 import org.apache.hadoop.hbase.HConstants;
25 import org.apache.hadoop.hbase.client.RegionInfo;
26 import org.apache.hadoop.hbase.regionserver.wal.FailedLogCloseException;
27 import org.apache.hadoop.hbase.regionserver.wal.WALActionsListener;
28 import org.apache.hadoop.hbase.regionserver.wal.WALCoprocessorHost;
29 import org.apache.hadoop.hbase.replication.regionserver.WALFileLengthProvider;
30 import org.apache.yetus.audience.InterfaceAudience;
31 import org.apache.yetus.audience.InterfaceStability;
33 import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
35 /**
36 * A Write Ahead Log (WAL) provides service for reading, writing waledits. This interface provides
37 * APIs for WAL users (such as RegionServer) to use the WAL (do append, sync, etc).
39 * Note that some internals, such as log rolling and performance evaluation tools, will use
40 * WAL.equals to determine if they have already seen a given WAL.
42 @InterfaceAudience.Private
43 @InterfaceStability.Evolving
44 public interface WAL extends Closeable, WALFileLengthProvider {
46 /**
47 * Registers WALActionsListener
49 void registerWALActionsListener(final WALActionsListener listener);
51 /**
52 * Unregisters WALActionsListener
54 boolean unregisterWALActionsListener(final WALActionsListener listener);
56 /**
57 * Roll the log writer. That is, start writing log messages to a new file.
59 * <p>
60 * The implementation is synchronized in order to make sure there's one rollWriter
61 * running at any given time.
63 * @return If lots of logs, flush the returned regions so next time through we
64 * can clean logs. Returns null if nothing to flush. Names are actual
65 * region names as returned by {@link RegionInfo#getEncodedName()}
67 byte[][] rollWriter() throws FailedLogCloseException, IOException;
69 /**
70 * Roll the log writer. That is, start writing log messages to a new file.
72 * <p>
73 * The implementation is synchronized in order to make sure there's one rollWriter
74 * running at any given time.
76 * @param force
77 * If true, force creation of a new writer even if no entries have
78 * been written to the current writer
79 * @return If lots of logs, flush the returned regions so next time through we
80 * can clean logs. Returns null if nothing to flush. Names are actual
81 * region names as returned by {@link RegionInfo#getEncodedName()}
83 byte[][] rollWriter(boolean force) throws FailedLogCloseException, IOException;
85 /**
86 * Stop accepting new writes. If we have unsynced writes still in buffer, sync them.
87 * Extant edits are left in place in backing storage to be replayed later.
89 void shutdown() throws IOException;
91 /**
92 * Caller no longer needs any edits from this WAL. Implementers are free to reclaim
93 * underlying resources after this call; i.e. filesystem based WALs can archive or
94 * delete files.
96 @Override
97 void close() throws IOException;
99 /**
100 * Append a set of edits to the WAL. The WAL is not flushed/sync'd after this transaction
101 * completes BUT on return this edit must have its region edit/sequence id assigned
102 * else it messes up our unification of mvcc and sequenceid. On return <code>key</code> will
103 * have the region edit/sequence id filled in.
104 * @param info the regioninfo associated with append
105 * @param key Modified by this call; we add to it this edits region edit/sequence id.
106 * @param edits Edits to append. MAY CONTAIN NO EDITS for case where we want to get an edit
107 * sequence id that is after all currently appended edits.
108 * @param inMemstore Always true except for case where we are writing a compaction completion
109 * record into the WAL; in this case the entry is just so we can finish an unfinished compaction
110 * -- it is not an edit for memstore.
111 * @return Returns a 'transaction id' and <code>key</code> will have the region edit/sequence id
112 * in it.
114 long append(RegionInfo info, WALKeyImpl key, WALEdit edits, boolean inMemstore) throws IOException;
117 * updates the seuence number of a specific store.
118 * depending on the flag: replaces current seq number if the given seq id is bigger,
119 * or even if it is lower than existing one
120 * @param encodedRegionName
121 * @param familyName
122 * @param sequenceid
123 * @param onlyIfGreater
125 void updateStore(byte[] encodedRegionName, byte[] familyName, Long sequenceid,
126 boolean onlyIfGreater);
129 * Sync what we have in the WAL.
130 * @throws IOException
132 void sync() throws IOException;
135 * Sync the WAL if the txId was not already sync'd.
136 * @param txid Transaction id to sync to.
137 * @throws IOException
139 void sync(long txid) throws IOException;
142 * @param forceSync Flag to force sync rather than flushing to the buffer. Example - Hadoop hflush
143 * vs hsync.
145 default void sync(boolean forceSync) throws IOException {
146 sync();
150 * @param txid Transaction id to sync to.
151 * @param forceSync Flag to force sync rather than flushing to the buffer. Example - Hadoop hflush
152 * vs hsync.
154 default void sync(long txid, boolean forceSync) throws IOException {
155 sync(txid);
159 * WAL keeps track of the sequence numbers that are as yet not flushed im memstores
160 * in order to be able to do accounting to figure which WALs can be let go. This method tells WAL
161 * that some region is about to flush. The flush can be the whole region or for a column family
162 * of the region only.
164 * <p>Currently, it is expected that the update lock is held for the region; i.e. no
165 * concurrent appends while we set up cache flush.
166 * @param families Families to flush. May be a subset of all families in the region.
167 * @return Returns {@link HConstants#NO_SEQNUM} if we are flushing the whole region OR if
168 * we are flushing a subset of all families but there are no edits in those families not
169 * being flushed; in other words, this is effectively same as a flush of all of the region
170 * though we were passed a subset of regions. Otherwise, it returns the sequence id of the
171 * oldest/lowest outstanding edit.
172 * @see #completeCacheFlush(byte[])
173 * @see #abortCacheFlush(byte[])
175 Long startCacheFlush(final byte[] encodedRegionName, Set<byte[]> families);
177 Long startCacheFlush(final byte[] encodedRegionName, Map<byte[], Long> familyToSeq);
180 * Complete the cache flush.
181 * @param encodedRegionName Encoded region name.
182 * @see #startCacheFlush(byte[], Set)
183 * @see #abortCacheFlush(byte[])
185 void completeCacheFlush(final byte[] encodedRegionName);
188 * Abort a cache flush. Call if the flush fails. Note that the only recovery
189 * for an aborted flush currently is a restart of the regionserver so the
190 * snapshot content dropped by the failure gets restored to the memstore.
191 * @param encodedRegionName Encoded region name.
193 void abortCacheFlush(byte[] encodedRegionName);
196 * @return Coprocessor host.
198 WALCoprocessorHost getCoprocessorHost();
201 * Gets the earliest unflushed sequence id in the memstore for the region.
202 * @param encodedRegionName The region to get the number for.
203 * @return The earliest/lowest/oldest sequence id if present, HConstants.NO_SEQNUM if absent.
204 * @deprecated Since version 1.2.0. Removing because not used and exposes subtle internal
205 * workings. Use {@link #getEarliestMemStoreSeqNum(byte[], byte[])}
207 @VisibleForTesting
208 @Deprecated
209 long getEarliestMemStoreSeqNum(byte[] encodedRegionName);
212 * Gets the earliest unflushed sequence id in the memstore for the store.
213 * @param encodedRegionName The region to get the number for.
214 * @param familyName The family to get the number for.
215 * @return The earliest/lowest/oldest sequence id if present, HConstants.NO_SEQNUM if absent.
217 long getEarliestMemStoreSeqNum(byte[] encodedRegionName, byte[] familyName);
220 * Human readable identifying information about the state of this WAL.
221 * Implementors are encouraged to include information appropriate for debugging.
222 * Consumers are advised not to rely on the details of the returned String; it does
223 * not have a defined structure.
225 @Override
226 String toString();
229 * When outside clients need to consume persisted WALs, they rely on a provided
230 * Reader.
232 interface Reader extends Closeable {
233 Entry next() throws IOException;
234 Entry next(Entry reuse) throws IOException;
235 void seek(long pos) throws IOException;
236 long getPosition() throws IOException;
237 void reset() throws IOException;
241 * Utility class that lets us keep track of the edit with it's key.
243 class Entry {
244 private final WALEdit edit;
245 private final WALKeyImpl key;
247 public Entry() {
248 this(new WALKeyImpl(), new WALEdit());
252 * Constructor for both params
254 * @param edit log's edit
255 * @param key log's key
257 public Entry(WALKeyImpl key, WALEdit edit) {
258 this.key = key;
259 this.edit = edit;
263 * Gets the edit
265 * @return edit
267 public WALEdit getEdit() {
268 return edit;
272 * Gets the key
274 * @return key
276 public WALKeyImpl getKey() {
277 return key;
280 @Override
281 public String toString() {
282 return this.key + "=" + this.edit;