@InterfaceAudience.Private public class FSHLog extends AbstractFSWAL<WALProvider.Writer>
Modifier and Type | Class and Description |
---|---|
(package private) class |
FSHLog.RingBufferEventHandler
Handler that is run by the disruptor ringbuffer consumer.
|
(package private) static class |
FSHLog.RingBufferExceptionHandler
Exception handler to pass the disruptor ringbuffer.
|
(package private) static class |
FSHLog.SafePointZigZagLatch
This class is used coordinating two threads holding one thread at a 'safe point' while the
orchestrating thread does some work that requires the first thread paused: e.g.
|
private class |
FSHLog.SyncRunner
Thread to runs the hdfs sync call.
|
WAL.Entry, WAL.Reader
Modifier and Type | Field and Description |
---|---|
private AtomicInteger |
closeErrorCount |
private int |
closeErrorsTolerated
Number of log close errors tolerated before we abort
|
private AtomicInteger |
consecutiveLogRolls |
private com.lmax.disruptor.dsl.Disruptor<RingBufferTruck> |
disruptor
The nexus at which all incoming handlers meet.
|
static long |
FIXED_OVERHEAD |
private org.apache.hadoop.fs.FSDataOutputStream |
hdfs_out
FSDataOutputStream associated with the current SequenceFile.writer
|
private static org.slf4j.Logger |
LOG |
private boolean |
lowReplicationRollEnabled |
private int |
lowReplicationRollLimit |
private int |
minTolerableReplication |
private FSHLog.RingBufferEventHandler |
ringBufferEventHandler
This fellow is run by the above appendExecutor service but it is all about batching up appends
and syncs; it may shutdown without cleaning out the last few appends or syncs.
|
blocksize, closed, conf, coprocessorHost, DEFAULT_SLOW_SYNC_TIME_MS, filenum, fs, highestSyncedTxid, highestUnsyncedTxid, implClassName, listeners, LOG_NAME_COMPARATOR, logrollsize, maxLogs, numEntries, ourFiles, prefixPathStr, rollRequested, rollWriterLock, sequenceIdAccounting, shutdown, slowSyncNs, totalLogSize, walArchiveDir, walDir, walFile2Props, walFilePrefix, walFileSuffix, writer
Constructor and Description |
---|
FSHLog(org.apache.hadoop.fs.FileSystem fs,
org.apache.hadoop.fs.Path root,
String logDir,
org.apache.hadoop.conf.Configuration conf)
Constructor.
|
FSHLog(org.apache.hadoop.fs.FileSystem fs,
org.apache.hadoop.fs.Path rootDir,
String logDir,
String archiveDir,
org.apache.hadoop.conf.Configuration conf,
List<WALActionsListener> listeners,
boolean failIfWALExists,
String prefix,
String suffix)
Create an edit log at the given
dir location. |
Modifier and Type | Method and Description |
---|---|
protected void |
afterCreatingZigZagLatch()
Used to manufacture race condition reliably.
|
long |
append(RegionInfo hri,
WALKeyImpl key,
WALEdit edits,
boolean inMemstore)
NOTE: This append, at a time that is usually after this call returns, starts an mvcc
transaction by calling 'begin' wherein which we assign this update a sequenceid.
|
protected void |
beforeWaitOnSafePoint() |
private void |
checkLogRoll()
Schedule a log roll if needed.
|
protected WALProvider.Writer |
createWriterInstance(org.apache.hadoop.fs.Path path)
This method allows subclasses to inject different writers without having to extend other
methods like rollWriter().
|
protected void |
doAppend(WALProvider.Writer writer,
FSWALEntry entry) |
protected boolean |
doCheckLogLowReplication() |
protected void |
doReplaceWriter(org.apache.hadoop.fs.Path oldPath,
org.apache.hadoop.fs.Path newPath,
WALProvider.Writer nextWriter)
Notice that you need to clear the
AbstractFSWAL.rollRequested flag in this method, as the new writer
will begin to work before returning from this method. |
protected void |
doShutdown() |
(package private) int |
getLogReplication()
This method gets the datanode replication count for the current WAL.
|
(package private) OutputStream |
getOutputStream()
Currently, we need to expose the writer's OutputStream to tests so that they can manipulate the
default behavior (such as setting the maxRecoveryErrorCount value).
|
(package private) org.apache.hadoop.hdfs.protocol.DatanodeInfo[] |
getPipeline()
This method gets the pipeline for the current WAL.
|
protected long |
getSequenceOnRingBuffer() |
(package private) boolean |
isLowReplicationRollEnabled() |
private void |
preemptiveSync(ProtobufLogWriter nextWriter)
Run a sync after opening to set up the pipeline.
|
private SyncFuture |
publishSyncOnRingBuffer() |
protected SyncFuture |
publishSyncOnRingBuffer(long sequence) |
private void |
publishSyncThenBlockOnCompletion(org.apache.htrace.core.TraceScope scope) |
void |
sync()
Sync what we have in the WAL.
|
void |
sync(long txid)
Sync the WAL if the txId was not already sync'd.
|
abortCacheFlush, append, atHeadOfRingBufferEventHandlerAppend, blockOnSync, checkLogLowReplication, close, completeCacheFlush, computeFilename, findRegionsToForceFlush, getCoprocessorHost, getCurrentFileName, getEarliestMemStoreSeqNum, getEarliestMemStoreSeqNum, getFilenum, getFileNumFromFileName, getFiles, getLogFileSize, getLogFileSizeIfBeingWritten, getNumLogFiles, getNumRolledLogFiles, getOldPath, getPreallocatedEventCount, getSyncFuture, getUnflushedEntriesCount, getWALArchivePath, isLogRollRequested, isUnflushedEntries, logRollAndSetupWalProps, main, postSync, registerWALActionsListener, replaceWriter, requestLogRoll, requestLogRoll, rollWriter, rollWriter, shutdown, stampSequenceIdAndPublishToRingBuffer, startCacheFlush, startCacheFlush, toString, unregisterWALActionsListener, updateStore
private static final org.slf4j.Logger LOG
private final com.lmax.disruptor.dsl.Disruptor<RingBufferTruck> disruptor
private final FSHLog.RingBufferEventHandler ringBufferEventHandler
private org.apache.hadoop.fs.FSDataOutputStream hdfs_out
private final int minTolerableReplication
private final AtomicInteger consecutiveLogRolls
private final int lowReplicationRollLimit
private volatile boolean lowReplicationRollEnabled
private final int closeErrorsTolerated
private final AtomicInteger closeErrorCount
public static final long FIXED_OVERHEAD
public FSHLog(org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path root, String logDir, org.apache.hadoop.conf.Configuration conf) throws IOException
fs
- filesystem handleroot
- path for stored and archived walslogDir
- dir where wals are storedconf
- configuration to useIOException
public FSHLog(org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path rootDir, String logDir, String archiveDir, org.apache.hadoop.conf.Configuration conf, List<WALActionsListener> listeners, boolean failIfWALExists, String prefix, String suffix) throws IOException
dir
location. You should never have to load an
existing log. If there is a log at startup, it should have already been processed and deleted
by the time the WAL object is started up.fs
- filesystem handlerootDir
- path to where logs and oldlogslogDir
- dir where wals are storedarchiveDir
- dir where wals are archivedconf
- configuration to uselisteners
- Listeners on WAL events. Listeners passed here will be registered before we do
anything else; e.g. the Constructor AbstractFSWAL.rollWriter()
.failIfWALExists
- If true IOException will be thrown if files related to this wal already
exist.prefix
- should always be hostname and port in distributed env and it will be URL encoded
before being used. If prefix is null, "wal" will be usedsuffix
- will be url encoded. null is treated as empty. non-empty must start with
AbstractFSWALProvider.WAL_FILE_NAME_DELIMITER
IOException
OutputStream getOutputStream()
private void preemptiveSync(ProtobufLogWriter nextWriter)
protected WALProvider.Writer createWriterInstance(org.apache.hadoop.fs.Path path) throws IOException
createWriterInstance
in class AbstractFSWAL<WALProvider.Writer>
IOException
protected void afterCreatingZigZagLatch()
beforeWaitOnSafePoint()
protected void beforeWaitOnSafePoint()
afterCreatingZigZagLatch()
protected void doAppend(WALProvider.Writer writer, FSWALEntry entry) throws IOException
doAppend
in class AbstractFSWAL<WALProvider.Writer>
IOException
protected void doReplaceWriter(org.apache.hadoop.fs.Path oldPath, org.apache.hadoop.fs.Path newPath, WALProvider.Writer nextWriter) throws IOException
AbstractFSWAL
AbstractFSWAL.rollRequested
flag in this method, as the new writer
will begin to work before returning from this method. If we clear the flag after returning from
this call, we may miss a roll request. The implementation class should choose a proper place to
clear the AbstractFSWAL.rollRequested
flag so we do not miss a roll request, typically before you
start writing to the new writer.doReplaceWriter
in class AbstractFSWAL<WALProvider.Writer>
IOException
protected void doShutdown() throws IOException
doShutdown
in class AbstractFSWAL<WALProvider.Writer>
IOException
public long append(RegionInfo hri, WALKeyImpl key, WALEdit edits, boolean inMemstore) throws IOException
AbstractFSWAL
walKey
parameter. Be warned that the WriteEntry is not
immediately available on return from this method. It WILL be available subsequent to a sync of
this append; otherwise, you will just have to wait on the WriteEntry to get filled in.append
in interface WAL
append
in class AbstractFSWAL<WALProvider.Writer>
hri
- the regioninfo associated with appendkey
- Modified by this call; we add to it this edits region edit/sequence id.edits
- Edits to append. MAY CONTAIN NO EDITS for case where we want to get an edit
sequence id that is after all currently appended edits.inMemstore
- Always true except for case where we are writing a compaction completion
record into the WAL; in this case the entry is just so we can finish an unfinished compaction
-- it is not an edit for memstore.key
will have the region edit/sequence id
in it.IOException
private void checkLogRoll()
protected boolean doCheckLogLowReplication()
doCheckLogLowReplication
in class AbstractFSWAL<WALProvider.Writer>
protected long getSequenceOnRingBuffer()
private SyncFuture publishSyncOnRingBuffer()
protected SyncFuture publishSyncOnRingBuffer(long sequence)
private void publishSyncThenBlockOnCompletion(org.apache.htrace.core.TraceScope scope) throws IOException
IOException
int getLogReplication()
If the pipeline isn't started yet or is empty, you will get the default replication factor. Therefore, if this function returns 0, it means you are not properly running with the HDFS-826 patch.
getLogReplication
in class AbstractFSWAL<WALProvider.Writer>
public void sync() throws IOException
WAL
IOException
public void sync(long txid) throws IOException
WAL
txid
- Transaction id to sync to.IOException
boolean isLowReplicationRollEnabled()
org.apache.hadoop.hdfs.protocol.DatanodeInfo[] getPipeline()
getPipeline
in class AbstractFSWAL<WALProvider.Writer>
Copyright © 2007–2019 The Apache Software Foundation. All rights reserved.