HBASE-26921 Rewrite the counting cells part in TestMultiVersions (#4316)
[hbase.git] / hbase-server / src / main / java / org / apache / hadoop / hbase / wal / WAL.java
blob2a434a73b672f9e02b7a5cb561e672dbc91e920e
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.List;
23 import java.util.Map;
24 import java.util.Set;
25 import org.apache.hadoop.hbase.HConstants;
26 import org.apache.hadoop.hbase.client.RegionInfo;
27 import org.apache.hadoop.hbase.regionserver.wal.FailedLogCloseException;
28 import org.apache.hadoop.hbase.regionserver.wal.WALActionsListener;
29 import org.apache.hadoop.hbase.regionserver.wal.WALCoprocessorHost;
30 import org.apache.hadoop.hbase.replication.regionserver.WALFileLengthProvider;
31 import org.apache.yetus.audience.InterfaceAudience;
32 import org.apache.yetus.audience.InterfaceStability;
34 /**
35 * A Write Ahead Log (WAL) provides service for reading, writing waledits. This interface provides
36 * APIs for WAL users (such as RegionServer) to use the WAL (do append, sync, etc).
38 * Note that some internals, such as log rolling and performance evaluation tools, will use
39 * WAL.equals to determine if they have already seen a given WAL.
41 @InterfaceAudience.Private
42 @InterfaceStability.Evolving
43 public interface WAL extends Closeable, WALFileLengthProvider {
45 /**
46 * Registers WALActionsListener
48 void registerWALActionsListener(final WALActionsListener listener);
50 /**
51 * Unregisters WALActionsListener
53 boolean unregisterWALActionsListener(final WALActionsListener listener);
55 /**
56 * Roll the log writer. That is, start writing log messages to a new file.
58 * <p/>
59 * The implementation is synchronized in order to make sure there's one rollWriter
60 * running at any given time.
62 * @return If lots of logs, flush the stores of returned regions so next time through we
63 * can clean logs. Returns null if nothing to flush. Names are actual
64 * region names as returned by {@link RegionInfo#getEncodedName()}
66 Map<byte[], List<byte[]>> rollWriter() throws FailedLogCloseException, IOException;
68 /**
69 * Roll the log writer. That is, start writing log messages to a new file.
71 * <p/>
72 * The implementation is synchronized in order to make sure there's one rollWriter
73 * running at any given time.
75 * @param force
76 * If true, force creation of a new writer even if no entries have
77 * been written to the current writer
78 * @return If lots of logs, flush the stores of returned regions so next time through we
79 * can clean logs. Returns null if nothing to flush. Names are actual
80 * region names as returned by {@link RegionInfo#getEncodedName()}
82 Map<byte[], List<byte[]>> rollWriter(boolean force) throws IOException;
84 /**
85 * Stop accepting new writes. If we have unsynced writes still in buffer, sync them.
86 * Extant edits are left in place in backing storage to be replayed later.
88 void shutdown() throws IOException;
90 /**
91 * Caller no longer needs any edits from this WAL. Implementers are free to reclaim
92 * underlying resources after this call; i.e. filesystem based WALs can archive or
93 * delete files.
95 @Override
96 void close() throws IOException;
98 /**
99 * Append a set of data edits to the WAL. 'Data' here means that the content in the edits will
100 * also have transitioned through the memstore.
101 * <p/>
102 * The WAL is not flushed/sync'd after this transaction completes BUT on return this edit must
103 * have its region edit/sequence id assigned else it messes up our unification of mvcc and
104 * sequenceid. On return <code>key</code> will have the region edit/sequence id filled in.
105 * @param info the regioninfo associated with append
106 * @param key Modified by this call; we add to it this edits region edit/sequence id.
107 * @param edits Edits to append. MAY CONTAIN NO EDITS for case where we want to get an edit
108 * sequence id that is after all currently appended edits.
109 * @return Returns a 'transaction id' and <code>key</code> will have the region edit/sequence id
110 * in it.
111 * @see #appendMarker(RegionInfo, WALKeyImpl, WALEdit)
113 long appendData(RegionInfo info, WALKeyImpl key, WALEdit edits) throws IOException;
116 * Append an operational 'meta' event marker edit to the WAL. A marker meta edit could
117 * be a FlushDescriptor, a compaction marker, or a region event marker; e.g. region open
118 * or region close. The difference between a 'marker' append and a 'data' append as in
119 * {@link #appendData(RegionInfo, WALKeyImpl, WALEdit)}is that a marker will not have
120 * transitioned through the memstore.
121 * <p/>
122 * The WAL is not flushed/sync'd after this transaction completes BUT on return this edit must
123 * have its region edit/sequence id assigned else it messes up our unification of mvcc and
124 * sequenceid. On return <code>key</code> will have the region edit/sequence id filled in.
125 * @param info the regioninfo associated with append
126 * @param key Modified by this call; we add to it this edits region edit/sequence id.
127 * @param edits Edits to append. MAY CONTAIN NO EDITS for case where we want to get an edit
128 * sequence id that is after all currently appended edits.
129 * @return Returns a 'transaction id' and <code>key</code> will have the region edit/sequence id
130 * in it.
131 * @see #appendData(RegionInfo, WALKeyImpl, WALEdit)
133 long appendMarker(RegionInfo info, WALKeyImpl key, WALEdit edits) throws IOException;
136 * updates the seuence number of a specific store.
137 * depending on the flag: replaces current seq number if the given seq id is bigger,
138 * or even if it is lower than existing one
140 void updateStore(byte[] encodedRegionName, byte[] familyName, Long sequenceid,
141 boolean onlyIfGreater);
144 * Sync what we have in the WAL.
146 void sync() throws IOException;
149 * Sync the WAL if the txId was not already sync'd.
150 * @param txid Transaction id to sync to.
152 void sync(long txid) throws IOException;
155 * @param forceSync Flag to force sync rather than flushing to the buffer. Example - Hadoop hflush
156 * vs hsync.
158 default void sync(boolean forceSync) throws IOException {
159 sync();
163 * @param txid Transaction id to sync to.
164 * @param forceSync Flag to force sync rather than flushing to the buffer. Example - Hadoop hflush
165 * vs hsync.
167 default void sync(long txid, boolean forceSync) throws IOException {
168 sync(txid);
172 * WAL keeps track of the sequence numbers that are as yet not flushed im memstores
173 * in order to be able to do accounting to figure which WALs can be let go. This method tells WAL
174 * that some region is about to flush. The flush can be the whole region or for a column family
175 * of the region only.
177 * <p>Currently, it is expected that the update lock is held for the region; i.e. no
178 * concurrent appends while we set up cache flush.
179 * @param families Families to flush. May be a subset of all families in the region.
180 * @return Returns {@link HConstants#NO_SEQNUM} if we are flushing the whole region OR if
181 * we are flushing a subset of all families but there are no edits in those families not
182 * being flushed; in other words, this is effectively same as a flush of all of the region
183 * though we were passed a subset of regions. Otherwise, it returns the sequence id of the
184 * oldest/lowest outstanding edit.
185 * @see #completeCacheFlush(byte[], long)
186 * @see #abortCacheFlush(byte[])
188 Long startCacheFlush(final byte[] encodedRegionName, Set<byte[]> families);
190 Long startCacheFlush(final byte[] encodedRegionName, Map<byte[], Long> familyToSeq);
193 * Complete the cache flush.
194 * @param encodedRegionName Encoded region name.
195 * @param maxFlushedSeqId The maxFlushedSeqId for this flush. There is no edit in memory that is
196 * less that this sequence id.
197 * @see #startCacheFlush(byte[], Set)
198 * @see #abortCacheFlush(byte[])
200 void completeCacheFlush(final byte[] encodedRegionName, long maxFlushedSeqId);
203 * Abort a cache flush. Call if the flush fails. Note that the only recovery
204 * for an aborted flush currently is a restart of the regionserver so the
205 * snapshot content dropped by the failure gets restored to the memstore.
206 * @param encodedRegionName Encoded region name.
208 void abortCacheFlush(byte[] encodedRegionName);
211 * @return Coprocessor host.
213 WALCoprocessorHost getCoprocessorHost();
216 * Gets the earliest unflushed sequence id in the memstore for the region.
217 * @param encodedRegionName The region to get the number for.
218 * @return The earliest/lowest/oldest sequence id if present, HConstants.NO_SEQNUM if absent.
219 * @deprecated Since version 1.2.0. Removing because not used and exposes subtle internal
220 * workings. Use {@link #getEarliestMemStoreSeqNum(byte[], byte[])}
222 @Deprecated
223 long getEarliestMemStoreSeqNum(byte[] encodedRegionName);
226 * Gets the earliest unflushed sequence id in the memstore for the store.
227 * @param encodedRegionName The region to get the number for.
228 * @param familyName The family to get the number for.
229 * @return The earliest/lowest/oldest sequence id if present, HConstants.NO_SEQNUM if absent.
231 long getEarliestMemStoreSeqNum(byte[] encodedRegionName, byte[] familyName);
234 * Human readable identifying information about the state of this WAL.
235 * Implementors are encouraged to include information appropriate for debugging.
236 * Consumers are advised not to rely on the details of the returned String; it does
237 * not have a defined structure.
239 @Override
240 String toString();
243 * When outside clients need to consume persisted WALs, they rely on a provided
244 * Reader.
246 interface Reader extends Closeable {
247 Entry next() throws IOException;
248 Entry next(Entry reuse) throws IOException;
249 void seek(long pos) throws IOException;
250 long getPosition() throws IOException;
251 void reset() throws IOException;
255 * Utility class that lets us keep track of the edit with it's key.
257 class Entry {
258 private final WALEdit edit;
259 private final WALKeyImpl key;
261 public Entry() {
262 this(new WALKeyImpl(), new WALEdit());
266 * Constructor for both params
268 * @param edit log's edit
269 * @param key log's key
271 public Entry(WALKeyImpl key, WALEdit edit) {
272 this.key = key;
273 this.edit = edit;
277 * Gets the edit
279 * @return edit
281 public WALEdit getEdit() {
282 return edit;
286 * Gets the key
288 * @return key
290 public WALKeyImpl getKey() {
291 return key;
294 @Override
295 public String toString() {
296 return this.key + "=" + this.edit;