@@ -19,7 +19,7 @@ package org.apache.spark.streaming.util
19
19
import java .io ._
20
20
import java .nio .ByteBuffer
21
21
22
- import org .apache .hadoop .fs .Path
22
+ import org .apache .hadoop .fs .{ FileStatus , Path }
23
23
24
24
import scala .collection .mutable .ArrayBuffer
25
25
import scala .concurrent .duration ._
@@ -41,14 +41,14 @@ class WriteAheadLogSuite extends FunSuite with BeforeAndAfter with BeforeAndAfte
41
41
val hadoopConf = new Configuration ()
42
42
val dfsDir = Files .createTempDir()
43
43
val TEST_BUILD_DATA_KEY : String = " test.build.data"
44
- val oldTestBuildDataProp = System .getProperty(TEST_BUILD_DATA_KEY )
44
+ val oldTestBuildDataProp = Option (System .getProperty(TEST_BUILD_DATA_KEY ))
45
+ System .setProperty(TEST_BUILD_DATA_KEY , dfsDir.toString)
45
46
val cluster = new MiniDFSCluster (new Configuration , 2 , true , null )
46
47
val nnPort = cluster.getNameNode.getNameNodeAddress.getPort
47
- val hdfsUrl = s " hdfs://localhost: $nnPort/ ${getRandomString()}/ "
48
+ val hdfsUrl = s " hdfs://localhost: $nnPort/ ${getRandomString()}/ "
48
49
var pathForTest : String = null
49
50
50
51
override def beforeAll () {
51
- System .setProperty(TEST_BUILD_DATA_KEY , dfsDir.toString)
52
52
cluster.waitActive()
53
53
}
54
54
@@ -59,7 +59,7 @@ class WriteAheadLogSuite extends FunSuite with BeforeAndAfter with BeforeAndAfte
59
59
override def afterAll () {
60
60
cluster.shutdown()
61
61
FileUtils .deleteDirectory(dfsDir)
62
- System .setProperty(TEST_BUILD_DATA_KEY , oldTestBuildDataProp )
62
+ oldTestBuildDataProp.foreach( System .setProperty(TEST_BUILD_DATA_KEY , _) )
63
63
}
64
64
65
65
test(" WriteAheadLogWriter - writing data" ) {
@@ -71,8 +71,7 @@ class WriteAheadLogSuite extends FunSuite with BeforeAndAfter with BeforeAndAfte
71
71
assert(writtenData.toArray === dataToWrite.toArray)
72
72
}
73
73
74
- test(" WriteAheadLogWriter - syncing of data by writing and reading immediately using " +
75
- " Minicluster" ) {
74
+ test(" WriteAheadLogWriter - syncing of data by writing and reading immediately" ) {
76
75
val dataToWrite = generateRandomData()
77
76
val writer = new WriteAheadLogWriter (pathForTest, hadoopConf)
78
77
dataToWrite.foreach { data =>
@@ -98,7 +97,7 @@ class WriteAheadLogSuite extends FunSuite with BeforeAndAfter with BeforeAndAfte
98
97
reader.close()
99
98
}
100
99
101
- test(" WriteAheadLogReader - sequentially reading data written with writer using Minicluster " ) {
100
+ test(" WriteAheadLogReader - sequentially reading data written with writer" ) {
102
101
// Write data manually for testing the sequential reader
103
102
val dataToWrite = generateRandomData()
104
103
writeDataUsingWriter(pathForTest, dataToWrite)
@@ -124,8 +123,7 @@ class WriteAheadLogSuite extends FunSuite with BeforeAndAfter with BeforeAndAfte
124
123
reader.close()
125
124
}
126
125
127
- test(" WriteAheadLogRandomReader - reading data using random reader written with writer using " +
128
- " Minicluster" ) {
126
+ test(" WriteAheadLogRandomReader - reading data using random reader written with writer" ) {
129
127
// Write data using writer for testing the random reader
130
128
val data = generateRandomData()
131
129
val segments = writeDataUsingWriter(pathForTest, data)
@@ -141,24 +139,23 @@ class WriteAheadLogSuite extends FunSuite with BeforeAndAfter with BeforeAndAfte
141
139
142
140
test(" WriteAheadLogManager - write rotating logs" ) {
143
141
// Write data using manager
144
- val dataToWrite = generateRandomData(10 )
142
+ val dataToWrite = generateRandomData()
145
143
val dir = pathForTest
146
144
writeDataUsingManager(dir, dataToWrite)
147
145
148
146
// Read data manually to verify the written data
149
147
val logFiles = getLogFilesInDirectory(dir)
150
148
assert(logFiles.size > 1 )
151
- val writtenData = logFiles.flatMap { file => readDataManually(file) }
152
- assert(writtenData.toSet === dataToWrite.toSet )
149
+ val writtenData = logFiles.flatMap { file => readDataManually(file)}
150
+ assert(writtenData.toList === dataToWrite.toList )
153
151
}
154
152
155
- // This one is failing right now -- commenting out for now.
156
153
test(" WriteAheadLogManager - read rotating logs" ) {
157
154
// Write data manually for testing reading through manager
158
155
val dir = pathForTest
159
156
val writtenData = (1 to 10 ).map { i =>
160
- val data = generateRandomData(10 )
161
- val file = dir + " /log-" + i
157
+ val data = generateRandomData()
158
+ val file = dir + s " /log- $i - $i "
162
159
writeDataManually(data, file)
163
160
data
164
161
}.flatten
@@ -169,12 +166,12 @@ class WriteAheadLogSuite extends FunSuite with BeforeAndAfter with BeforeAndAfte
169
166
170
167
// Read data using manager and verify
171
168
val readData = readDataUsingManager(dir)
172
- // assert(readData.toList === writtenData.toList)
169
+ assert(readData.toList === writtenData.toList)
173
170
}
174
171
175
172
test(" WriteAheadLogManager - recover past logs when creating new manager" ) {
176
173
// Write data with manager, recover with new manager and verify
177
- val dataToWrite = generateRandomData(100 )
174
+ val dataToWrite = generateRandomData()
178
175
val dir = pathForTest
179
176
writeDataUsingManager(dir, dataToWrite)
180
177
val logFiles = getLogFilesInDirectory(dir)
@@ -186,7 +183,7 @@ class WriteAheadLogSuite extends FunSuite with BeforeAndAfter with BeforeAndAfte
186
183
test(" WriteAheadLogManager - cleanup old logs" ) {
187
184
// Write data with manager, recover with new manager and verify
188
185
val dir = pathForTest
189
- val dataToWrite = generateRandomData(100 )
186
+ val dataToWrite = generateRandomData()
190
187
val fakeClock = new ManualClock
191
188
val manager = new WriteAheadLogManager (dir, hadoopConf,
192
189
rollingIntervalSecs = 1 , callerName = " WriteAheadLogSuite" , clock = fakeClock)
@@ -201,7 +198,6 @@ class WriteAheadLogSuite extends FunSuite with BeforeAndAfter with BeforeAndAfte
201
198
assert(getLogFilesInDirectory(dir).size < logFiles.size)
202
199
}
203
200
}
204
-
205
201
// TODO (Hari, TD): Test different failure conditions of writers and readers.
206
202
// - Failure while reading incomplete/corrupt file
207
203
}
@@ -243,8 +239,10 @@ object WriteAheadLogSuite {
243
239
244
240
def writeDataUsingManager (logDirectory : String , data : Seq [String ]) {
245
241
val fakeClock = new ManualClock
242
+ fakeClock.setTime(1000000 )
246
243
val manager = new WriteAheadLogManager (logDirectory, hadoopConf,
247
244
rollingIntervalSecs = 1 , callerName = " WriteAheadLogSuite" , clock = fakeClock)
245
+ // Ensure that 500 does not get sorted after 2000, so put a high base value.
248
246
data.foreach { item =>
249
247
fakeClock.addToTime(500 )
250
248
manager.writeToLog(item)
@@ -271,7 +269,8 @@ object WriteAheadLogSuite {
271
269
val reader = HdfsUtils .getInputStream(file, hadoopConf)
272
270
val buffer = new ArrayBuffer [String ]
273
271
try {
274
- while (true ) { // Read till EOF is thrown
272
+ while (true ) {
273
+ // Read till EOF is thrown
275
274
val length = reader.readInt()
276
275
val bytes = new Array [Byte ](length)
277
276
reader.read(bytes)
@@ -293,8 +292,10 @@ object WriteAheadLogSuite {
293
292
data
294
293
}
295
294
296
- def generateRandomData (numItems : Int = 50 , itemSize : Int = 50 ): Seq [String ] = {
297
- (1 to numItems).map { _.toString }
295
+ def generateRandomData (): Seq [String ] = {
296
+ (1 to 50 ).map {
297
+ _.toString
298
+ }
298
299
}
299
300
300
301
def getLogFilesInDirectory (directory : String ): Seq [String ] = {
0 commit comments