Skip to content

Commit 6a40a76

Browse files
harishreedharantdas
authored andcommitted
[SPARK-4026][Streaming] Write ahead log management
As part of the effort to avoid data loss on Spark Streaming driver failure, we want to implement a write ahead log that can write received data to HDFS. This allows the received data to be persist across driver failures. So when the streaming driver is restarted, it can find and reprocess all the data that were received but not processed. This was primarily implemented by @harishreedharan. This is still WIP, as he is going to improve the unitests by using HDFS mini cluster. Author: Hari Shreedharan <[email protected]> Author: Tathagata Das <[email protected]> Closes apache#2882 from tdas/driver-ha-wal and squashes the following commits: e4bee20 [Tathagata Das] Removed synchronized, Path.getFileSystem is threadsafe 55514e2 [Tathagata Das] Minor changes based on PR comments. d29fddd [Tathagata Das] Merge pull request #20 from harishreedharan/driver-ha-wal a317a4d [Hari Shreedharan] Directory deletion should not fail tests 9514dc8 [Tathagata Das] Added unit tests to test reading of corrupted data and other minor edits 3881706 [Tathagata Das] Merge pull request #19 from harishreedharan/driver-ha-wal 4705fff [Hari Shreedharan] Sort listed files by name. Use local files for WAL tests. eb356ca [Tathagata Das] Merge pull request #18 from harishreedharan/driver-ha-wal 82ce56e [Hari Shreedharan] Fix file ordering issue in WALManager tests 5ff90ee [Hari Shreedharan] Fix tests to not ignore ordering and also assert all data is present ef8db09 [Tathagata Das] Merge pull request #17 from harishreedharan/driver-ha-wal 7e40e56 [Hari Shreedharan] Restore old build directory after tests 587b876 [Hari Shreedharan] Fix broken test. Call getFileSystem only from synchronized method. b4be0c1 [Hari Shreedharan] Remove unused method edcbee1 [Hari Shreedharan] Tests reading and writing data using writers now use Minicluster. 5c70d1f [Hari Shreedharan] Remove underlying stream from the WALWriter. 4ab602a [Tathagata Das] Refactored write ahead stuff from streaming.storage to streaming.util b06be2b [Tathagata Das] Adding missing license. 5182ffb [Hari Shreedharan] Added documentation 172358d [Tathagata Das] Pulled WriteAheadLog-related stuff from tdas/spark/tree/driver-ha-working
1 parent 7c89a8f commit 6a40a76

File tree

7 files changed

+892
-0
lines changed

7 files changed

+892
-0
lines changed
Lines changed: 72 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,72 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one or more
3+
* contributor license agreements. See the NOTICE file distributed with
4+
* this work for additional information regarding copyright ownership.
5+
* The ASF licenses this file to You under the Apache License, Version 2.0
6+
* (the "License"); you may not use this file except in compliance with
7+
* the License. You may obtain a copy of the License at
8+
*
9+
* http://www.apache.org/licenses/LICENSE-2.0
10+
*
11+
* Unless required by applicable law or agreed to in writing, software
12+
* distributed under the License is distributed on an "AS IS" BASIS,
13+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14+
* See the License for the specific language governing permissions and
15+
* limitations under the License.
16+
*/
17+
package org.apache.spark.streaming.util
18+
19+
import org.apache.hadoop.conf.Configuration
20+
import org.apache.hadoop.fs._
21+
22+
private[streaming] object HdfsUtils {
23+
24+
def getOutputStream(path: String, conf: Configuration): FSDataOutputStream = {
25+
val dfsPath = new Path(path)
26+
val dfs = getFileSystemForPath(dfsPath, conf)
27+
// If the file exists and we have append support, append instead of creating a new file
28+
val stream: FSDataOutputStream = {
29+
if (dfs.isFile(dfsPath)) {
30+
if (conf.getBoolean("hdfs.append.support", false)) {
31+
dfs.append(dfsPath)
32+
} else {
33+
throw new IllegalStateException("File exists and there is no append support!")
34+
}
35+
} else {
36+
dfs.create(dfsPath)
37+
}
38+
}
39+
stream
40+
}
41+
42+
def getInputStream(path: String, conf: Configuration): FSDataInputStream = {
43+
val dfsPath = new Path(path)
44+
val dfs = getFileSystemForPath(dfsPath, conf)
45+
val instream = dfs.open(dfsPath)
46+
instream
47+
}
48+
49+
def checkState(state: Boolean, errorMsg: => String) {
50+
if (!state) {
51+
throw new IllegalStateException(errorMsg)
52+
}
53+
}
54+
55+
def getBlockLocations(path: String, conf: Configuration): Option[Array[String]] = {
56+
val dfsPath = new Path(path)
57+
val dfs = getFileSystemForPath(dfsPath, conf)
58+
val fileStatus = dfs.getFileStatus(dfsPath)
59+
val blockLocs = Option(dfs.getFileBlockLocations(fileStatus, 0, fileStatus.getLen))
60+
blockLocs.map(_.flatMap(_.getHosts))
61+
}
62+
63+
def getFileSystemForPath(path: Path, conf: Configuration): FileSystem = {
64+
// For local file systems, return the raw loca file system, such calls to flush()
65+
// actually flushes the stream.
66+
val fs = path.getFileSystem(conf)
67+
fs match {
68+
case localFs: LocalFileSystem => localFs.getRawFileSystem
69+
case _ => fs
70+
}
71+
}
72+
}
Lines changed: 20 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,20 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one or more
3+
* contributor license agreements. See the NOTICE file distributed with
4+
* this work for additional information regarding copyright ownership.
5+
* The ASF licenses this file to You under the Apache License, Version 2.0
6+
* (the "License"); you may not use this file except in compliance with
7+
* the License. You may obtain a copy of the License at
8+
*
9+
* http://www.apache.org/licenses/LICENSE-2.0
10+
*
11+
* Unless required by applicable law or agreed to in writing, software
12+
* distributed under the License is distributed on an "AS IS" BASIS,
13+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14+
* See the License for the specific language governing permissions and
15+
* limitations under the License.
16+
*/
17+
package org.apache.spark.streaming.util
18+
19+
/** Class for representing a segment of data in a write ahead log file */
20+
private[streaming] case class WriteAheadLogFileSegment (path: String, offset: Long, length: Int)
Lines changed: 224 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,224 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one or more
3+
* contributor license agreements. See the NOTICE file distributed with
4+
* this work for additional information regarding copyright ownership.
5+
* The ASF licenses this file to You under the Apache License, Version 2.0
6+
* (the "License"); you may not use this file except in compliance with
7+
* the License. You may obtain a copy of the License at
8+
*
9+
* http://www.apache.org/licenses/LICENSE-2.0
10+
*
11+
* Unless required by applicable law or agreed to in writing, software
12+
* distributed under the License is distributed on an "AS IS" BASIS,
13+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14+
* See the License for the specific language governing permissions and
15+
* limitations under the License.
16+
*/
17+
package org.apache.spark.streaming.util
18+
19+
import java.nio.ByteBuffer
20+
21+
import scala.collection.mutable.ArrayBuffer
22+
import scala.concurrent.{ExecutionContext, Future}
23+
24+
import org.apache.hadoop.conf.Configuration
25+
import org.apache.hadoop.fs.Path
26+
import org.apache.hadoop.fs.permission.FsPermission
27+
import org.apache.spark.Logging
28+
import org.apache.spark.util.Utils
29+
import WriteAheadLogManager._
30+
31+
/**
32+
* This class manages write ahead log files.
33+
* - Writes records (bytebuffers) to periodically rotating log files.
34+
* - Recovers the log files and the reads the recovered records upon failures.
35+
* - Cleans up old log files.
36+
*
37+
* Uses [[org.apache.spark.streaming.util.WriteAheadLogWriter]] to write
38+
* and [[org.apache.spark.streaming.util.WriteAheadLogReader]] to read.
39+
*
40+
* @param logDirectory Directory when rotating log files will be created.
41+
* @param hadoopConf Hadoop configuration for reading/writing log files.
42+
* @param rollingIntervalSecs The interval in seconds with which logs will be rolled over.
43+
* Default is one minute.
44+
* @param maxFailures Max number of failures that is tolerated for every attempt to write to log.
45+
* Default is three.
46+
* @param callerName Optional name of the class who is using this manager.
47+
* @param clock Optional clock that is used to check for rotation interval.
48+
*/
49+
private[streaming] class WriteAheadLogManager(
50+
logDirectory: String,
51+
hadoopConf: Configuration,
52+
rollingIntervalSecs: Int = 60,
53+
maxFailures: Int = 3,
54+
callerName: String = "",
55+
clock: Clock = new SystemClock
56+
) extends Logging {
57+
58+
private val pastLogs = new ArrayBuffer[LogInfo]
59+
private val callerNameTag =
60+
if (callerName.nonEmpty) s" for $callerName" else ""
61+
private val threadpoolName = s"WriteAheadLogManager $callerNameTag"
62+
implicit private val executionContext = ExecutionContext.fromExecutorService(
63+
Utils.newDaemonFixedThreadPool(1, threadpoolName))
64+
override protected val logName = s"WriteAheadLogManager $callerNameTag"
65+
66+
private var currentLogPath: Option[String] = None
67+
private var currentLogWriter: WriteAheadLogWriter = null
68+
private var currentLogWriterStartTime: Long = -1L
69+
private var currentLogWriterStopTime: Long = -1L
70+
71+
initializeOrRecover()
72+
73+
/**
74+
* Write a byte buffer to the log file. This method synchronously writes the data in the
75+
* ByteBuffer to HDFS. When this method returns, the data is guaranteed to have been flushed
76+
* to HDFS, and will be available for readers to read.
77+
*/
78+
def writeToLog(byteBuffer: ByteBuffer): WriteAheadLogFileSegment = synchronized {
79+
var fileSegment: WriteAheadLogFileSegment = null
80+
var failures = 0
81+
var lastException: Exception = null
82+
var succeeded = false
83+
while (!succeeded && failures < maxFailures) {
84+
try {
85+
fileSegment = getLogWriter(clock.currentTime).write(byteBuffer)
86+
succeeded = true
87+
} catch {
88+
case ex: Exception =>
89+
lastException = ex
90+
logWarning("Failed to write to write ahead log")
91+
resetWriter()
92+
failures += 1
93+
}
94+
}
95+
if (fileSegment == null) {
96+
logError(s"Failed to write to write ahead log after $failures failures")
97+
throw lastException
98+
}
99+
fileSegment
100+
}
101+
102+
/**
103+
* Read all the existing logs from the log directory.
104+
*
105+
* Note that this is typically called when the caller is initializing and wants
106+
* to recover past state from the write ahead logs (that is, before making any writes).
107+
* If this is called after writes have been made using this manager, then it may not return
108+
* the latest the records. This does not deal with currently active log files, and
109+
* hence the implementation is kept simple.
110+
*/
111+
def readFromLog(): Iterator[ByteBuffer] = synchronized {
112+
val logFilesToRead = pastLogs.map{ _.path} ++ currentLogPath
113+
logInfo("Reading from the logs: " + logFilesToRead.mkString("\n"))
114+
logFilesToRead.iterator.map { file =>
115+
logDebug(s"Creating log reader with $file")
116+
new WriteAheadLogReader(file, hadoopConf)
117+
} flatMap { x => x }
118+
}
119+
120+
/**
121+
* Delete the log files that are older than the threshold time.
122+
*
123+
* Its important to note that the threshold time is based on the time stamps used in the log
124+
* files, which is usually based on the local system time. So if there is coordination necessary
125+
* between the node calculating the threshTime (say, driver node), and the local system time
126+
* (say, worker node), the caller has to take account of possible time skew.
127+
*/
128+
def cleanupOldLogs(threshTime: Long): Unit = {
129+
val oldLogFiles = synchronized { pastLogs.filter { _.endTime < threshTime } }
130+
logInfo(s"Attempting to clear ${oldLogFiles.size} old log files in $logDirectory " +
131+
s"older than $threshTime: ${oldLogFiles.map { _.path }.mkString("\n")}")
132+
133+
def deleteFiles() {
134+
oldLogFiles.foreach { logInfo =>
135+
try {
136+
val path = new Path(logInfo.path)
137+
val fs = HdfsUtils.getFileSystemForPath(path, hadoopConf)
138+
fs.delete(path, true)
139+
synchronized { pastLogs -= logInfo }
140+
logDebug(s"Cleared log file $logInfo")
141+
} catch {
142+
case ex: Exception =>
143+
logWarning(s"Error clearing write ahead log file $logInfo", ex)
144+
}
145+
}
146+
logInfo(s"Cleared log files in $logDirectory older than $threshTime")
147+
}
148+
if (!executionContext.isShutdown) {
149+
Future { deleteFiles() }
150+
}
151+
}
152+
153+
/** Stop the manager, close any open log writer */
154+
def stop(): Unit = synchronized {
155+
if (currentLogWriter != null) {
156+
currentLogWriter.close()
157+
}
158+
executionContext.shutdown()
159+
logInfo("Stopped write ahead log manager")
160+
}
161+
162+
/** Get the current log writer while taking care of rotation */
163+
private def getLogWriter(currentTime: Long): WriteAheadLogWriter = synchronized {
164+
if (currentLogWriter == null || currentTime > currentLogWriterStopTime) {
165+
resetWriter()
166+
currentLogPath.foreach {
167+
pastLogs += LogInfo(currentLogWriterStartTime, currentLogWriterStopTime, _)
168+
}
169+
currentLogWriterStartTime = currentTime
170+
currentLogWriterStopTime = currentTime + (rollingIntervalSecs * 1000)
171+
val newLogPath = new Path(logDirectory,
172+
timeToLogFile(currentLogWriterStartTime, currentLogWriterStopTime))
173+
currentLogPath = Some(newLogPath.toString)
174+
currentLogWriter = new WriteAheadLogWriter(currentLogPath.get, hadoopConf)
175+
}
176+
currentLogWriter
177+
}
178+
179+
/** Initialize the log directory or recover existing logs inside the directory */
180+
private def initializeOrRecover(): Unit = synchronized {
181+
val logDirectoryPath = new Path(logDirectory)
182+
val fileSystem = HdfsUtils.getFileSystemForPath(logDirectoryPath, hadoopConf)
183+
184+
if (fileSystem.exists(logDirectoryPath) && fileSystem.getFileStatus(logDirectoryPath).isDir) {
185+
val logFileInfo = logFilesTologInfo(fileSystem.listStatus(logDirectoryPath).map { _.getPath })
186+
pastLogs.clear()
187+
pastLogs ++= logFileInfo
188+
logInfo(s"Recovered ${logFileInfo.size} write ahead log files from $logDirectory")
189+
logDebug(s"Recovered files are:\n${logFileInfo.map(_.path).mkString("\n")}")
190+
}
191+
}
192+
193+
private def resetWriter(): Unit = synchronized {
194+
if (currentLogWriter != null) {
195+
currentLogWriter.close()
196+
currentLogWriter = null
197+
}
198+
}
199+
}
200+
201+
private[util] object WriteAheadLogManager {
202+
203+
case class LogInfo(startTime: Long, endTime: Long, path: String)
204+
205+
val logFileRegex = """log-(\d+)-(\d+)""".r
206+
207+
def timeToLogFile(startTime: Long, stopTime: Long): String = {
208+
s"log-$startTime-$stopTime"
209+
}
210+
211+
/** Convert a sequence of files to a sequence of sorted LogInfo objects */
212+
def logFilesTologInfo(files: Seq[Path]): Seq[LogInfo] = {
213+
files.flatMap { file =>
214+
logFileRegex.findFirstIn(file.getName()) match {
215+
case Some(logFileRegex(startTimeStr, stopTimeStr)) =>
216+
val startTime = startTimeStr.toLong
217+
val stopTime = stopTimeStr.toLong
218+
Some(LogInfo(startTime, stopTime, file.toString))
219+
case None =>
220+
None
221+
}
222+
}.sortBy { _.startTime }
223+
}
224+
}
Lines changed: 55 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,55 @@
1+
/*
2+
* Licensed to the Apache Software Foundation (ASF) under one or more
3+
* contributor license agreements. See the NOTICE file distributed with
4+
* this work for additional information regarding copyright ownership.
5+
* The ASF licenses this file to You under the Apache License, Version 2.0
6+
* (the "License"); you may not use this file except in compliance with
7+
* the License. You may obtain a copy of the License at
8+
*
9+
* http://www.apache.org/licenses/LICENSE-2.0
10+
*
11+
* Unless required by applicable law or agreed to in writing, software
12+
* distributed under the License is distributed on an "AS IS" BASIS,
13+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
14+
* See the License for the specific language governing permissions and
15+
* limitations under the License.
16+
*/
17+
package org.apache.spark.streaming.util
18+
19+
import java.io.Closeable
20+
import java.nio.ByteBuffer
21+
22+
import org.apache.hadoop.conf.Configuration
23+
24+
/**
25+
* A random access reader for reading write ahead log files written using
26+
* [[org.apache.spark.streaming.util.WriteAheadLogWriter]]. Given the file segment info,
27+
* this reads the record (bytebuffer) from the log file.
28+
*/
29+
private[streaming] class WriteAheadLogRandomReader(path: String, conf: Configuration)
30+
extends Closeable {
31+
32+
private val instream = HdfsUtils.getInputStream(path, conf)
33+
private var closed = false
34+
35+
def read(segment: WriteAheadLogFileSegment): ByteBuffer = synchronized {
36+
assertOpen()
37+
instream.seek(segment.offset)
38+
val nextLength = instream.readInt()
39+
HdfsUtils.checkState(nextLength == segment.length,
40+
s"Expected message length to be ${segment.length}, but was $nextLength")
41+
val buffer = new Array[Byte](nextLength)
42+
instream.readFully(buffer)
43+
ByteBuffer.wrap(buffer)
44+
}
45+
46+
override def close(): Unit = synchronized {
47+
closed = true
48+
instream.close()
49+
}
50+
51+
private def assertOpen() {
52+
HdfsUtils.checkState(!closed, "Stream is closed. Create a new Reader to read from the file.")
53+
}
54+
}
55+

0 commit comments

Comments
 (0)