@@ -91,7 +91,8 @@ class BlockManagerSuite extends FunSuite with Matchers with BeforeAndAfter
9191 conf.set(" spark.driver.port" , boundPort.toString)
9292 conf.set(" spark.storage.unrollFraction" , " 0.4" )
9393 conf.set(" spark.storage.unrollMemoryThreshold" , " 512" )
94-
94+ conf.set(" spark.core.connection.ack.wait.timeout" , " 1" )
95+ conf.set(" spark.storage.cachedPeersTtl" , " 10" )
9596 master = new BlockManagerMaster (
9697 actorSystem.actorOf(Props (new BlockManagerMasterActor (true , conf, new LiveListenerBus ))),
9798 conf, true )
@@ -1300,59 +1301,55 @@ class BlockManagerSuite extends FunSuite with Matchers with BeforeAndAfter
13001301 testReplication(5 , storageLevels)
13011302 }
13021303
1303- test(" block replication with addition and removal of executors" ) {
1304+ test(" block replication with addition and deletion of executors" ) {
13041305 val blockSize = 1000
13051306 val storeSize = 10000
13061307 val allStores = new ArrayBuffer [BlockManager ]()
13071308
1309+
13081310 try {
13091311 val initialStores = (1 to 2 ).map { i => makeBlockManager(storeSize, s " store $i" ) }
13101312 allStores ++= initialStores
13111313
1312- // 2x replication works
1313- initialStores(0 ).putSingle(" a1" , new Array [Byte ](blockSize), StorageLevel .MEMORY_AND_DISK_2 )
1314- assert(master.getLocations(" a1" ).size === 2 )
1314+ def testPut (blockId : String , storageLevel : StorageLevel , expectedNumLocations : Int ) {
1315+ initialStores(0 ).putSingle(blockId, new Array [Byte ](blockSize), storageLevel)
1316+ assert(master.getLocations(blockId).size === expectedNumLocations)
1317+ master.removeBlock(blockId)
1318+ }
13151319
1316- // 3x replication should only replicate 2x
1317- initialStores(0 ).putSingle(
1318- " a2" , new Array [Byte ](blockSize), StorageLevel (true , true , false , true , 3 ))
1319- assert(master.getLocations(" a2" ).size === 2 )
1320+ // 2x replication should work, 3x replication should only replicate 2x
1321+ testPut(" a1" , StorageLevel .MEMORY_AND_DISK_2 , 2 )
1322+ testPut(" a2" , StorageLevel (true , true , false , true , 3 ), 2 )
13201323
1324+ // Add another store, 3x replication should work now, 4x replication should only replicate 3x
13211325 val newStore1 = makeBlockManager(storeSize, s " newstore1 " )
13221326 allStores += newStore1
1327+ eventually(timeout(1000 milliseconds), interval(10 milliseconds)) {
1328+ testPut(" a3" , StorageLevel (true , true , false , true , 3 ), 3 )
1329+ }
1330+ testPut(" a4" ,StorageLevel (true , true , false , true , 4 ), 3 )
13231331
1324- // 3x replication should work now
1325- initialStores(0 ).putSingle(
1326- " a3" , new Array [Byte ](blockSize), StorageLevel (true , true , false , true , 3 ))
1327- assert(master.getLocations(" a3" ).size === 3 )
1328-
1329- // 4x replication should only replicate 3x
1330- initialStores(0 ).putSingle(
1331- " a4" , new Array [Byte ](blockSize), StorageLevel (true , true , false , true , 4 ))
1332- assert(master.getLocations(" a4" ).size === 3 )
1333-
1332+ // Add another store, 4x replication should work now
13341333 val newStore2 = makeBlockManager(storeSize, s " newstore2 " )
13351334 allStores += newStore2
1335+ eventually(timeout(1000 milliseconds), interval(10 milliseconds)) {
1336+ testPut(" a5" , StorageLevel (true , true , false , true , 4 ), 4 )
1337+ }
13361338
1337- // 4x replication should work now
1338- initialStores(0 ).putSingle(
1339- " a5" , new Array [Byte ](blockSize), StorageLevel (true , true , false , true , 4 ))
1340- assert(master.getLocations(" a5" ).size === 4 )
1341-
1342- // Remove all the stores and add new stores
1343- (initialStores ++ Seq (newStore1, newStore2)).map { store =>
1344- store.blockManagerId.executorId
1345- }.foreach { execId =>
1346- master.removeExecutor(execId)
1339+ // Remove all but the 1st store, 2x replication should fail
1340+ (initialStores.slice(1 , initialStores.size) ++ Seq (newStore1, newStore2)).foreach {
1341+ store =>
1342+ master.removeExecutor(store.blockManagerId.executorId)
1343+ store.stop()
13471344 }
1345+ testPut(" a6" , StorageLevel .MEMORY_AND_DISK_2 , 1 )
13481346
1349- // Add new stores and test if replication works
1347+ // Add new stores, 3x replication should work
13501348 val newStores = (3 to 5 ).map { i => makeBlockManager(storeSize, s " newstore $i" ) }
13511349 allStores ++= newStores
1352-
1353- newStores(0 ).putSingle(
1354- " a6" , new Array [Byte ](blockSize), StorageLevel (true , true , false , true , 3 ))
1355- assert(master.getLocations(" a6" ).size === 3 )
1350+ eventually(timeout(1000 milliseconds), interval(10 milliseconds)) {
1351+ testPut(" a7" , StorageLevel (true , true , false , true , 3 ), 3 )
1352+ }
13561353 } finally {
13571354 allStores.foreach { _.stop() }
13581355 }
0 commit comments