!tes #2971 Make TestKit.remaining throw AssertionError outside of within
This commit is contained in:
parent
dfef14a590
commit
826cc74de3
41 changed files with 156 additions and 120 deletions
|
|
@ -441,8 +441,8 @@ class ActorRefSpec extends AkkaSpec with DefaultTimeout {
|
|||
def receive = { case name: String ⇒ sender() ! context.child(name).isDefined }
|
||||
}), "parent")
|
||||
|
||||
assert(Await.result((parent ? "child"), remaining) === true)
|
||||
assert(Await.result((parent ? "whatnot"), remaining) === false)
|
||||
assert(Await.result((parent ? "child"), timeout.duration) === true)
|
||||
assert(Await.result((parent ? "whatnot"), timeout.duration) === false)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -149,7 +149,7 @@ trait SchedulerSpec extends BeforeAndAfterEach with DefaultTimeout with Implicit
|
|||
"not be canceled if cancel is performed after execution" in {
|
||||
val latch = TestLatch(1)
|
||||
val task = collectCancellable(system.scheduler.scheduleOnce(10 millis)(latch.countDown()))
|
||||
Await.ready(latch, remaining)
|
||||
Await.ready(latch, remainingOrDefault)
|
||||
task.cancel() should be(false)
|
||||
task.isCancelled should be(false)
|
||||
task.cancel() should be(false)
|
||||
|
|
|
|||
|
|
@ -445,12 +445,12 @@ abstract class ActorModelSpec(config: String) extends AkkaSpec(config) with Defa
|
|||
stack foreach (s ⇒ println(s"\t$s"))
|
||||
}
|
||||
}
|
||||
assert(Await.result(f1, remaining) === "foo")
|
||||
assert(Await.result(f2, remaining) === "bar")
|
||||
assert(Await.result(f4, remaining) === "foo2")
|
||||
assert(intercept[ActorInterruptedException](Await.result(f3, remaining)).getCause.getMessage === "Ping!")
|
||||
assert(Await.result(f6, remaining) === "bar2")
|
||||
assert(intercept[ActorInterruptedException](Await.result(f5, remaining)).getCause.getMessage === "Ping!")
|
||||
assert(Await.result(f1, timeout.duration) === "foo")
|
||||
assert(Await.result(f2, timeout.duration) === "bar")
|
||||
assert(Await.result(f4, timeout.duration) === "foo2")
|
||||
assert(intercept[ActorInterruptedException](Await.result(f3, timeout.duration)).getCause.getMessage === "Ping!")
|
||||
assert(Await.result(f6, timeout.duration) === "bar2")
|
||||
assert(intercept[ActorInterruptedException](Await.result(f5, timeout.duration)).getCause.getMessage === "Ping!")
|
||||
c.cancel()
|
||||
Thread.sleep(300) // give the EventFilters a chance of catching all messages
|
||||
}
|
||||
|
|
@ -467,12 +467,12 @@ abstract class ActorModelSpec(config: String) extends AkkaSpec(config) with Defa
|
|||
val f5 = a ? InterruptNicely("baz2")
|
||||
val f6 = a ? Reply("bar2")
|
||||
|
||||
assert(Await.result(f1, remaining) === "foo")
|
||||
assert(Await.result(f2, remaining) === "bar")
|
||||
assert(Await.result(f3, remaining) === "baz")
|
||||
assert(Await.result(f4, remaining) === "foo2")
|
||||
assert(Await.result(f5, remaining) === "baz2")
|
||||
assert(Await.result(f6, remaining) === "bar2")
|
||||
assert(Await.result(f1, timeout.duration) === "foo")
|
||||
assert(Await.result(f2, timeout.duration) === "bar")
|
||||
assert(Await.result(f3, timeout.duration) === "baz")
|
||||
assert(Await.result(f4, timeout.duration) === "foo2")
|
||||
assert(Await.result(f5, timeout.duration) === "baz2")
|
||||
assert(Await.result(f6, timeout.duration) === "bar2")
|
||||
// clear the interrupted flag (only needed for the CallingThreadDispatcher) so the next test can continue normally
|
||||
Thread.interrupted()
|
||||
}
|
||||
|
|
@ -489,10 +489,10 @@ abstract class ActorModelSpec(config: String) extends AkkaSpec(config) with Defa
|
|||
val f5 = a ? ThrowException(new RemoteException("RemoteException"))
|
||||
val f6 = a ? Reply("bar2")
|
||||
|
||||
assert(Await.result(f1, remaining) === "foo")
|
||||
assert(Await.result(f2, remaining) === "bar")
|
||||
assert(Await.result(f4, remaining) === "foo2")
|
||||
assert(Await.result(f6, remaining) === "bar2")
|
||||
assert(Await.result(f1, timeout.duration) === "foo")
|
||||
assert(Await.result(f2, timeout.duration) === "bar")
|
||||
assert(Await.result(f4, timeout.duration) === "foo2")
|
||||
assert(Await.result(f6, timeout.duration) === "bar2")
|
||||
assert(f3.value.isEmpty)
|
||||
assert(f5.value.isEmpty)
|
||||
}
|
||||
|
|
|
|||
|
|
@ -89,7 +89,7 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
def assertMyDispatcherIsUsed(actor: ActorRef): Unit = {
|
||||
actor ! "what's the name?"
|
||||
val Expected = "(DispatchersSpec-myapp.mydispatcher-[1-9][0-9]*)".r
|
||||
expectMsgPF(remaining) {
|
||||
expectMsgPF() {
|
||||
case Expected(x) ⇒
|
||||
}
|
||||
}
|
||||
|
|
@ -143,7 +143,7 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
"include system name and dispatcher id in thread names for thread-pool-executor" in {
|
||||
system.actorOf(Props[ThreadNameEcho].withDispatcher("myapp.thread-pool-dispatcher")) ! "what's the name?"
|
||||
val Expected = "(DispatchersSpec-myapp.thread-pool-dispatcher-[1-9][0-9]*)".r
|
||||
expectMsgPF(remaining) {
|
||||
expectMsgPF() {
|
||||
case Expected(x) ⇒
|
||||
}
|
||||
}
|
||||
|
|
@ -151,7 +151,7 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
"include system name and dispatcher id in thread names for default-dispatcher" in {
|
||||
system.actorOf(Props[ThreadNameEcho]) ! "what's the name?"
|
||||
val Expected = "(DispatchersSpec-akka.actor.default-dispatcher-[1-9][0-9]*)".r
|
||||
expectMsgPF(remaining) {
|
||||
expectMsgPF() {
|
||||
case Expected(x) ⇒
|
||||
}
|
||||
}
|
||||
|
|
@ -159,7 +159,7 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
"include system name and dispatcher id in thread names for pinned dispatcher" in {
|
||||
system.actorOf(Props[ThreadNameEcho].withDispatcher("myapp.my-pinned-dispatcher")) ! "what's the name?"
|
||||
val Expected = "(DispatchersSpec-myapp.my-pinned-dispatcher-[1-9][0-9]*)".r
|
||||
expectMsgPF(remaining) {
|
||||
expectMsgPF() {
|
||||
case Expected(x) ⇒
|
||||
}
|
||||
}
|
||||
|
|
@ -167,7 +167,7 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
"include system name and dispatcher id in thread names for balancing dispatcher" in {
|
||||
system.actorOf(Props[ThreadNameEcho].withDispatcher("myapp.balancing-dispatcher")) ! "what's the name?"
|
||||
val Expected = "(DispatchersSpec-myapp.balancing-dispatcher-[1-9][0-9]*)".r
|
||||
expectMsgPF(remaining) {
|
||||
expectMsgPF() {
|
||||
case Expected(x) ⇒
|
||||
}
|
||||
}
|
||||
|
|
@ -187,7 +187,7 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
val routee = expectMsgType[ActorIdentity].ref.get
|
||||
routee ! "what's the name?"
|
||||
val Expected = """(DispatchersSpec-akka\.actor\.deployment\./pool1\.pool-dispatcher-[1-9][0-9]*)""".r
|
||||
expectMsgPF(remaining) {
|
||||
expectMsgPF() {
|
||||
case Expected(x) ⇒
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -385,7 +385,7 @@ class FutureSpec extends AkkaSpec with Checkers with BeforeAndAfterAll with Defa
|
|||
}
|
||||
|
||||
"fold" in {
|
||||
Await.result(Future.fold((1 to 10).toList map { i ⇒ Future(i) })(0)(_ + _), remaining) should be(55)
|
||||
Await.result(Future.fold((1 to 10).toList map { i ⇒ Future(i) })(0)(_ + _), remainingOrDefault) should be(55)
|
||||
}
|
||||
|
||||
"zip" in {
|
||||
|
|
@ -417,7 +417,7 @@ class FutureSpec extends AkkaSpec with Checkers with BeforeAndAfterAll with Defa
|
|||
case 6 ⇒ Future(throw new IllegalArgumentException("shouldFoldResultsWithException: expected"))
|
||||
case i ⇒ Future(i)
|
||||
}
|
||||
intercept[Throwable] { Await.result(Future.fold(futures)(0)(_ + _), remaining) }.getMessage should be("shouldFoldResultsWithException: expected")
|
||||
intercept[Throwable] { Await.result(Future.fold(futures)(0)(_ + _), remainingOrDefault) }.getMessage should be("shouldFoldResultsWithException: expected")
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -443,7 +443,7 @@ class FutureSpec extends AkkaSpec with Checkers with BeforeAndAfterAll with Defa
|
|||
|
||||
"reduce results" in {
|
||||
val futures = (1 to 10).toList map { i ⇒ Future(i) }
|
||||
assert(Await.result(Future.reduce(futures)(_ + _), remaining) === 55)
|
||||
assert(Await.result(Future.reduce(futures)(_ + _), remainingOrDefault) === 55)
|
||||
}
|
||||
|
||||
"reduce results with Exception" in {
|
||||
|
|
@ -452,7 +452,7 @@ class FutureSpec extends AkkaSpec with Checkers with BeforeAndAfterAll with Defa
|
|||
case 6 ⇒ Future(throw new IllegalArgumentException("shouldReduceResultsWithException: expected"))
|
||||
case i ⇒ Future(i)
|
||||
}
|
||||
intercept[Throwable] { Await.result(Future.reduce(futures)(_ + _), remaining) }.getMessage should be("shouldReduceResultsWithException: expected")
|
||||
intercept[Throwable] { Await.result(Future.reduce(futures)(_ + _), remainingOrDefault) }.getMessage should be("shouldReduceResultsWithException: expected")
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -146,7 +146,7 @@ abstract class MailboxSpec extends AkkaSpec with BeforeAndAfterAll with BeforeAn
|
|||
val ps = for (i ← (1 to enqueueN by step).toList) yield createProducer(i, Math.min(enqueueN, i + step - 1))
|
||||
|
||||
if (parallel == false)
|
||||
ps foreach { Await.ready(_, remaining) }
|
||||
ps foreach { Await.ready(_, remainingOrDefault) }
|
||||
|
||||
ps
|
||||
}
|
||||
|
|
@ -162,8 +162,8 @@ abstract class MailboxSpec extends AkkaSpec with BeforeAndAfterAll with BeforeAn
|
|||
|
||||
val consumers = List.fill(maxConsumers)(createConsumer)
|
||||
|
||||
val ps = producers.map(Await.result(_, remaining))
|
||||
val cs = consumers.map(Await.result(_, remaining))
|
||||
val ps = producers.map(Await.result(_, remainingOrDefault))
|
||||
val cs = consumers.map(Await.result(_, remainingOrDefault))
|
||||
|
||||
ps.map(_.size).sum should be(enqueueN) //Must have produced 1000 messages
|
||||
cs.map(_.size).sum should be(dequeueN) //Must have consumed all produced messages
|
||||
|
|
|
|||
|
|
@ -57,7 +57,7 @@ class ConsistentHashingRouterSpec extends AkkaSpec(ConsistentHashingRouterSpec.c
|
|||
|
||||
"consistent hashing router" must {
|
||||
"create routees from configuration" in {
|
||||
val currentRoutees = Await.result(router1 ? CurrentRoutees, remaining).asInstanceOf[RouterRoutees]
|
||||
val currentRoutees = Await.result(router1 ? CurrentRoutees, remainingOrDefault).asInstanceOf[RouterRoutees]
|
||||
currentRoutees.routees.size should be(3)
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -50,7 +50,7 @@ class ResizerSpec extends AkkaSpec(ResizerSpec.config) with DefaultTimeout with
|
|||
}
|
||||
|
||||
def routeeSize(router: ActorRef): Int =
|
||||
Await.result(router ? CurrentRoutees, remaining).asInstanceOf[RouterRoutees].routees.size
|
||||
Await.result(router ? CurrentRoutees, remainingOrDefault).asInstanceOf[RouterRoutees].routees.size
|
||||
|
||||
"DefaultResizer" must {
|
||||
|
||||
|
|
@ -108,7 +108,7 @@ class ResizerSpec extends AkkaSpec(ResizerSpec.config) with DefaultTimeout with
|
|||
router ! latch
|
||||
router ! latch
|
||||
|
||||
Await.ready(latch, remaining)
|
||||
Await.ready(latch, remainingOrDefault)
|
||||
|
||||
// messagesPerResize is 10 so there is no risk of additional resize
|
||||
routeeSize(router) should be(2)
|
||||
|
|
@ -123,7 +123,7 @@ class ResizerSpec extends AkkaSpec(ResizerSpec.config) with DefaultTimeout with
|
|||
router ! latch
|
||||
router ! latch
|
||||
|
||||
Await.ready(latch, remaining)
|
||||
Await.ready(latch, remainingOrDefault)
|
||||
|
||||
routeeSize(router) should be(2)
|
||||
}
|
||||
|
|
|
|||
|
|
@ -95,7 +95,7 @@ class RoutingSpec extends AkkaSpec(RoutingSpec.config) with DefaultTimeout with
|
|||
}
|
||||
val router = system.actorOf(Props[TestActor].withRouter(RoundRobinRouter(resizer = Some(resizer))))
|
||||
watch(router)
|
||||
Await.ready(latch, remaining)
|
||||
Await.ready(latch, remainingOrDefault)
|
||||
router ! CurrentRoutees
|
||||
val routees = expectMsgType[RouterRoutees].routees
|
||||
routees.size should be(2)
|
||||
|
|
@ -156,7 +156,7 @@ class RoutingSpec extends AkkaSpec(RoutingSpec.config) with DefaultTimeout with
|
|||
}
|
||||
}
|
||||
val router = system.actorOf(Props[TestActor].withRouter(RoundRobinRouter(resizer = Some(resizer))), "router3")
|
||||
Await.ready(latch, remaining)
|
||||
Await.ready(latch, remainingOrDefault)
|
||||
router ! CurrentRoutees
|
||||
expectMsgType[RouterRoutees].routees.size should be(3)
|
||||
system.stop(router)
|
||||
|
|
@ -296,7 +296,7 @@ class RoutingSpec extends AkkaSpec(RoutingSpec.config) with DefaultTimeout with
|
|||
|
||||
routedActor ! Broadcast("end")
|
||||
//now wait some and do validations.
|
||||
Await.ready(doneLatch, remaining)
|
||||
Await.ready(doneLatch, remainingOrDefault)
|
||||
|
||||
for (i ← 0 until connectionCount)
|
||||
counters(i).get should be((iterationCount * (i + 1)))
|
||||
|
|
@ -326,7 +326,7 @@ class RoutingSpec extends AkkaSpec(RoutingSpec.config) with DefaultTimeout with
|
|||
routedActor ! Broadcast(1)
|
||||
routedActor ! Broadcast("end")
|
||||
|
||||
Await.ready(doneLatch, remaining)
|
||||
Await.ready(doneLatch, remainingOrDefault)
|
||||
|
||||
counter1.get should be(1)
|
||||
counter2.get should be(1)
|
||||
|
|
@ -364,7 +364,7 @@ class RoutingSpec extends AkkaSpec(RoutingSpec.config) with DefaultTimeout with
|
|||
routedActor ! Broadcast(1)
|
||||
routedActor ! Broadcast("end")
|
||||
|
||||
Await.ready(doneLatch, remaining)
|
||||
Await.ready(doneLatch, remainingOrDefault)
|
||||
|
||||
counter1.get should be(1)
|
||||
counter2.get should be(1)
|
||||
|
|
@ -455,7 +455,7 @@ class RoutingSpec extends AkkaSpec(RoutingSpec.config) with DefaultTimeout with
|
|||
routedActor ! 1
|
||||
routedActor ! "end"
|
||||
|
||||
Await.ready(doneLatch, remaining)
|
||||
Await.ready(doneLatch, remainingOrDefault)
|
||||
|
||||
counter1.get should be(1)
|
||||
counter2.get should be(1)
|
||||
|
|
@ -486,7 +486,7 @@ class RoutingSpec extends AkkaSpec(RoutingSpec.config) with DefaultTimeout with
|
|||
routedActor ? 1
|
||||
routedActor ! "end"
|
||||
|
||||
Await.ready(doneLatch, remaining)
|
||||
Await.ready(doneLatch, remainingOrDefault)
|
||||
|
||||
counter1.get should be(1)
|
||||
counter2.get should be(1)
|
||||
|
|
|
|||
|
|
@ -43,7 +43,7 @@ class AskSpec extends AkkaSpec {
|
|||
val f = ask(null: ActorRef, 3.14)
|
||||
f.isCompleted should be(true)
|
||||
intercept[IllegalArgumentException] {
|
||||
Await.result(f, remaining)
|
||||
Await.result(f, timeout.duration)
|
||||
}.getMessage should be("Unsupported recipient ActorRef type, question not sent to [null]")
|
||||
}
|
||||
|
||||
|
|
@ -53,7 +53,7 @@ class AskSpec extends AkkaSpec {
|
|||
val f = echo ? "foo"
|
||||
val expectedMsg = "Timeout length must not be negative, question not sent to [%s]" format echo
|
||||
intercept[IllegalArgumentException] {
|
||||
Await.result(f, remaining)
|
||||
Await.result(f, timeout.duration)
|
||||
}.getMessage should be(expectedMsg)
|
||||
}
|
||||
|
||||
|
|
@ -63,7 +63,7 @@ class AskSpec extends AkkaSpec {
|
|||
val f = echo ? "foo"
|
||||
val expectedMsg = "Timeout length must not be negative, question not sent to [%s]" format echo
|
||||
intercept[IllegalArgumentException] {
|
||||
Await.result(f, remaining)
|
||||
Await.result(f, timeout.duration)
|
||||
}.getMessage should be(expectedMsg)
|
||||
}
|
||||
|
||||
|
|
@ -72,7 +72,7 @@ class AskSpec extends AkkaSpec {
|
|||
val silentOne = system.actorOf(Props.empty, "silent")
|
||||
val f = silentOne ? "noreply"
|
||||
intercept[AskTimeoutException] {
|
||||
Await.result(f, remaining)
|
||||
Await.result(f, 1 second)
|
||||
}.getMessage.contains("/user/silent") should be(true)
|
||||
}
|
||||
|
||||
|
|
@ -80,7 +80,7 @@ class AskSpec extends AkkaSpec {
|
|||
implicit val timeout = Timeout(0.5 seconds)
|
||||
val f = system.actorOf(Props.empty) ? "noreply"
|
||||
intercept[AskTimeoutException] {
|
||||
Await.result(f, remaining)
|
||||
Await.result(f, 1 second)
|
||||
}.getMessage should include(timeout.duration.toMillis.toString)
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -24,7 +24,7 @@ class CircuitBreakerMTSpec extends AkkaSpec {
|
|||
Await.result(breaker.withCircuitBreaker(Future(throw new RuntimeException("FAIL"))) recover {
|
||||
case _: CircuitBreakerOpenException ⇒ true
|
||||
case _ ⇒ false
|
||||
}, remaining)
|
||||
}, remainingOrDefault)
|
||||
|
||||
// fire some failing calls
|
||||
1 to (maxFailures + 1) foreach { _ ⇒ failingCall() }
|
||||
|
|
|
|||
|
|
@ -122,7 +122,7 @@ class CircuitBreakerSpec extends AkkaSpec with BeforeAndAfter {
|
|||
"increment failure count on callTimeout" in {
|
||||
val breaker = CircuitBreakerSpec.shortCallTimeoutCb()
|
||||
breaker().withSyncCircuitBreaker(Thread.sleep(100.millis.dilated.toMillis))
|
||||
awaitCond(breaker().currentFailureCount == 1, remaining)
|
||||
awaitCond(breaker().currentFailureCount == 1)
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -44,8 +44,8 @@ class PatternSpec extends AkkaSpec("akka.actor.serialize-messages = off") {
|
|||
"complete Future with AskTimeoutException when actor not terminated within timeout" in {
|
||||
val target = system.actorOf(Props[TargetActor])
|
||||
val latch = TestLatch()
|
||||
target ! ((latch, remaining))
|
||||
intercept[AskTimeoutException] { Await.result(gracefulStop(target, 500 millis), remaining) }
|
||||
target ! ((latch, remainingOrDefault))
|
||||
intercept[AskTimeoutException] { Await.result(gracefulStop(target, 500 millis), remainingOrDefault) }
|
||||
latch.open()
|
||||
}
|
||||
}
|
||||
|
|
@ -56,7 +56,7 @@ class PatternSpec extends AkkaSpec("akka.actor.serialize-messages = off") {
|
|||
val f = akka.pattern.after(1 second, using = system.scheduler)(Promise.successful(5).future)
|
||||
|
||||
val r = Future.firstCompletedOf(Seq(Promise[Int]().future, f))
|
||||
Await.result(r, remaining) should be(5)
|
||||
Await.result(r, remainingOrDefault) should be(5)
|
||||
}
|
||||
|
||||
"be completed abnormally eventually" in {
|
||||
|
|
@ -64,7 +64,7 @@ class PatternSpec extends AkkaSpec("akka.actor.serialize-messages = off") {
|
|||
val f = akka.pattern.after(1 second, using = system.scheduler)(Promise.failed(new IllegalStateException("Mexico")).future)
|
||||
|
||||
val r = Future.firstCompletedOf(Seq(Promise[Int]().future, f))
|
||||
intercept[IllegalStateException] { Await.result(r, remaining) }.getMessage should be("Mexico")
|
||||
intercept[IllegalStateException] { Await.result(r, remainingOrDefault) }.getMessage should be("Mexico")
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -46,7 +46,7 @@ class BroadcastSpec extends AkkaSpec with DefaultTimeout with ImplicitSender {
|
|||
routedActor ! 1
|
||||
routedActor ! "end"
|
||||
|
||||
Await.ready(doneLatch, remaining)
|
||||
Await.ready(doneLatch, remainingOrDefault)
|
||||
|
||||
counter1.get should be(1)
|
||||
counter2.get should be(1)
|
||||
|
|
@ -78,7 +78,7 @@ class BroadcastSpec extends AkkaSpec with DefaultTimeout with ImplicitSender {
|
|||
routedActor ? 1
|
||||
routedActor ! "end"
|
||||
|
||||
Await.ready(doneLatch, remaining)
|
||||
Await.ready(doneLatch, remainingOrDefault)
|
||||
|
||||
counter1.get should be(1)
|
||||
counter2.get should be(1)
|
||||
|
|
|
|||
|
|
@ -55,7 +55,7 @@ class ConsistentHashingRouterSpec extends AkkaSpec(ConsistentHashingRouterSpec.c
|
|||
|
||||
"consistent hashing router" must {
|
||||
"create routees from configuration" in {
|
||||
val currentRoutees = Await.result(router1 ? GetRoutees, remaining).asInstanceOf[Routees]
|
||||
val currentRoutees = Await.result(router1 ? GetRoutees, timeout.duration).asInstanceOf[Routees]
|
||||
currentRoutees.routees.size should be(3)
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -48,7 +48,7 @@ class ResizerSpec extends AkkaSpec(ResizerSpec.config) with DefaultTimeout with
|
|||
}
|
||||
|
||||
def routeeSize(router: ActorRef): Int =
|
||||
Await.result(router ? GetRoutees, remaining).asInstanceOf[Routees].routees.size
|
||||
Await.result(router ? GetRoutees, timeout.duration).asInstanceOf[Routees].routees.size
|
||||
|
||||
"DefaultResizer" must {
|
||||
|
||||
|
|
@ -107,7 +107,7 @@ class ResizerSpec extends AkkaSpec(ResizerSpec.config) with DefaultTimeout with
|
|||
router ! latch
|
||||
router ! latch
|
||||
|
||||
Await.ready(latch, remaining)
|
||||
Await.ready(latch, remainingOrDefault)
|
||||
|
||||
// messagesPerResize is 10 so there is no risk of additional resize
|
||||
routeeSize(router) should be(2)
|
||||
|
|
@ -122,7 +122,7 @@ class ResizerSpec extends AkkaSpec(ResizerSpec.config) with DefaultTimeout with
|
|||
router ! latch
|
||||
router ! latch
|
||||
|
||||
Await.ready(latch, remaining)
|
||||
Await.ready(latch, remainingOrDefault)
|
||||
|
||||
routeeSize(router) should be(2)
|
||||
}
|
||||
|
|
|
|||
|
|
@ -18,7 +18,7 @@ import akka.actor.ActorRef
|
|||
class RoundRobinSpec extends AkkaSpec with DefaultTimeout with ImplicitSender {
|
||||
|
||||
def routeeSize(router: ActorRef): Int =
|
||||
Await.result(router ? GetRoutees, remaining).asInstanceOf[Routees].routees.size
|
||||
Await.result(router ? GetRoutees, timeout.duration).asInstanceOf[Routees].routees.size
|
||||
|
||||
"round robin pool" must {
|
||||
|
||||
|
|
|
|||
|
|
@ -96,7 +96,7 @@ class RoutingSpec extends AkkaSpec(RoutingSpec.config) with DefaultTimeout with
|
|||
val router = system.actorOf(RoundRobinPool(nrOfInstances = 0, resizer = Some(resizer)).props(
|
||||
routeeProps = Props[TestActor]))
|
||||
watch(router)
|
||||
Await.ready(latch, remaining)
|
||||
Await.ready(latch, remainingOrDefault)
|
||||
router ! GetRoutees
|
||||
val routees = expectMsgType[Routees].routees
|
||||
routees.size should be(2)
|
||||
|
|
@ -132,7 +132,7 @@ class RoutingSpec extends AkkaSpec(RoutingSpec.config) with DefaultTimeout with
|
|||
}
|
||||
val router = system.actorOf(RoundRobinPool(nrOfInstances = 0, resizer = Some(resizer)).props(
|
||||
routeeProps = Props[TestActor]), "router3")
|
||||
Await.ready(latch, remaining)
|
||||
Await.ready(latch, remainingOrDefault)
|
||||
router ! GetRoutees
|
||||
expectMsgType[Routees].routees.size should be(3)
|
||||
system.stop(router)
|
||||
|
|
|
|||
|
|
@ -27,7 +27,7 @@ class ActivationTrackerTest extends TestKit(ActorSystem("test")) with WordSpecLi
|
|||
val at = system.actorOf(Props[ActivationTracker], name = "activationTrackker")
|
||||
"ActivationTracker" must {
|
||||
def publish(msg: Any) = at ! msg
|
||||
implicit def timeout = remaining
|
||||
implicit def timeout = remainingOrDefault
|
||||
"forwards activation message to all awaiting parties" taggedAs TimingTest in {
|
||||
awaiting.awaitActivation()
|
||||
anotherAwaiting.awaitActivation()
|
||||
|
|
|
|||
|
|
@ -236,7 +236,7 @@ abstract class ClusterDeathWatchSpec
|
|||
|
||||
enterBarrier("first-unavailable")
|
||||
|
||||
val timeout = remaining
|
||||
val timeout = remainingOrDefault
|
||||
try system.awaitTermination(timeout) catch {
|
||||
case _: TimeoutException ⇒
|
||||
fail("Failed to stop [%s] within [%s] \n%s".format(system.name, timeout,
|
||||
|
|
|
|||
|
|
@ -215,7 +215,7 @@ trait MultiNodeClusterSpec extends Suite with STMultiNodeSpec with WatchedByCoro
|
|||
* Join the specific node within the given period by sending repeated join
|
||||
* requests at periodic intervals until we succeed.
|
||||
*/
|
||||
def joinWithin(joinNode: RoleName, max: Duration = remaining, interval: Duration = 1.second): Unit = {
|
||||
def joinWithin(joinNode: RoleName, max: Duration = remainingOrDefault, interval: Duration = 1.second): Unit = {
|
||||
def memberInState(member: Address, status: Seq[MemberStatus]): Boolean =
|
||||
clusterView.members.exists { m ⇒ (m.address == member) && status.contains(m.status) }
|
||||
|
||||
|
|
|
|||
|
|
@ -100,7 +100,7 @@ abstract class RestartFirstSeedNodeSpec
|
|||
|
||||
// shutdown seed1System
|
||||
runOn(seed1) {
|
||||
shutdown(seed1System, remaining)
|
||||
shutdown(seed1System, remainingOrDefault)
|
||||
}
|
||||
runOn(seed2, seed3) {
|
||||
awaitMembersUp(2, canNotBePartOfMemberRing = Set(seedNodes.head))
|
||||
|
|
|
|||
|
|
@ -802,7 +802,7 @@ abstract class StressSpec
|
|||
clusterResultAggregator match {
|
||||
case Some(r) ⇒
|
||||
watch(r)
|
||||
expectMsgPF(remaining) { case Terminated(a) if a.path == r.path ⇒ true }
|
||||
expectMsgPF() { case Terminated(a) if a.path == r.path ⇒ true }
|
||||
case None ⇒ // ok, already terminated
|
||||
}
|
||||
}
|
||||
|
|
@ -829,7 +829,7 @@ abstract class StressSpec
|
|||
runOn(currentRoles.last) {
|
||||
cluster.join(roles.head)
|
||||
}
|
||||
awaitMembersUp(currentRoles.size, timeout = remaining)
|
||||
awaitMembersUp(currentRoles.size, timeout = remainingOrDefault)
|
||||
}
|
||||
|
||||
}
|
||||
|
|
@ -849,7 +849,7 @@ abstract class StressSpec
|
|||
if (toSeedNodes) cluster.joinSeedNodes(seedNodes.toIndexedSeq map address)
|
||||
else cluster.join(roles.head)
|
||||
}
|
||||
awaitMembersUp(currentRoles.size, timeout = remaining)
|
||||
awaitMembersUp(currentRoles.size, timeout = remainingOrDefault)
|
||||
}
|
||||
|
||||
}
|
||||
|
|
@ -892,14 +892,14 @@ abstract class StressSpec
|
|||
testConductor.exit(removeRole, 0).await
|
||||
}
|
||||
}
|
||||
awaitMembersUp(currentRoles.size, timeout = remaining)
|
||||
awaitMembersUp(currentRoles.size, timeout = remainingOrDefault)
|
||||
awaitAllReachable()
|
||||
}
|
||||
}
|
||||
|
||||
runOn(roles.head) {
|
||||
val expectedPath = RootActorPath(removeAddress) / "user" / "watchee"
|
||||
expectMsgPF(remaining) {
|
||||
expectMsgPF() {
|
||||
case Terminated(a) if a.path == expectedPath ⇒ true
|
||||
}
|
||||
}
|
||||
|
|
@ -927,7 +927,7 @@ abstract class StressSpec
|
|||
testConductor.exit(r, 0).await
|
||||
}
|
||||
}
|
||||
awaitMembersUp(currentRoles.size, timeout = remaining)
|
||||
awaitMembersUp(currentRoles.size, timeout = remainingOrDefault)
|
||||
awaitAllReachable()
|
||||
}
|
||||
}
|
||||
|
|
@ -978,7 +978,7 @@ abstract class StressSpec
|
|||
awaitMembersUp(
|
||||
nbrUsedRoles + activeRoles.size,
|
||||
canNotBePartOfMemberRing = allPreviousAddresses,
|
||||
timeout = remaining)
|
||||
timeout = remainingOrDefault)
|
||||
awaitAllReachable()
|
||||
}
|
||||
val nextAddresses = clusterView.members.map(_.address) -- usedAddresses
|
||||
|
|
@ -1000,7 +1000,7 @@ abstract class StressSpec
|
|||
loop(1, None, Set.empty) foreach { as ⇒ TestKit.shutdownActorSystem(as) }
|
||||
within(loopDuration) {
|
||||
runOn(usedRoles: _*) {
|
||||
awaitMembersUp(nbrUsedRoles, timeout = remaining)
|
||||
awaitMembersUp(nbrUsedRoles, timeout = remainingOrDefault)
|
||||
awaitAllReachable()
|
||||
phiObserver ! Reset
|
||||
statsObserver ! Reset
|
||||
|
|
@ -1142,7 +1142,7 @@ abstract class StressSpec
|
|||
runOn((seedNodes ++ otherNodesJoiningSeedNodes): _*) {
|
||||
reportResult {
|
||||
cluster.joinSeedNodes(seedNodes.toIndexedSeq map address)
|
||||
awaitMembersUp(size, timeout = remaining)
|
||||
awaitMembersUp(size, timeout = remainingOrDefault)
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -94,7 +94,7 @@ abstract class AdaptiveLoadBalancingRouterSpec extends MultiNodeSpec(AdaptiveLoa
|
|||
import AdaptiveLoadBalancingRouterMultiJvmSpec._
|
||||
|
||||
def currentRoutees(router: ActorRef) =
|
||||
Await.result(router ? CurrentRoutees, remaining).asInstanceOf[RouterRoutees].routees
|
||||
Await.result(router ? CurrentRoutees, timeout.duration).asInstanceOf[RouterRoutees].routees
|
||||
|
||||
def receiveReplies(expectedReplies: Int): Map[Address, Int] = {
|
||||
val zero = Map.empty[Address, Int] ++ roles.map(address(_) -> 0)
|
||||
|
|
|
|||
|
|
@ -70,7 +70,7 @@ abstract class ClusterConsistentHashingRouterSpec extends MultiNodeSpec(ClusterC
|
|||
lazy val router1 = system.actorOf(Props[Echo].withRouter(FromConfig()), "router1")
|
||||
|
||||
def currentRoutees(router: ActorRef) =
|
||||
Await.result(router ? CurrentRoutees, remaining).asInstanceOf[RouterRoutees].routees
|
||||
Await.result(router ? CurrentRoutees, timeout.duration).asInstanceOf[RouterRoutees].routees
|
||||
|
||||
/**
|
||||
* Fills in self address for local ActorRef
|
||||
|
|
|
|||
|
|
@ -127,7 +127,7 @@ abstract class ClusterRoundRobinRoutedActorSpec extends MultiNodeSpec(ClusterRou
|
|||
}
|
||||
|
||||
def currentRoutees(router: ActorRef) =
|
||||
Await.result(router ? CurrentRoutees, remaining).asInstanceOf[RouterRoutees].routees
|
||||
Await.result(router ? CurrentRoutees, timeout.duration).asInstanceOf[RouterRoutees].routees
|
||||
|
||||
"A cluster router with a RoundRobin router" must {
|
||||
"start cluster with 2 nodes" taggedAs LongRunningTest in {
|
||||
|
|
|
|||
|
|
@ -94,7 +94,7 @@ abstract class AdaptiveLoadBalancingRouterSpec extends MultiNodeSpec(AdaptiveLoa
|
|||
import AdaptiveLoadBalancingRouterMultiJvmSpec._
|
||||
|
||||
def currentRoutees(router: ActorRef) =
|
||||
Await.result(router ? GetRoutees, remaining).asInstanceOf[Routees].routees
|
||||
Await.result(router ? GetRoutees, timeout.duration).asInstanceOf[Routees].routees
|
||||
|
||||
def receiveReplies(expectedReplies: Int): Map[Address, Int] = {
|
||||
val zero = Map.empty[Address, Int] ++ roles.map(address(_) -> 0)
|
||||
|
|
|
|||
|
|
@ -59,7 +59,7 @@ abstract class ClusterConsistentHashingGroupSpec extends MultiNodeSpec(ClusterCo
|
|||
}
|
||||
|
||||
def currentRoutees(router: ActorRef) =
|
||||
Await.result(router ? GetRoutees, remaining).asInstanceOf[Routees].routees
|
||||
Await.result(router ? GetRoutees, timeout.duration).asInstanceOf[Routees].routees
|
||||
|
||||
"A cluster router with a consistent hashing group" must {
|
||||
"start cluster with 3 nodes" taggedAs LongRunningTest in {
|
||||
|
|
|
|||
|
|
@ -70,7 +70,7 @@ abstract class ClusterConsistentHashingRouterSpec extends MultiNodeSpec(ClusterC
|
|||
lazy val router1 = system.actorOf(FromConfig.props(Props[Echo]), "router1")
|
||||
|
||||
def currentRoutees(router: ActorRef) =
|
||||
Await.result(router ? GetRoutees, remaining).asInstanceOf[Routees].routees
|
||||
Await.result(router ? GetRoutees, timeout.duration).asInstanceOf[Routees].routees
|
||||
|
||||
/**
|
||||
* Fills in self address for local ActorRef
|
||||
|
|
|
|||
|
|
@ -130,7 +130,7 @@ abstract class ClusterRoundRobinSpec extends MultiNodeSpec(ClusterRoundRobinMult
|
|||
}
|
||||
|
||||
def currentRoutees(router: ActorRef) =
|
||||
Await.result(router ? GetRoutees, remaining).asInstanceOf[Routees].routees
|
||||
Await.result(router ? GetRoutees, timeout.duration).asInstanceOf[Routees].routees
|
||||
|
||||
"A cluster router with a RoundRobin router" must {
|
||||
"start cluster with 2 nodes" taggedAs LongRunningTest in {
|
||||
|
|
|
|||
20
akka-docs/rst/project/migration-guide-2.3.x-2.4.x.rst
Normal file
20
akka-docs/rst/project/migration-guide-2.3.x-2.4.x.rst
Normal file
|
|
@ -0,0 +1,20 @@
|
|||
.. _migration-2.4:
|
||||
|
||||
################################
|
||||
Migration Guide 2.3.x to 2.4.x
|
||||
################################
|
||||
|
||||
The 2.4 release contains some structural changes that require some
|
||||
simple, mechanical source-level changes in client code.
|
||||
|
||||
When migrating from earlier versions you should first follow the instructions for
|
||||
migrating :ref:`1.3.x to 2.0.x <migration-2.0>` and then :ref:`2.0.x to 2.1.x <migration-2.1>`
|
||||
and then :ref:`2.1.x to 2.2.x <migration-2.2>` and then :ref:`2.2.x to 2.3.x <migration-2.3>`.
|
||||
|
||||
TestKit.remaining throws AssertionError
|
||||
=======================================
|
||||
|
||||
In earlier versions of Akka `TestKit.remaining` returned the default timeout configurable under
|
||||
"akka.test.single-expect-default". This was a bit confusing and thus it has been changed to throw an
|
||||
AssertionError if called outside of within. The old behavior however can still be achieved by
|
||||
calling `TestKit.remainingOrDefault` instead.
|
||||
|
|
@ -10,4 +10,5 @@ Migration Guides
|
|||
migration-guide-2.0.x-2.1.x
|
||||
migration-guide-2.1.x-2.2.x
|
||||
migration-guide-2.2.x-2.3.x
|
||||
migration-guide-2.3.x-2.4.x
|
||||
migration-guide-eventsourced-2.3.x
|
||||
|
|
|
|||
|
|
@ -96,7 +96,7 @@ class NewRemoteActorSpec extends MultiNodeSpec(NewRemoteActorMultiJvmSpec)
|
|||
|
||||
// master system is supposed to be shutdown after slave
|
||||
// this should be triggered by slave system shutdown
|
||||
expectMsgPF(remaining) { case Terminated(`actor`) ⇒ true }
|
||||
expectMsgPF() { case Terminated(`actor`) ⇒ true }
|
||||
}
|
||||
|
||||
runOn(slave) {
|
||||
|
|
|
|||
|
|
@ -76,7 +76,7 @@ abstract class RemoteDeploymentDeathWatchSpec
|
|||
|
||||
sleep()
|
||||
// if the remote deployed actor is not removed the system will not shutdown
|
||||
val timeout = remaining
|
||||
val timeout = remainingOrDefault
|
||||
try system.awaitTermination(timeout) catch {
|
||||
case _: TimeoutException ⇒
|
||||
fail("Failed to stop [%s] within [%s] \n%s".format(system.name, timeout,
|
||||
|
|
|
|||
|
|
@ -124,7 +124,7 @@ class RoundRobinRoutedRemoteActorSpec extends MultiNodeSpec(RoundRobinRoutedRemo
|
|||
val repliesFrom: Set[ActorRef] =
|
||||
(for (n ← 2 to 8) yield {
|
||||
actor ! "hit"
|
||||
awaitCond(Await.result(actor ? CurrentRoutees, remaining).asInstanceOf[RouterRoutees].routees.size == n)
|
||||
awaitCond(Await.result(actor ? CurrentRoutees, timeout.duration).asInstanceOf[RouterRoutees].routees.size == n)
|
||||
expectMsgType[ActorRef]
|
||||
}).toSet
|
||||
|
||||
|
|
|
|||
|
|
@ -141,7 +141,7 @@ class RemoteRoundRobinSpec extends MultiNodeSpec(RemoteRoundRobinMultiJvmSpec)
|
|||
(for (n ← 3 to 9) yield {
|
||||
// each message trigger a resize, incrementing number of routees with 1
|
||||
actor ! "hit"
|
||||
Await.result(actor ? GetRoutees, remaining).asInstanceOf[Routees].routees.size should be(n)
|
||||
Await.result(actor ? GetRoutees, timeout.duration).asInstanceOf[Routees].routees.size should be(n)
|
||||
expectMsgType[ActorRef]
|
||||
}).toSet
|
||||
|
||||
|
|
|
|||
|
|
@ -448,7 +448,7 @@ class RemotingSpec extends AkkaSpec(RemotingSpec.cfg) with ImplicitSender with D
|
|||
"not fail ask across node boundaries" in within(5.seconds) {
|
||||
import system.dispatcher
|
||||
val f = for (_ ← 1 to 1000) yield here ? "ping" mapTo manifest[(String, ActorRef)]
|
||||
Await.result(Future.sequence(f), remaining).map(_._1).toSet should be(Set("pong"))
|
||||
Await.result(Future.sequence(f), timeout.duration).map(_._1).toSet should be(Set("pong"))
|
||||
}
|
||||
|
||||
"be able to use multiple transports and use the appropriate one (TCP)" in {
|
||||
|
|
|
|||
|
|
@ -371,7 +371,7 @@ public class JavaTestKit {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as <code>expectMsgEquals(remaining(), obj)</code>, but correctly
|
||||
* Same as <code>expectMsgEquals(remainingOrDefault(), obj)</code>, but correctly
|
||||
* treating the timeFactor.
|
||||
*/
|
||||
public <T> T expectMsgEquals(T msg) {
|
||||
|
|
@ -390,7 +390,7 @@ public class JavaTestKit {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as <code>expectMsgClass(remaining(), clazz)</code>, but correctly
|
||||
* Same as <code>expectMsgClass(remainingOrDefault(), clazz)</code>, but correctly
|
||||
* treating the timeFactor.
|
||||
*/
|
||||
public <T> T expectMsgClass(Class<T> clazz) {
|
||||
|
|
@ -409,7 +409,7 @@ public class JavaTestKit {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as <code>expectMsgAnyOf(remaining(), obj...)</code>, but correctly
|
||||
* Same as <code>expectMsgAnyOf(remainingOrDefault(), obj...)</code>, but correctly
|
||||
* treating the timeFactor.
|
||||
*/
|
||||
public Object expectMsgAnyOf(Object... msgs) {
|
||||
|
|
@ -428,7 +428,7 @@ public class JavaTestKit {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as <code>expectMsgAllOf(remaining(), obj...)</code>, but correctly
|
||||
* Same as <code>expectMsgAllOf(remainingOrDefault(), obj...)</code>, but correctly
|
||||
* treating the timeFactor.
|
||||
*/
|
||||
public Object[] expectMsgAllOf(Object... msgs) {
|
||||
|
|
@ -447,7 +447,7 @@ public class JavaTestKit {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as <code>expectMsgAnyClassOf(remaining(), obj...)</code>, but
|
||||
* Same as <code>expectMsgAnyClassOf(remainingOrDefault(), obj...)</code>, but
|
||||
* correctly treating the timeFactor.
|
||||
*/
|
||||
@SuppressWarnings("unchecked")
|
||||
|
|
@ -468,7 +468,7 @@ public class JavaTestKit {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as <code>expectNoMsg(remaining())</code>, but correctly treating the
|
||||
* Same as <code>expectNoMsg(remainingOrDefault())</code>, but correctly treating the
|
||||
* timeFactor.
|
||||
*/
|
||||
public void expectNoMsg() {
|
||||
|
|
@ -492,7 +492,7 @@ public class JavaTestKit {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as <code>expectTerminated(remaining(), target)</code>,
|
||||
* Same as <code>expectTerminated(remainingOrDefault(), target)</code>,
|
||||
* but correctly treating the timeFactor.
|
||||
* Don't forget to 'watch' it first!
|
||||
*/
|
||||
|
|
|
|||
|
|
@ -183,7 +183,17 @@ trait TestKitBase {
|
|||
* block or missing that it returns the properly dilated default for this
|
||||
* case from settings (key "akka.test.single-expect-default").
|
||||
*/
|
||||
def remaining: FiniteDuration = remainingOr(testKitSettings.SingleExpectDefaultTimeout.dilated)
|
||||
def remainingOrDefault = remainingOr(testKitSettings.SingleExpectDefaultTimeout.dilated)
|
||||
|
||||
/**
|
||||
* Obtain time remaining for execution of the innermost enclosing `within`
|
||||
* block or throw an [[AssertionError]] if no `within` block surrounds this
|
||||
* call.
|
||||
*/
|
||||
def remaining: FiniteDuration = end match {
|
||||
case f: FiniteDuration ⇒ f - now
|
||||
case _ ⇒ throw new AssertionError("`remaining` may not be called outside of `within`")
|
||||
}
|
||||
|
||||
/**
|
||||
* Obtain time remaining for execution of the innermost enclosing `within`
|
||||
|
|
@ -196,7 +206,7 @@ trait TestKitBase {
|
|||
}
|
||||
|
||||
private def remainingOrDilated(max: Duration): FiniteDuration = max match {
|
||||
case x if x eq Duration.Undefined ⇒ remaining
|
||||
case x if x eq Duration.Undefined ⇒ remainingOrDefault
|
||||
case x if !x.isFinite ⇒ throw new IllegalArgumentException("max duration cannot be infinite")
|
||||
case f: FiniteDuration ⇒ f.dilated
|
||||
}
|
||||
|
|
@ -309,9 +319,9 @@ trait TestKitBase {
|
|||
def within[T](max: FiniteDuration)(f: ⇒ T): T = within(0 seconds, max)(f)
|
||||
|
||||
/**
|
||||
* Same as `expectMsg(remaining, obj)`, but correctly treating the timeFactor.
|
||||
* Same as `expectMsg(remainingOrDefault, obj)`, but correctly treating the timeFactor.
|
||||
*/
|
||||
def expectMsg[T](obj: T): T = expectMsg_internal(remaining, obj)
|
||||
def expectMsg[T](obj: T): T = expectMsg_internal(remainingOrDefault, obj)
|
||||
|
||||
/**
|
||||
* Receive one message from the test actor and assert that it equals the
|
||||
|
|
@ -380,9 +390,9 @@ trait TestKitBase {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as `expectMsgType[T](remaining)`, but correctly treating the timeFactor.
|
||||
* Same as `expectMsgType[T](remainingOrDefault)`, but correctly treating the timeFactor.
|
||||
*/
|
||||
def expectMsgType[T](implicit t: ClassTag[T]): T = expectMsgClass_internal(remaining, t.runtimeClass.asInstanceOf[Class[T]])
|
||||
def expectMsgType[T](implicit t: ClassTag[T]): T = expectMsgClass_internal(remainingOrDefault, t.runtimeClass.asInstanceOf[Class[T]])
|
||||
|
||||
/**
|
||||
* Receive one message from the test actor and assert that it conforms to the
|
||||
|
|
@ -394,9 +404,9 @@ trait TestKitBase {
|
|||
def expectMsgType[T](max: FiniteDuration)(implicit t: ClassTag[T]): T = expectMsgClass_internal(max.dilated, t.runtimeClass.asInstanceOf[Class[T]])
|
||||
|
||||
/**
|
||||
* Same as `expectMsgClass(remaining, c)`, but correctly treating the timeFactor.
|
||||
* Same as `expectMsgClass(remainingOrDefault, c)`, but correctly treating the timeFactor.
|
||||
*/
|
||||
def expectMsgClass[C](c: Class[C]): C = expectMsgClass_internal(remaining, c)
|
||||
def expectMsgClass[C](c: Class[C]): C = expectMsgClass_internal(remainingOrDefault, c)
|
||||
|
||||
/**
|
||||
* Receive one message from the test actor and assert that it conforms to
|
||||
|
|
@ -415,9 +425,9 @@ trait TestKitBase {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as `expectMsgAnyOf(remaining, obj...)`, but correctly treating the timeFactor.
|
||||
* Same as `expectMsgAnyOf(remainingOrDefault, obj...)`, but correctly treating the timeFactor.
|
||||
*/
|
||||
def expectMsgAnyOf[T](obj: T*): T = expectMsgAnyOf_internal(remaining, obj: _*)
|
||||
def expectMsgAnyOf[T](obj: T*): T = expectMsgAnyOf_internal(remainingOrDefault, obj: _*)
|
||||
|
||||
/**
|
||||
* Receive one message from the test actor and assert that it equals one of
|
||||
|
|
@ -436,9 +446,9 @@ trait TestKitBase {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as `expectMsgAnyClassOf(remaining, obj...)`, but correctly treating the timeFactor.
|
||||
* Same as `expectMsgAnyClassOf(remainingOrDefault, obj...)`, but correctly treating the timeFactor.
|
||||
*/
|
||||
def expectMsgAnyClassOf[C](obj: Class[_ <: C]*): C = expectMsgAnyClassOf_internal(remaining, obj: _*)
|
||||
def expectMsgAnyClassOf[C](obj: Class[_ <: C]*): C = expectMsgAnyClassOf_internal(remainingOrDefault, obj: _*)
|
||||
|
||||
/**
|
||||
* Receive one message from the test actor and assert that it conforms to
|
||||
|
|
@ -457,9 +467,9 @@ trait TestKitBase {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as `expectMsgAllOf(remaining, obj...)`, but correctly treating the timeFactor.
|
||||
* Same as `expectMsgAllOf(remainingOrDefault, obj...)`, but correctly treating the timeFactor.
|
||||
*/
|
||||
def expectMsgAllOf[T](obj: T*): immutable.Seq[T] = expectMsgAllOf_internal(remaining, obj: _*)
|
||||
def expectMsgAllOf[T](obj: T*): immutable.Seq[T] = expectMsgAllOf_internal(remainingOrDefault, obj: _*)
|
||||
|
||||
/**
|
||||
* Receive a number of messages from the test actor matching the given
|
||||
|
|
@ -492,9 +502,9 @@ trait TestKitBase {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as `expectMsgAllClassOf(remaining, obj...)`, but correctly treating the timeFactor.
|
||||
* Same as `expectMsgAllClassOf(remainingOrDefault, obj...)`, but correctly treating the timeFactor.
|
||||
*/
|
||||
def expectMsgAllClassOf[T](obj: Class[_ <: T]*): immutable.Seq[T] = internalExpectMsgAllClassOf(remaining, obj: _*)
|
||||
def expectMsgAllClassOf[T](obj: Class[_ <: T]*): immutable.Seq[T] = internalExpectMsgAllClassOf(remainingOrDefault, obj: _*)
|
||||
|
||||
/**
|
||||
* Receive a number of messages from the test actor matching the given
|
||||
|
|
@ -515,9 +525,9 @@ trait TestKitBase {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as `expectMsgAllConformingOf(remaining, obj...)`, but correctly treating the timeFactor.
|
||||
* Same as `expectMsgAllConformingOf(remainingOrDefault, obj...)`, but correctly treating the timeFactor.
|
||||
*/
|
||||
def expectMsgAllConformingOf[T](obj: Class[_ <: T]*): immutable.Seq[T] = internalExpectMsgAllConformingOf(remaining, obj: _*)
|
||||
def expectMsgAllConformingOf[T](obj: Class[_ <: T]*): immutable.Seq[T] = internalExpectMsgAllConformingOf(remainingOrDefault, obj: _*)
|
||||
|
||||
/**
|
||||
* Receive a number of messages from the test actor matching the given
|
||||
|
|
@ -541,9 +551,9 @@ trait TestKitBase {
|
|||
}
|
||||
|
||||
/**
|
||||
* Same as `expectNoMsg(remaining)`, but correctly treating the timeFactor.
|
||||
* Same as `expectNoMsg(remainingOrDefault)`, but correctly treating the timeFactor.
|
||||
*/
|
||||
def expectNoMsg() { expectNoMsg_internal(remaining) }
|
||||
def expectNoMsg() { expectNoMsg_internal(remainingOrDefault) }
|
||||
|
||||
/**
|
||||
* Assert that no message is received for the specified time.
|
||||
|
|
@ -607,7 +617,7 @@ trait TestKitBase {
|
|||
* Same as `receiveN(n, remaining)` but correctly taking into account
|
||||
* Duration.timeFactor.
|
||||
*/
|
||||
def receiveN(n: Int): immutable.Seq[AnyRef] = receiveN_internal(n, remaining)
|
||||
def receiveN(n: Int): immutable.Seq[AnyRef] = receiveN_internal(n, remainingOrDefault)
|
||||
|
||||
/**
|
||||
* Receive N messages in a row before the given deadline.
|
||||
|
|
|
|||
|
|
@ -29,6 +29,11 @@ class TestTimeSpec extends AkkaSpec(Map("akka.test.timefactor" -> 2.0)) {
|
|||
}
|
||||
}
|
||||
|
||||
}
|
||||
"throw if `remaining` is called outside of `within`" in {
|
||||
intercept[AssertionError] {
|
||||
remaining
|
||||
}
|
||||
}
|
||||
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -86,7 +86,7 @@ class CoordinatedIncrementSpec extends AkkaSpec(CoordinatedIncrement.config) wit
|
|||
counters(0) ! coordinated(Increment(counters.tail))
|
||||
coordinated.await
|
||||
for (counter ← counters) {
|
||||
Await.result((counter ? GetCount).mapTo[Int], remaining) should be(1)
|
||||
Await.result((counter ? GetCount).mapTo[Int], timeout.duration) should be(1)
|
||||
}
|
||||
counters foreach (system.stop(_))
|
||||
system.stop(failer)
|
||||
|
|
@ -103,7 +103,7 @@ class CoordinatedIncrementSpec extends AkkaSpec(CoordinatedIncrement.config) wit
|
|||
counters(0) ! Coordinated(Increment(counters.tail :+ failer))
|
||||
coordinated.await
|
||||
for (counter ← counters) {
|
||||
Await.result(counter ? GetCount, remaining) should be(0)
|
||||
Await.result(counter ? GetCount, timeout.duration) should be(0)
|
||||
}
|
||||
counters foreach (system.stop(_))
|
||||
system.stop(failer)
|
||||
|
|
|
|||
Loading…
Add table
Add a link
Reference in a new issue