View Javadoc

1   /**
2    *
3    * Licensed to the Apache Software Foundation (ASF) under one
4    * or more contributor license agreements.  See the NOTICE file
5    * distributed with this work for additional information
6    * regarding copyright ownership.  The ASF licenses this file
7    * to you under the Apache License, Version 2.0 (the
8    * "License"); you may not use this file except in compliance
9    * with the License.  You may obtain a copy of the License at
10   *
11   *     http://www.apache.org/licenses/LICENSE-2.0
12   *
13   * Unless required by applicable law or agreed to in writing, software
14   * distributed under the License is distributed on an "AS IS" BASIS,
15   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
16   * See the License for the specific language governing permissions and
17   * limitations under the License.
18   */
19  
20  package org.apache.hadoop.hbase.wal;
21  
22  import java.io.Closeable;
23  import java.io.IOException;
24  import java.util.Set;
25  
26  import org.apache.hadoop.hbase.HConstants;
27  import org.apache.hadoop.hbase.HRegionInfo;
28  import org.apache.hadoop.hbase.HTableDescriptor;
29  import org.apache.hadoop.hbase.classification.InterfaceAudience;
30  import org.apache.hadoop.hbase.classification.InterfaceStability;
31  // imports we use from yet-to-be-moved regionsever.wal
32  import org.apache.hadoop.hbase.regionserver.wal.CompressionContext;
33  import org.apache.hadoop.hbase.regionserver.wal.FailedLogCloseException;
34  import org.apache.hadoop.hbase.regionserver.wal.HLogKey;
35  import org.apache.hadoop.hbase.regionserver.wal.WALActionsListener;
36  import org.apache.hadoop.hbase.regionserver.wal.WALCoprocessorHost;
37  import org.apache.hadoop.hbase.regionserver.wal.WALEdit;
38  
39  import com.google.common.annotations.VisibleForTesting;
40  
41  /**
42   * A Write Ahead Log (WAL) provides service for reading, writing waledits. This interface provides
43   * APIs for WAL users (such as RegionServer) to use the WAL (do append, sync, etc).
44   *
45   * Note that some internals, such as log rolling and performance evaluation tools, will use
46   * WAL.equals to determine if they have already seen a given WAL.
47   */
48  @InterfaceAudience.Private
49  @InterfaceStability.Evolving
50  public interface WAL extends Closeable {
51  
52    /**
53     * Registers WALActionsListener
54     */
55    void registerWALActionsListener(final WALActionsListener listener);
56  
57    /**
58     * Unregisters WALActionsListener
59     */
60    boolean unregisterWALActionsListener(final WALActionsListener listener);
61  
62    /**
63     * Roll the log writer. That is, start writing log messages to a new file.
64     *
65     * <p>
66     * The implementation is synchronized in order to make sure there's one rollWriter
67     * running at any given time.
68     *
69     * @return If lots of logs, flush the returned regions so next time through we
70     *         can clean logs. Returns null if nothing to flush. Names are actual
71     *         region names as returned by {@link HRegionInfo#getEncodedName()}
72     */
73    byte[][] rollWriter() throws FailedLogCloseException, IOException;
74  
75    /**
76     * Roll the log writer. That is, start writing log messages to a new file.
77     *
78     * <p>
79     * The implementation is synchronized in order to make sure there's one rollWriter
80     * running at any given time.
81     *
82     * @param force
83     *          If true, force creation of a new writer even if no entries have
84     *          been written to the current writer
85     * @return If lots of logs, flush the returned regions so next time through we
86     *         can clean logs. Returns null if nothing to flush. Names are actual
87     *         region names as returned by {@link HRegionInfo#getEncodedName()}
88     */
89    byte[][] rollWriter(boolean force) throws FailedLogCloseException, IOException;
90  
91    /**
92     * Stop accepting new writes. If we have unsynced writes still in buffer, sync them.
93     * Extant edits are left in place in backing storage to be replayed later.
94     */
95    void shutdown() throws IOException;
96  
97    /**
98     * Caller no longer needs any edits from this WAL. Implementers are free to reclaim
99     * underlying resources after this call; i.e. filesystem based WALs can archive or
100    * delete files.
101    */
102   void close() throws IOException;
103 
104   /**
105    * Append a set of edits to the WAL. The WAL is not flushed/sync'd after this transaction
106    * completes BUT on return this edit must have its region edit/sequence id assigned
107    * else it messes up our unification of mvcc and sequenceid.  On return <code>key</code> will
108    * have the region edit/sequence id filled in.
109    * @param info
110    * @param key Modified by this call; we add to it this edits region edit/sequence id.
111    * @param edits Edits to append. MAY CONTAIN NO EDITS for case where we want to get an edit
112    * sequence id that is after all currently appended edits.
113    * @param htd used to give scope for replication TODO refactor out in favor of table name and
114    * info
115    * @param inMemstore Always true except for case where we are writing a compaction completion
116    * record into the WAL; in this case the entry is just so we can finish an unfinished compaction
117    * -- it is not an edit for memstore.
118    * @return Returns a 'transaction id' and <code>key</code> will have the region edit/sequence id
119    * in it.
120    */
121   long append(HTableDescriptor htd, HRegionInfo info, WALKey key, WALEdit edits,
122     boolean inMemstore)
123   throws IOException;
124 
125   /**
126    * Sync what we have in the WAL.
127    * @throws IOException
128    */
129   void sync() throws IOException;
130 
131   /**
132    * Sync the WAL if the txId was not already sync'd.
133    * @param txid Transaction id to sync to.
134    * @throws IOException
135    */
136   void sync(long txid) throws IOException;
137 
138   /**
139    * WAL keeps track of the sequence numbers that are as yet not flushed im memstores
140    * in order to be able to do accounting to figure which WALs can be let go. This method tells WAL
141    * that some region is about to flush. The flush can be the whole region or for a column family
142    * of the region only.
143    *
144    * <p>Currently, it is expected that the update lock is held for the region; i.e. no
145    * concurrent appends while we set up cache flush.
146    * @param families Families to flush. May be a subset of all families in the region.
147    * @return Returns {@link HConstants#NO_SEQNUM} if we are flushing the whole region OR if
148    * we are flushing a subset of all families but there are no edits in those families not
149    * being flushed; in other words, this is effectively same as a flush of all of the region
150    * though we were passed a subset of regions. Otherwise, it returns the sequence id of the
151    * oldest/lowest outstanding edit.
152    * @see #completeCacheFlush(byte[])
153    * @see #abortCacheFlush(byte[])
154    */
155   Long startCacheFlush(final byte[] encodedRegionName, Set<byte[]> families);
156 
157   /**
158    * Complete the cache flush.
159    * @param encodedRegionName Encoded region name.
160    * @see #startCacheFlush(byte[], Set)
161    * @see #abortCacheFlush(byte[])
162    */
163   void completeCacheFlush(final byte[] encodedRegionName);
164 
165   /**
166    * Abort a cache flush. Call if the flush fails. Note that the only recovery
167    * for an aborted flush currently is a restart of the regionserver so the
168    * snapshot content dropped by the failure gets restored to the memstore.
169    * @param encodedRegionName Encoded region name.
170    */
171   void abortCacheFlush(byte[] encodedRegionName);
172 
173   /**
174    * @return Coprocessor host.
175    */
176   WALCoprocessorHost getCoprocessorHost();
177 
178   /**
179    * Gets the earliest unflushed sequence id in the memstore for the region.
180    * @param encodedRegionName The region to get the number for.
181    * @return The earliest/lowest/oldest sequence id if present, HConstants.NO_SEQNUM if absent.
182    * @deprecated Since version 1.2.0. Removing because not used and exposes subtle internal
183    * workings. Use {@link #getEarliestMemstoreSeqNum(byte[], byte[])}
184    */
185   @VisibleForTesting
186   @Deprecated
187   long getEarliestMemstoreSeqNum(byte[] encodedRegionName);
188 
189   /**
190    * Gets the earliest unflushed sequence id in the memstore for the store.
191    * @param encodedRegionName The region to get the number for.
192    * @param familyName The family to get the number for.
193    * @return The earliest/lowest/oldest sequence id if present, HConstants.NO_SEQNUM if absent.
194    */
195   long getEarliestMemstoreSeqNum(byte[] encodedRegionName, byte[] familyName);
196 
197   /**
198    * Human readable identifying information about the state of this WAL.
199    * Implementors are encouraged to include information appropriate for debugging.
200    * Consumers are advised not to rely on the details of the returned String; it does
201    * not have a defined structure.
202    */
203   @Override
204   String toString();
205 
206   /**
207    * When outside clients need to consume persisted WALs, they rely on a provided
208    * Reader.
209    */
210   interface Reader extends Closeable {
211     Entry next() throws IOException;
212     Entry next(Entry reuse) throws IOException;
213     void seek(long pos) throws IOException;
214     long getPosition() throws IOException;
215     void reset() throws IOException;
216   }
217 
218   /**
219    * Utility class that lets us keep track of the edit with it's key.
220    */
221   class Entry {
222     private WALEdit edit;
223     private WALKey key;
224 
225     public Entry() {
226       edit = new WALEdit();
227       // we use HLogKey here instead of WALKey directly to support legacy coprocessors.
228       key = new HLogKey();
229     }
230 
231     /**
232      * Constructor for both params
233      *
234      * @param edit log's edit
235      * @param key log's key
236      */
237     public Entry(WALKey key, WALEdit edit) {
238       super();
239       this.key = key;
240       this.edit = edit;
241     }
242 
243     /**
244      * Gets the edit
245      *
246      * @return edit
247      */
248     public WALEdit getEdit() {
249       return edit;
250     }
251 
252     /**
253      * Gets the key
254      *
255      * @return key
256      */
257     public WALKey getKey() {
258       return key;
259     }
260 
261     /**
262      * Set compression context for this entry.
263      *
264      * @param compressionContext
265      *          Compression context
266      */
267     public void setCompressionContext(CompressionContext compressionContext) {
268       edit.setCompressionContext(compressionContext);
269       key.setCompressionContext(compressionContext);
270     }
271 
272     @Override
273     public String toString() {
274       return this.key + "=" + this.edit;
275     }
276 
277   }
278 
279 }