mirror of https://github.com/apache/kafka.git
kafka-903; Attempt to swap the new high watermark file with the old one failed on Windows; patched by Jun Rao; reviewed by Neha Narkhede, Jay Kreps and Sriram Subramania
This commit is contained in:
parent
4f387ae435
commit
43c43b1c6d
|
@ -36,7 +36,7 @@ object HighwaterMarkCheckpoint {
|
||||||
|
|
||||||
class HighwaterMarkCheckpoint(val path: String) extends Logging {
|
class HighwaterMarkCheckpoint(val path: String) extends Logging {
|
||||||
/* create the highwatermark file handle for all partitions */
|
/* create the highwatermark file handle for all partitions */
|
||||||
val name = path + "/" + HighwaterMarkCheckpoint.highWatermarkFileName
|
val name = path + File.separator + HighwaterMarkCheckpoint.highWatermarkFileName
|
||||||
private val hwFile = new File(name)
|
private val hwFile = new File(name)
|
||||||
private val hwFileLock = new ReentrantLock()
|
private val hwFileLock = new ReentrantLock()
|
||||||
// recover from previous tmp file, if required
|
// recover from previous tmp file, if required
|
||||||
|
@ -63,10 +63,14 @@ class HighwaterMarkCheckpoint(val path: String) extends Logging {
|
||||||
hwFileWriter.flush()
|
hwFileWriter.flush()
|
||||||
hwFileWriter.close()
|
hwFileWriter.close()
|
||||||
// swap new high watermark file with previous one
|
// swap new high watermark file with previous one
|
||||||
|
if(!tempHwFile.renameTo(hwFile)) {
|
||||||
|
// renameTo() fails on Windows if the destination file exists.
|
||||||
|
hwFile.delete()
|
||||||
if(!tempHwFile.renameTo(hwFile)) {
|
if(!tempHwFile.renameTo(hwFile)) {
|
||||||
fatal("Attempt to swap the new high watermark file with the old one failed")
|
fatal("Attempt to swap the new high watermark file with the old one failed")
|
||||||
System.exit(1)
|
System.exit(1)
|
||||||
}
|
}
|
||||||
|
}
|
||||||
}finally {
|
}finally {
|
||||||
hwFileLock.unlock()
|
hwFileLock.unlock()
|
||||||
}
|
}
|
||||||
|
|
Loading…
Reference in New Issue