Compiler warnings for actor-tests and remote-tests (#26685)
This commit is contained in:
parent
4399e499c4
commit
10e525062a
93 changed files with 439 additions and 387 deletions
|
|
@ -58,7 +58,7 @@ object ActorCreationPerfSpec {
|
|||
case IsAlive =>
|
||||
sender() ! Alive
|
||||
case Create(number, propsCreator) =>
|
||||
for (i <- 1 to number) {
|
||||
for (_ <- 1 to number) {
|
||||
val start = System.nanoTime()
|
||||
context.actorOf(propsCreator.apply())
|
||||
// yes, we are aware of this being skewed
|
||||
|
|
@ -92,7 +92,7 @@ object ActorCreationPerfSpec {
|
|||
case IsAlive =>
|
||||
sender() ! Alive
|
||||
case Create(number, propsCreator) =>
|
||||
for (i <- 1 to number) {
|
||||
for (_ <- 1 to number) {
|
||||
context.actorOf(propsCreator.apply())
|
||||
}
|
||||
sender() ! Created
|
||||
|
|
|
|||
|
|
@ -5,14 +5,16 @@
|
|||
package akka.actor
|
||||
|
||||
import language.postfixOps
|
||||
|
||||
import akka.testkit.{ AkkaSpec, EventFilter }
|
||||
import akka.actor.ActorDSL._
|
||||
import akka.event.Logging.Warning
|
||||
|
||||
import scala.concurrent.{ Await, Future }
|
||||
import scala.concurrent.duration._
|
||||
import java.util.concurrent.TimeoutException
|
||||
|
||||
import akka.testkit.TimingTest
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
class ActorDSLDummy {
|
||||
//#import
|
||||
|
|
@ -22,6 +24,7 @@ class ActorDSLDummy {
|
|||
//#import
|
||||
}
|
||||
|
||||
@silent
|
||||
class ActorDSLSpec extends AkkaSpec {
|
||||
|
||||
val echo = system.actorOf(Props(new Actor {
|
||||
|
|
@ -88,13 +91,13 @@ class ActorDSLSpec extends AkkaSpec {
|
|||
system.eventStream.subscribe(testActor, classOf[Warning])
|
||||
try {
|
||||
for (_ <- 1 to 1000) i.receiver ! 0
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
EventFilter.warning(start = "dropping message", occurrences = 1).intercept {
|
||||
i.receiver ! 42
|
||||
}
|
||||
expectMsgType[Warning]
|
||||
i.receiver ! 42
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
val gotit = for (_ <- 1 to 1000) yield i.receive()
|
||||
gotit should ===((1 to 1000).map(_ => 0))
|
||||
intercept[TimeoutException] {
|
||||
|
|
@ -178,7 +181,7 @@ class ActorDSLSpec extends AkkaSpec {
|
|||
become {
|
||||
case "die" => throw new Exception
|
||||
}
|
||||
whenFailing { case m @ (cause, msg) => testActor ! m }
|
||||
whenFailing { case m @ (_, _) => testActor ! m }
|
||||
whenRestarted { cause =>
|
||||
testActor ! cause
|
||||
}
|
||||
|
|
@ -188,7 +191,7 @@ class ActorDSLSpec extends AkkaSpec {
|
|||
EventFilter[Exception](occurrences = 1).intercept {
|
||||
a ! "die"
|
||||
}
|
||||
expectMsgPF() { case (x: Exception, Some("die")) => }
|
||||
expectMsgPF() { case (_: Exception, Some("die")) => }
|
||||
expectMsgPF() { case _: Exception => }
|
||||
}
|
||||
|
||||
|
|
@ -217,7 +220,7 @@ class ActorDSLSpec extends AkkaSpec {
|
|||
EventFilter.warning("hi", occurrences = 1).intercept {
|
||||
a ! new Exception("hi")
|
||||
}
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
EventFilter[Exception]("hello", occurrences = 1).intercept {
|
||||
a ! new Exception("hello")
|
||||
}
|
||||
|
|
@ -247,7 +250,7 @@ class ActorDSLSpec extends AkkaSpec {
|
|||
become {
|
||||
case 1 => stash()
|
||||
case 2 =>
|
||||
testActor ! 2; unstashAll();
|
||||
testActor ! 2; unstashAll()
|
||||
becomeStacked {
|
||||
case 1 => testActor ! 1; unbecome()
|
||||
}
|
||||
|
|
|
|||
|
|
@ -68,7 +68,7 @@ class ActorLifeCycleSpec
|
|||
expectMsg(("OK", id, 3))
|
||||
restarter ! Kill
|
||||
expectMsg(("postStop", id, 3))
|
||||
expectNoMsg(1 seconds)
|
||||
expectNoMessage(1 seconds)
|
||||
system.stop(supervisor)
|
||||
}
|
||||
}
|
||||
|
|
@ -100,7 +100,7 @@ class ActorLifeCycleSpec
|
|||
expectMsg(("OK", id, 3))
|
||||
restarter ! Kill
|
||||
expectMsg(("postStop", id, 3))
|
||||
expectNoMsg(1 seconds)
|
||||
expectNoMessage(1 seconds)
|
||||
system.stop(supervisor)
|
||||
}
|
||||
}
|
||||
|
|
@ -117,7 +117,7 @@ class ActorLifeCycleSpec
|
|||
expectMsg(("OK", id, 0))
|
||||
system.stop(a)
|
||||
expectMsg(("postStop", id, 0))
|
||||
expectNoMsg(1 seconds)
|
||||
expectNoMessage(1 seconds)
|
||||
system.stop(supervisor)
|
||||
}
|
||||
|
||||
|
|
@ -136,14 +136,14 @@ class ActorLifeCycleSpec
|
|||
val a = system.actorOf(Props(new Actor {
|
||||
def receive = {
|
||||
case Become(beh) => { context.become(beh(context), discardOld = false); sender() ! "ok" }
|
||||
case x => sender() ! 42
|
||||
case _ => sender() ! 42
|
||||
}
|
||||
}))
|
||||
a ! "hello"
|
||||
expectMsg(42)
|
||||
a ! Become(ctx => {
|
||||
case "fail" => throw new RuntimeException("buh")
|
||||
case x => ctx.sender() ! 43
|
||||
case _ => ctx.sender() ! 43
|
||||
})
|
||||
expectMsg("ok")
|
||||
a ! "hello"
|
||||
|
|
|
|||
|
|
@ -5,11 +5,12 @@
|
|||
package akka.actor
|
||||
|
||||
import language.postfixOps
|
||||
|
||||
import akka.testkit._
|
||||
|
||||
import scala.concurrent.duration._
|
||||
import scala.concurrent.Await
|
||||
import akka.pattern.ask
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
object ActorLookupSpec {
|
||||
|
||||
|
|
@ -23,6 +24,7 @@ object ActorLookupSpec {
|
|||
|
||||
val p = Props[Node]
|
||||
|
||||
@silent
|
||||
class Node extends Actor {
|
||||
def receive = {
|
||||
case Create(name) => sender() ! context.actorOf(p, name)
|
||||
|
|
@ -35,6 +37,7 @@ object ActorLookupSpec {
|
|||
|
||||
}
|
||||
|
||||
@silent
|
||||
class ActorLookupSpec extends AkkaSpec with DefaultTimeout {
|
||||
import ActorLookupSpec._
|
||||
|
||||
|
|
@ -84,7 +87,7 @@ class ActorLookupSpec extends AkkaSpec with DefaultTimeout {
|
|||
expectTerminated(a1)
|
||||
|
||||
// let it be completely removed from user guardian
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
|
||||
// not equal because it's terminated
|
||||
system.actorFor(a1.path.toString) should not be (a1)
|
||||
|
|
|
|||
|
|
@ -7,10 +7,12 @@ package akka.actor
|
|||
import com.typesafe.config.ConfigFactory
|
||||
import akka.testkit._
|
||||
import akka.dispatch._
|
||||
|
||||
import scala.concurrent.duration.{ Duration, FiniteDuration }
|
||||
import akka.ConfigurationException
|
||||
import com.typesafe.config.Config
|
||||
import akka.util.Helpers.ConfigOps
|
||||
import akka.util.unused
|
||||
|
||||
object ActorMailboxSpec {
|
||||
val mailboxConf = ConfigFactory.parseString(s"""
|
||||
|
|
@ -184,7 +186,7 @@ object ActorMailboxSpec {
|
|||
|
||||
class StashQueueReportingActor extends QueueReportingActor with Stash
|
||||
|
||||
class StashQueueReportingActorWithParams(i: Int, s: String) extends StashQueueReportingActor
|
||||
class StashQueueReportingActorWithParams(@unused i: Int, @unused s: String) extends StashQueueReportingActor
|
||||
|
||||
val UnboundedMailboxTypes = Seq(classOf[UnboundedMessageQueueSemantics])
|
||||
val BoundedMailboxTypes = Seq(classOf[BoundedMessageQueueSemantics])
|
||||
|
|
@ -209,14 +211,14 @@ object ActorMailboxSpec {
|
|||
classOf[UnboundedControlAwareMessageQueueSemantics])
|
||||
|
||||
trait MCBoundedMessageQueueSemantics extends MessageQueue with MultipleConsumerSemantics
|
||||
final case class MCBoundedMailbox(val capacity: Int, val pushTimeOut: FiniteDuration)
|
||||
final case class MCBoundedMailbox(capacity: Int, pushTimeOut: FiniteDuration)
|
||||
extends MailboxType
|
||||
with ProducesMessageQueue[MCBoundedMessageQueueSemantics] {
|
||||
|
||||
def this(settings: ActorSystem.Settings, config: Config) =
|
||||
this(config.getInt("mailbox-capacity"), config.getNanosDuration("mailbox-push-timeout-time"))
|
||||
|
||||
final override def create(owner: Option[ActorRef], system: Option[ActorSystem]): MessageQueue =
|
||||
override def create(owner: Option[ActorRef], system: Option[ActorSystem]): MessageQueue =
|
||||
new BoundedMailbox.MessageQueue(capacity, pushTimeOut)
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -251,7 +251,6 @@ class ActorSelectionSpec extends AkkaSpec with DefaultTimeout {
|
|||
lookup should ===(result)
|
||||
}
|
||||
def check(looker: ActorRef): Unit = {
|
||||
val lookname = looker.path.elements.mkString("", "/", "/")
|
||||
for ((l, r) <- Seq(
|
||||
SelectString("a/b/c") -> None,
|
||||
SelectString("akka://all-systems/Nobody") -> None,
|
||||
|
|
@ -292,7 +291,7 @@ class ActorSelectionSpec extends AkkaSpec with DefaultTimeout {
|
|||
case `c1` => lastSender
|
||||
}
|
||||
actors should ===(Set(c1, c2))
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
}
|
||||
|
||||
"drop messages which cannot be delivered" in {
|
||||
|
|
@ -302,7 +301,7 @@ class ActorSelectionSpec extends AkkaSpec with DefaultTimeout {
|
|||
case `c2` => lastSender
|
||||
}
|
||||
actors should ===(Seq(c21))
|
||||
expectNoMsg(200.millis)
|
||||
expectNoMessage(200.millis)
|
||||
}
|
||||
|
||||
"resolve one actor with explicit timeout" in {
|
||||
|
|
@ -369,33 +368,33 @@ class ActorSelectionSpec extends AkkaSpec with DefaultTimeout {
|
|||
system.actorSelection("/user/a/*").tell(Identify(1), probe.ref)
|
||||
probe.receiveN(2).map { case ActorIdentity(1, r) => r }.toSet should ===(
|
||||
Set[Option[ActorRef]](Some(b1), Some(b2)))
|
||||
probe.expectNoMsg(200.millis)
|
||||
probe.expectNoMessage(200.millis)
|
||||
|
||||
system.actorSelection("/user/a/b1/*").tell(Identify(2), probe.ref)
|
||||
probe.expectMsg(ActorIdentity(2, None))
|
||||
|
||||
system.actorSelection("/user/a/*/c").tell(Identify(3), probe.ref)
|
||||
probe.expectMsg(ActorIdentity(3, Some(c)))
|
||||
probe.expectNoMsg(200.millis)
|
||||
probe.expectNoMessage(200.millis)
|
||||
|
||||
system.actorSelection("/user/a/b2/*/d").tell(Identify(4), probe.ref)
|
||||
probe.expectMsg(ActorIdentity(4, Some(d)))
|
||||
probe.expectNoMsg(200.millis)
|
||||
probe.expectNoMessage(200.millis)
|
||||
|
||||
system.actorSelection("/user/a/*/*/d").tell(Identify(5), probe.ref)
|
||||
probe.expectMsg(ActorIdentity(5, Some(d)))
|
||||
probe.expectNoMsg(200.millis)
|
||||
probe.expectNoMessage(200.millis)
|
||||
|
||||
system.actorSelection("/user/a/*/c/*").tell(Identify(6), probe.ref)
|
||||
probe.expectMsg(ActorIdentity(6, Some(d)))
|
||||
probe.expectNoMsg(200.millis)
|
||||
probe.expectNoMessage(200.millis)
|
||||
|
||||
system.actorSelection("/user/a/b2/*/d/e").tell(Identify(7), probe.ref)
|
||||
probe.expectMsg(ActorIdentity(7, None))
|
||||
probe.expectNoMsg(200.millis)
|
||||
probe.expectNoMessage(200.millis)
|
||||
|
||||
system.actorSelection("/user/a/*/c/d/e").tell(Identify(8), probe.ref)
|
||||
probe.expectNoMsg(500.millis)
|
||||
probe.expectNoMessage(500.millis)
|
||||
}
|
||||
|
||||
"forward to selection" in {
|
||||
|
|
|
|||
|
|
@ -15,7 +15,9 @@ import akka.testkit._
|
|||
import akka.testkit.TestKit
|
||||
import akka.util.Helpers.ConfigOps
|
||||
import akka.util.{ Switch, Timeout }
|
||||
import com.github.ghik.silencer.silent
|
||||
import com.typesafe.config.{ Config, ConfigFactory }
|
||||
|
||||
import scala.concurrent.duration._
|
||||
import scala.concurrent.{ Await, ExecutionContext, Future }
|
||||
import scala.language.postfixOps
|
||||
|
|
@ -30,7 +32,7 @@ object ActorSystemSpec {
|
|||
def receive = {
|
||||
case n: Int =>
|
||||
master = sender()
|
||||
terminaters = Set() ++ (for (i <- 1 to n) yield {
|
||||
terminaters = Set() ++ (for (_ <- 1 to n) yield {
|
||||
val man = context.watch(context.system.actorOf(Props[Terminater]))
|
||||
man ! "run"
|
||||
man
|
||||
|
|
@ -63,6 +65,7 @@ object ActorSystemSpec {
|
|||
}
|
||||
}
|
||||
|
||||
@silent
|
||||
final case class FastActor(latch: TestLatch, testActor: ActorRef) extends Actor {
|
||||
val ref1 = context.actorOf(Props.empty)
|
||||
val ref2 = context.actorFor(ref1.path.toString)
|
||||
|
|
@ -125,6 +128,7 @@ object ActorSystemSpec {
|
|||
|
||||
}
|
||||
|
||||
@silent
|
||||
class ActorSystemSpec extends AkkaSpec(ActorSystemSpec.config) with ImplicitSender {
|
||||
|
||||
import ActorSystemSpec.FastActor
|
||||
|
|
@ -283,7 +287,7 @@ class ActorSystemSpec extends AkkaSpec(ActorSystemSpec.config) with ImplicitSend
|
|||
"reliably create waves of actors" in {
|
||||
import system.dispatcher
|
||||
implicit val timeout = Timeout((20 seconds).dilated)
|
||||
val waves = for (i <- 1 to 3) yield system.actorOf(Props[ActorSystemSpec.Waves]) ? 50000
|
||||
val waves = for (_ <- 1 to 3) yield system.actorOf(Props[ActorSystemSpec.Waves]) ? 50000
|
||||
Await.result(Future.sequence(waves), timeout.duration + 5.seconds) should ===(Vector("done", "done", "done"))
|
||||
}
|
||||
|
||||
|
|
@ -406,7 +410,7 @@ class ActorSystemSpec extends AkkaSpec(ActorSystemSpec.config) with ImplicitSend
|
|||
|
||||
ref.tell("ping", probe.ref)
|
||||
|
||||
ecProbe.expectNoMsg(200.millis)
|
||||
ecProbe.expectNoMessage(200.millis)
|
||||
probe.expectMsg(1.second, "ping")
|
||||
} finally {
|
||||
shutdown(system2)
|
||||
|
|
|
|||
|
|
@ -5,13 +5,14 @@
|
|||
package akka.actor
|
||||
|
||||
import language.postfixOps
|
||||
|
||||
import akka.testkit._
|
||||
import akka.testkit.DefaultTimeout
|
||||
import akka.testkit.TestEvent._
|
||||
import akka.dispatch.BoundedDequeBasedMailbox
|
||||
|
||||
import scala.concurrent.duration._
|
||||
import akka.actor.ActorSystem.Settings
|
||||
import akka.util.unused
|
||||
import com.typesafe.config.{ Config, ConfigFactory }
|
||||
import org.scalatest.BeforeAndAfterEach
|
||||
|
||||
|
|
@ -55,9 +56,9 @@ object ActorWithBoundedStashSpec {
|
|||
}
|
||||
|
||||
// bounded deque-based mailbox with capacity 10
|
||||
class Bounded10(settings: Settings, config: Config) extends BoundedDequeBasedMailbox(10, 500 millis)
|
||||
class Bounded10(@unused settings: Settings, @unused config: Config) extends BoundedDequeBasedMailbox(10, 500 millis)
|
||||
|
||||
class Bounded100(settings: Settings, config: Config) extends BoundedDequeBasedMailbox(100, 500 millis)
|
||||
class Bounded100(@unused settings: Settings, @unused config: Config) extends BoundedDequeBasedMailbox(100, 500 millis)
|
||||
|
||||
val dispatcherId1 = "my-dispatcher-1"
|
||||
val dispatcherId2 = "my-dispatcher-2"
|
||||
|
|
|
|||
|
|
@ -5,15 +5,17 @@
|
|||
package akka.actor
|
||||
|
||||
import language.postfixOps
|
||||
|
||||
import akka.testkit._
|
||||
import akka.testkit.DefaultTimeout
|
||||
import akka.testkit.TestEvent._
|
||||
|
||||
import scala.concurrent.Await
|
||||
import akka.pattern.ask
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
import scala.concurrent.duration._
|
||||
import org.scalatest.BeforeAndAfterEach
|
||||
import org.scalatest.junit.JUnitSuiteLike
|
||||
import org.scalatestplus.junit.JUnitSuiteLike
|
||||
|
||||
object ActorWithStashSpec {
|
||||
|
||||
|
|
@ -31,7 +33,7 @@ object ActorWithStashSpec {
|
|||
state.s = "hello"
|
||||
unstashAll()
|
||||
context.become(greeted)
|
||||
case msg => stash()
|
||||
case _ => stash()
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -42,10 +44,10 @@ object ActorWithStashSpec {
|
|||
stash()
|
||||
stash()
|
||||
} catch {
|
||||
case e: IllegalStateException =>
|
||||
case _: IllegalStateException =>
|
||||
state.expectedException.open()
|
||||
}
|
||||
case msg => // do nothing
|
||||
case _ => // do nothing
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -59,10 +61,10 @@ object ActorWithStashSpec {
|
|||
case "close" =>
|
||||
unstashAll()
|
||||
context.unbecome()
|
||||
case msg => stash()
|
||||
case _ => stash()
|
||||
}
|
||||
case "done" => state.finished.await
|
||||
case msg => stash()
|
||||
case _ => stash()
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -102,6 +104,7 @@ object ActorWithStashSpec {
|
|||
|
||||
class JavaActorWithStashSpec extends StashJavaAPI with JUnitSuiteLike
|
||||
|
||||
@silent
|
||||
class ActorWithStashSpec extends AkkaSpec(ActorWithStashSpec.testConf) with DefaultTimeout with BeforeAndAfterEach {
|
||||
import ActorWithStashSpec._
|
||||
|
||||
|
|
@ -155,7 +158,7 @@ class ActorWithStashSpec extends AkkaSpec(ActorWithStashSpec.testConf) with Defa
|
|||
throw new Exception("Crashing...")
|
||||
|
||||
// when restartLatch is not yet open, stash all messages != "crash"
|
||||
case msg if !restartLatch.isOpen =>
|
||||
case _ if !restartLatch.isOpen =>
|
||||
stash()
|
||||
|
||||
// when restartLatch is open, must receive "hello"
|
||||
|
|
@ -193,7 +196,7 @@ class ActorWithStashSpec extends AkkaSpec(ActorWithStashSpec.testConf) with Defa
|
|||
become {
|
||||
case "die" => throw new RuntimeException("dying")
|
||||
}
|
||||
whenRestarted { thr =>
|
||||
whenRestarted { _ =>
|
||||
testActor ! "restarted"
|
||||
}
|
||||
})
|
||||
|
|
|
|||
|
|
@ -71,7 +71,7 @@ object Chameneos {
|
|||
case FADED => FADED
|
||||
}
|
||||
|
||||
override def toString = cid + "(" + colour + ")"
|
||||
override def toString = s"$cid($colour)"
|
||||
}
|
||||
|
||||
class Mall(var n: Int, numChameneos: Int) extends Actor {
|
||||
|
|
@ -92,7 +92,7 @@ object Chameneos {
|
|||
context.stop(self)
|
||||
}
|
||||
|
||||
case msg @ Meet(a, c) =>
|
||||
case msg: Meet =>
|
||||
if (n > 0) {
|
||||
waitingChameneo match {
|
||||
case Some(chameneo) =>
|
||||
|
|
@ -112,7 +112,7 @@ object Chameneos {
|
|||
// System.setProperty("akka.config", "akka.conf")
|
||||
Chameneos.start = System.currentTimeMillis
|
||||
val system = ActorSystem()
|
||||
val actor = system.actorOf(Props(new Mall(1000000, 4)))
|
||||
system.actorOf(Props(new Mall(1000000, 4)))
|
||||
Thread.sleep(10000)
|
||||
println("Elapsed: " + (end - start))
|
||||
system.terminate()
|
||||
|
|
|
|||
|
|
@ -38,7 +38,7 @@ object ConsistencySpec {
|
|||
if (lastStep != (step - 1))
|
||||
sender() ! "Test failed: Last step %s, this step %s".format(lastStep, step)
|
||||
|
||||
var shouldBeFortyTwo = left.value + right.value
|
||||
val shouldBeFortyTwo = left.value + right.value
|
||||
if (shouldBeFortyTwo != 42)
|
||||
sender() ! "Test failed: 42 failed"
|
||||
else {
|
||||
|
|
@ -69,7 +69,7 @@ class ConsistencySpec extends AkkaSpec(ConsistencySpec.config) {
|
|||
|
||||
for (a <- actors) { a.tell("done", testActor) }
|
||||
|
||||
for (a <- actors) expectMsg(5 minutes, "done")
|
||||
for (_ <- actors) expectMsg(5 minutes, "done")
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -246,7 +246,7 @@ class CoordinatedShutdownSpec
|
|||
intercept[TimeoutException] {
|
||||
Await.result(result, remainingOrDefault)
|
||||
}
|
||||
expectNoMsg(200.millis) // C not run
|
||||
expectNoMessage(200.millis) // C not run
|
||||
}
|
||||
|
||||
"skip tasks in disabled phase" in {
|
||||
|
|
|
|||
|
|
@ -42,14 +42,14 @@ class DeadLetterSupressionSpec extends AkkaSpec with ImplicitSender {
|
|||
deadActor ! NormalMsg
|
||||
|
||||
deadListener.expectMsg(DeadLetter(NormalMsg, testActor, deadActor))
|
||||
deadListener.expectNoMsg(200.millis)
|
||||
deadListener.expectNoMessage(200.millis)
|
||||
|
||||
suppressedListener.expectMsg(SuppressedDeadLetter(SuppressedMsg, testActor, system.deadLetters))
|
||||
suppressedListener.expectNoMsg(200.millis)
|
||||
suppressedListener.expectNoMessage(200.millis)
|
||||
|
||||
allListener.expectMsg(SuppressedDeadLetter(SuppressedMsg, testActor, system.deadLetters))
|
||||
allListener.expectMsg(DeadLetter(NormalMsg, testActor, deadActor))
|
||||
allListener.expectNoMsg(200.millis)
|
||||
allListener.expectNoMessage(200.millis)
|
||||
}
|
||||
|
||||
s"must suppress message from default dead-letters logging (sent to dead: ${Logging.simpleName(system.deadLetters)})" in {
|
||||
|
|
@ -73,8 +73,8 @@ class DeadLetterSupressionSpec extends AkkaSpec with ImplicitSender {
|
|||
allListener.expectMsg(200.millis, DeadLetter(NormalMsg, testActor, system.deadLetters))
|
||||
|
||||
Thread.sleep(200)
|
||||
deadListener.expectNoMsg(Duration.Zero)
|
||||
suppressedListener.expectNoMsg(Duration.Zero)
|
||||
allListener.expectNoMsg(Duration.Zero)
|
||||
deadListener.expectNoMessage(Duration.Zero)
|
||||
suppressedListener.expectNoMessage(Duration.Zero)
|
||||
allListener.expectNoMessage(Duration.Zero)
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -5,10 +5,13 @@
|
|||
package akka.actor
|
||||
|
||||
import akka.actor.Props.EmptyActor
|
||||
|
||||
import language.postfixOps
|
||||
import akka.dispatch.sysmsg.{ DeathWatchNotification, Failed }
|
||||
import akka.pattern.ask
|
||||
import akka.testkit._
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
import scala.concurrent.duration._
|
||||
import scala.concurrent.Await
|
||||
|
||||
|
|
@ -71,6 +74,7 @@ object DeathWatchSpec {
|
|||
final case class Latches(t1: TestLatch, t2: TestLatch) extends NoSerializationVerificationNeeded
|
||||
}
|
||||
|
||||
@silent
|
||||
trait DeathWatchSpec { this: AkkaSpec with ImplicitSender with DefaultTimeout =>
|
||||
|
||||
import DeathWatchSpec._
|
||||
|
|
@ -152,7 +156,7 @@ trait DeathWatchSpec { this: AkkaSpec with ImplicitSender with DefaultTimeout =>
|
|||
val terminalProps = TestActors.echoActorProps
|
||||
val terminal = Await.result((supervisor ? terminalProps).mapTo[ActorRef], timeout.duration)
|
||||
|
||||
val monitor = startWatching(terminal)
|
||||
startWatching(terminal)
|
||||
|
||||
terminal ! Kill
|
||||
terminal ! Kill
|
||||
|
|
@ -215,7 +219,7 @@ trait DeathWatchSpec { this: AkkaSpec with ImplicitSender with DefaultTimeout =>
|
|||
.sendSystemMessage(DeathWatchNotification(subject, existenceConfirmed = true, addressTerminated = false))
|
||||
|
||||
// the testActor is not watching subject and will not receive a Terminated msg
|
||||
expectNoMsg
|
||||
expectNoMessage
|
||||
}
|
||||
|
||||
"discard Terminated when unwatched between sysmsg and processing" in {
|
||||
|
|
|
|||
|
|
@ -10,7 +10,7 @@ import akka.testkit.EventFilter
|
|||
import akka.testkit.TestKit._
|
||||
import com.typesafe.config.ConfigFactory
|
||||
import org.scalatest.{ Matchers, WordSpec }
|
||||
import org.scalatest.junit.JUnitSuiteLike
|
||||
import org.scalatestplus.junit.JUnitSuiteLike
|
||||
|
||||
import scala.util.control.NoStackTrace
|
||||
|
||||
|
|
|
|||
|
|
@ -6,11 +6,13 @@ package akka.actor
|
|||
|
||||
import language.postfixOps
|
||||
import akka.testkit._
|
||||
|
||||
import scala.concurrent.duration._
|
||||
import akka.event._
|
||||
import com.typesafe.config.ConfigFactory
|
||||
|
||||
import scala.concurrent.Await
|
||||
import akka.util.Timeout
|
||||
import akka.util.{ unused, Timeout }
|
||||
|
||||
object FSMActorSpec {
|
||||
|
||||
|
|
@ -34,7 +36,6 @@ object FSMActorSpec {
|
|||
class Lock(code: String, timeout: FiniteDuration, latches: Latches) extends Actor with FSM[LockState, CodeState] {
|
||||
|
||||
import latches._
|
||||
import FSM.`->`
|
||||
|
||||
startWith(Locked, CodeState("", code))
|
||||
|
||||
|
|
@ -47,7 +48,7 @@ object FSMActorSpec {
|
|||
doUnlock()
|
||||
goto(Open).using(CodeState("", code)).forMax(timeout)
|
||||
}
|
||||
case wrong => {
|
||||
case _ => {
|
||||
stay.using(CodeState("", code))
|
||||
}
|
||||
}
|
||||
|
|
@ -78,7 +79,7 @@ object FSMActorSpec {
|
|||
// verify that old-style does still compile
|
||||
onTransition(transitionHandler _)
|
||||
|
||||
def transitionHandler(from: LockState, to: LockState) = {
|
||||
def transitionHandler(@unused from: LockState, @unused to: LockState) = {
|
||||
// dummy
|
||||
}
|
||||
|
||||
|
|
@ -101,7 +102,6 @@ object FSMActorSpec {
|
|||
|
||||
class FSMActorSpec extends AkkaSpec(Map("akka.actor.debug.fsm" -> true)) with ImplicitSender {
|
||||
import FSMActorSpec._
|
||||
import FSM.`->`
|
||||
|
||||
val timeout = Timeout(2 seconds)
|
||||
|
||||
|
|
@ -296,7 +296,7 @@ class FSMActorSpec extends AkkaSpec(Map("akka.actor.debug.fsm" -> true)) with Im
|
|||
true
|
||||
}
|
||||
expectMsgAllOf(1 second, Logging.Debug(name, fsmClass, "canceling timer 't'"), FSM.Normal)
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
system.eventStream.unsubscribe(testActor)
|
||||
}
|
||||
}
|
||||
|
|
@ -315,7 +315,6 @@ class FSMActorSpec extends AkkaSpec(Map("akka.actor.debug.fsm" -> true)) with Im
|
|||
}
|
||||
})
|
||||
fsmref ! "log"
|
||||
val fsm = fsmref.underlyingActor
|
||||
import FSM.LogEntry
|
||||
expectMsg(1 second, IndexedSeq(LogEntry(1, 0, "log")))
|
||||
fsmref ! "count"
|
||||
|
|
@ -333,7 +332,7 @@ class FSMActorSpec extends AkkaSpec(Map("akka.actor.debug.fsm" -> true)) with Im
|
|||
when(0)(transform {
|
||||
case Event("go", _) => stay
|
||||
}.using {
|
||||
case x => goto(1)
|
||||
case _ => goto(1)
|
||||
})
|
||||
when(1) {
|
||||
case _ => stay
|
||||
|
|
@ -373,7 +372,7 @@ class FSMActorSpec extends AkkaSpec(Map("akka.actor.debug.fsm" -> true)) with Im
|
|||
|
||||
fsm ! OverrideTimeoutToInf
|
||||
p.expectMsg(OverrideTimeoutToInf)
|
||||
p.expectNoMsg(1.seconds)
|
||||
p.expectNoMessage(1.seconds)
|
||||
} finally {
|
||||
TestKit.shutdownActorSystem(sys)
|
||||
}
|
||||
|
|
|
|||
|
|
@ -56,7 +56,7 @@ class FSMTimingSpec extends AkkaSpec with ImplicitSender {
|
|||
// the timeout in state TestStateTimeout is 800 ms, then it will change to Initial
|
||||
within(400 millis) {
|
||||
fsm ! TestStateTimeoutOverride
|
||||
expectNoMsg
|
||||
expectNoMessage
|
||||
}
|
||||
within(1 second) {
|
||||
fsm ! Cancel
|
||||
|
|
@ -72,7 +72,7 @@ class FSMTimingSpec extends AkkaSpec with ImplicitSender {
|
|||
expectMsg(Tick)
|
||||
expectMsg(Transition(fsm, TestSingleTimer, Initial))
|
||||
}
|
||||
expectNoMsg
|
||||
expectNoMessage
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -86,7 +86,7 @@ class FSMTimingSpec extends AkkaSpec with ImplicitSender {
|
|||
expectMsg(Tock)
|
||||
expectMsg(Transition(fsm, TestSingleTimerResubmit, Initial))
|
||||
}
|
||||
expectNoMsg
|
||||
expectNoMessage
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -10,7 +10,6 @@ import scala.concurrent.duration._
|
|||
import scala.language.postfixOps
|
||||
|
||||
object FSMTransitionSpec {
|
||||
import FSM.`->`
|
||||
|
||||
class Supervisor extends Actor {
|
||||
def receive = { case _ => }
|
||||
|
|
@ -66,7 +65,6 @@ object FSMTransitionSpec {
|
|||
class FSMTransitionSpec extends AkkaSpec with ImplicitSender {
|
||||
|
||||
import FSMTransitionSpec._
|
||||
import FSM.`->`
|
||||
|
||||
"A FSM transition notifier" must {
|
||||
|
||||
|
|
@ -74,7 +72,7 @@ class FSMTransitionSpec extends AkkaSpec with ImplicitSender {
|
|||
val fsm = system.actorOf(Props(new SendAnyTransitionFSM(testActor)))
|
||||
expectMsg(0 -> 0) // caused by initialize(), OK.
|
||||
fsm ! "stay" // no transition event
|
||||
expectNoMsg(500.millis)
|
||||
expectNoMessage(500.millis)
|
||||
fsm ! "goto" // goto(current state)
|
||||
expectMsg(0 -> 0)
|
||||
}
|
||||
|
|
@ -102,7 +100,7 @@ class FSMTransitionSpec extends AkkaSpec with ImplicitSender {
|
|||
expectMsg(FSM.CurrentState(fsm, 0))
|
||||
akka.pattern.gracefulStop(forward, 5 seconds)
|
||||
fsm ! "tick"
|
||||
expectNoMsg()
|
||||
expectNoMessage()
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
@ -142,7 +140,7 @@ class FSMTransitionSpec extends AkkaSpec with ImplicitSender {
|
|||
fsm ! FSM.SubscribeTransitionCallBack(forward)
|
||||
expectMsg(FSM.CurrentState(fsm, 0))
|
||||
fsm ! "stay"
|
||||
expectNoMsg()
|
||||
expectNoMessage()
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -100,7 +100,7 @@ class FunctionRefSpec extends AkkaSpec with ImplicitSender {
|
|||
// needs to be something that fails when the deserialized form is not a FunctionRef
|
||||
// this relies upon serialize-messages during tests
|
||||
testActor ! DropForwarder(ref)
|
||||
expectNoMsg(1.second)
|
||||
expectNoMessage(1.second)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -25,7 +25,7 @@ class HotSwapSpec extends AkkaSpec with ImplicitSender {
|
|||
|
||||
"be able to become multiple times in its constructor" in {
|
||||
val a = system.actorOf(Props(new Becomer {
|
||||
for (i <- 1 to 4) context.become({ case always => sender() ! i + ":" + always })
|
||||
for (i <- 1 to 4) context.become({ case always => sender() ! s"$i:$always" })
|
||||
def receive = { case _ => sender() ! "FAILURE" }
|
||||
}))
|
||||
a ! "pigdog"
|
||||
|
|
@ -45,7 +45,7 @@ class HotSwapSpec extends AkkaSpec with ImplicitSender {
|
|||
|
||||
"be able to become, with stacking, multiple times in its constructor" in {
|
||||
val a = system.actorOf(Props(new Becomer {
|
||||
for (i <- 1 to 4) context.become({ case always => sender() ! i + ":" + always; context.unbecome() }, false)
|
||||
for (i <- 1 to 4) context.become({ case always => sender() ! s"$i:$always"; context.unbecome() }, false)
|
||||
def receive = { case _ => sender() ! "FAILURE" }
|
||||
}))
|
||||
a ! "pigdog"
|
||||
|
|
|
|||
|
|
@ -4,6 +4,6 @@
|
|||
|
||||
package akka.actor
|
||||
|
||||
import org.scalatest.junit.JUnitSuiteLike
|
||||
import org.scalatestplus.junit.JUnitSuiteLike
|
||||
|
||||
class JavaAPISpec extends JavaAPI with JUnitSuiteLike
|
||||
|
|
|
|||
|
|
@ -6,9 +6,12 @@ package akka.actor
|
|||
|
||||
import language.postfixOps
|
||||
import akka.testkit._
|
||||
|
||||
import scala.concurrent.Await
|
||||
import scala.concurrent.duration._
|
||||
import akka.util.Timeout
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
import scala.concurrent.Future
|
||||
import scala.util.Success
|
||||
import scala.util.Failure
|
||||
|
|
@ -30,6 +33,7 @@ object LocalActorRefProviderSpec {
|
|||
"""
|
||||
}
|
||||
|
||||
@silent
|
||||
class LocalActorRefProviderSpec extends AkkaSpec(LocalActorRefProviderSpec.config) {
|
||||
"An LocalActorRefProvider" must {
|
||||
|
||||
|
|
@ -131,12 +135,12 @@ class LocalActorRefProviderSpec extends AkkaSpec(LocalActorRefProviderSpec.confi
|
|||
for (i <- 0 until 100) {
|
||||
val address = "new-actor" + i
|
||||
implicit val timeout = Timeout(5 seconds)
|
||||
val actors = for (j <- 1 to 4)
|
||||
val actors = for (_ <- 1 to 4)
|
||||
yield Future(system.actorOf(Props(new Actor { def receive = { case _ => } }), address))
|
||||
val set = Set() ++ actors.map(a =>
|
||||
Await.ready(a, timeout.duration).value match {
|
||||
case Some(Success(a: ActorRef)) => 1
|
||||
case Some(Failure(ex: InvalidActorNameException)) => 2
|
||||
case Some(Success(_: ActorRef)) => 1
|
||||
case Some(Failure(_: InvalidActorNameException)) => 2
|
||||
case x => x
|
||||
})
|
||||
set should ===(Set[Any](1, 2))
|
||||
|
|
|
|||
|
|
@ -5,6 +5,8 @@
|
|||
package akka.actor
|
||||
|
||||
import akka.testkit.AkkaSpec
|
||||
import akka.util.unused
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
object PropsCreationSpec {
|
||||
|
||||
|
|
@ -12,11 +14,11 @@ object PropsCreationSpec {
|
|||
|
||||
final class B
|
||||
|
||||
class OneParamActor(blackhole: A) extends Actor {
|
||||
class OneParamActor(@unused blackhole: A) extends Actor {
|
||||
override def receive = Actor.emptyBehavior
|
||||
}
|
||||
|
||||
class TwoParamActor(blackhole1: A, blackhole2: B) extends Actor {
|
||||
class TwoParamActor(@unused blackhole1: A, @unused blackhole2: B) extends Actor {
|
||||
override def receive = Actor.emptyBehavior
|
||||
}
|
||||
|
||||
|
|
@ -51,6 +53,7 @@ class PropsCreationSpec extends AkkaSpec("akka.actor.serialize-creators = on") {
|
|||
|
||||
"Props Java API" must {
|
||||
"work with create(creator)" in {
|
||||
@silent
|
||||
val p = Props.create(OneParamActorCreator)
|
||||
system.actorOf(p)
|
||||
}
|
||||
|
|
|
|||
|
|
@ -5,17 +5,20 @@
|
|||
package akka.actor
|
||||
|
||||
import language.postfixOps
|
||||
|
||||
import java.lang.Thread.sleep
|
||||
|
||||
import scala.concurrent.Await
|
||||
import akka.testkit.TestEvent._
|
||||
import akka.testkit.EventFilter
|
||||
import akka.testkit.AkkaSpec
|
||||
import akka.testkit.DefaultTimeout
|
||||
import akka.testkit.TestLatch
|
||||
|
||||
import scala.concurrent.duration._
|
||||
import akka.pattern.ask
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
@silent
|
||||
class RestartStrategySpec extends AkkaSpec("akka.actor.serialize-messages = off") with DefaultTimeout {
|
||||
|
||||
override def atStartup: Unit = {
|
||||
|
|
@ -216,7 +219,7 @@ class RestartStrategySpec extends AkkaSpec("akka.actor.serialize-messages = off"
|
|||
override val supervisorStrategy = OneForOneStrategy(withinTimeRange = 1 second)(List(classOf[Throwable]))
|
||||
def receive = {
|
||||
case p: Props => sender() ! context.watch(context.actorOf(p))
|
||||
case t: Terminated => maxNoOfRestartsLatch.open()
|
||||
case _: Terminated => maxNoOfRestartsLatch.open()
|
||||
}
|
||||
}))
|
||||
|
||||
|
|
|
|||
|
|
@ -8,15 +8,19 @@ import language.postfixOps
|
|||
import java.io.Closeable
|
||||
import java.util.concurrent._
|
||||
import atomic.{ AtomicInteger, AtomicReference }
|
||||
|
||||
import scala.concurrent.{ Await, ExecutionContext, Future }
|
||||
import scala.concurrent.duration._
|
||||
import java.util.concurrent.ThreadLocalRandom
|
||||
|
||||
import scala.util.Try
|
||||
import scala.util.control.NonFatal
|
||||
import org.scalatest.BeforeAndAfterEach
|
||||
import com.typesafe.config.{ Config, ConfigFactory }
|
||||
import akka.pattern.ask
|
||||
import akka.testkit._
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
import scala.util.control.NoStackTrace
|
||||
|
||||
object SchedulerSpec {
|
||||
|
|
@ -56,7 +60,7 @@ trait SchedulerSpec extends BeforeAndAfterEach with DefaultTimeout with Implicit
|
|||
expectMsg(Tock)
|
||||
expectMsg(Tock)
|
||||
expectMsg(Tock)
|
||||
expectNoMsg(500 millis)
|
||||
expectNoMessage(500 millis)
|
||||
|
||||
collectCancellable(system.scheduler.schedule(0 milliseconds, 50 milliseconds)(tickActor2 ! Tick))
|
||||
|
||||
|
|
@ -64,7 +68,7 @@ trait SchedulerSpec extends BeforeAndAfterEach with DefaultTimeout with Implicit
|
|||
expectMsg(Tock)
|
||||
expectMsg(Tock)
|
||||
expectMsg(Tock)
|
||||
expectNoMsg(500 millis)
|
||||
expectNoMessage(500 millis)
|
||||
}
|
||||
|
||||
"stop continuous scheduling if the receiving actor has been terminated" taggedAs TimingTest in {
|
||||
|
|
@ -77,7 +81,7 @@ trait SchedulerSpec extends BeforeAndAfterEach with DefaultTimeout with Implicit
|
|||
// stop the actor and, hence, the continuous messaging from happening
|
||||
system.stop(actor)
|
||||
|
||||
expectNoMsg(500 millis)
|
||||
expectNoMessage(500 millis)
|
||||
}
|
||||
|
||||
"stop continuous scheduling if the task throws exception" taggedAs TimingTest in {
|
||||
|
|
@ -90,7 +94,7 @@ trait SchedulerSpec extends BeforeAndAfterEach with DefaultTimeout with Implicit
|
|||
expectMsg(1)
|
||||
expectMsg(2)
|
||||
expectMsg(3)
|
||||
expectNoMsg(500 millis)
|
||||
expectNoMessage(500 millis)
|
||||
}
|
||||
|
||||
"schedule once" taggedAs TimingTest in {
|
||||
|
|
@ -120,7 +124,7 @@ trait SchedulerSpec extends BeforeAndAfterEach with DefaultTimeout with Implicit
|
|||
"be cancellable" taggedAs TimingTest in {
|
||||
for (_ <- 1 to 10) system.scheduler.scheduleOnce(1 second, testActor, "fail").cancel()
|
||||
|
||||
expectNoMsg(2 seconds)
|
||||
expectNoMessage(2 seconds)
|
||||
}
|
||||
|
||||
"be cancellable during initial delay" taggedAs TimingTest in {
|
||||
|
|
@ -219,7 +223,7 @@ trait SchedulerSpec extends BeforeAndAfterEach with DefaultTimeout with Implicit
|
|||
}
|
||||
}))
|
||||
|
||||
(1 to 300).foreach { i =>
|
||||
(1 to 300).foreach { _ =>
|
||||
collectCancellable(system.scheduler.scheduleOnce(20 milliseconds, actor, Msg(System.nanoTime)))
|
||||
Thread.sleep(5)
|
||||
}
|
||||
|
|
@ -342,7 +346,7 @@ class LightArrayRevolverSchedulerSpec extends AkkaSpec(SchedulerSpec.testConfRev
|
|||
intercept[IllegalArgumentException] {
|
||||
system.scheduler.schedule(100.millis, maxDelay + tickDuration, testActor, "Too long")
|
||||
}
|
||||
expectNoMsg(1.second)
|
||||
expectNoMessage(1.second)
|
||||
}
|
||||
|
||||
"survive being stressed with cancellation" taggedAs TimingTest in {
|
||||
|
|
@ -377,7 +381,7 @@ class LightArrayRevolverSchedulerSpec extends AkkaSpec(SchedulerSpec.testConfRev
|
|||
for (k <- histogram.keys.toSeq.sorted) {
|
||||
system.log.info(f"${k * 100}%3d: ${histogram(k).size}")
|
||||
}
|
||||
expectNoMsg(1.second)
|
||||
expectNoMessage(1.second)
|
||||
}
|
||||
|
||||
"survive vicious enqueueing" taggedAs TimingTest in {
|
||||
|
|
@ -409,7 +413,7 @@ class LightArrayRevolverSchedulerSpec extends AkkaSpec(SchedulerSpec.testConfRev
|
|||
import driver._
|
||||
val start = step / 2
|
||||
(0 to 3).foreach(i => sched.scheduleOnce(start + step * i, testActor, "hello"))
|
||||
expectNoMsg(step)
|
||||
expectNoMessage(step)
|
||||
wakeUp(step)
|
||||
expectWait(step)
|
||||
wakeUp(step * 4 + step / 2)
|
||||
|
|
@ -439,7 +443,7 @@ class LightArrayRevolverSchedulerSpec extends AkkaSpec(SchedulerSpec.testConfRev
|
|||
import driver._
|
||||
val start = step / 2
|
||||
(0 to 3).foreach(i => sched.scheduleOnce(start + step * i, probe.ref, "hello"))
|
||||
probe.expectNoMsg(step)
|
||||
probe.expectNoMessage(step)
|
||||
wakeUp(step)
|
||||
expectWait(step)
|
||||
// the following are no for-comp to see which iteration fails
|
||||
|
|
@ -466,7 +470,7 @@ class LightArrayRevolverSchedulerSpec extends AkkaSpec(SchedulerSpec.testConfRev
|
|||
import driver._
|
||||
val start = step / 2
|
||||
(0 to 3).foreach(i => sched.scheduleOnce(start + step * i, testActor, "hello"))
|
||||
expectNoMsg(step)
|
||||
expectNoMessage(step)
|
||||
wakeUp(step)
|
||||
expectWait(step)
|
||||
// the following are no for-comp to see which iteration fails
|
||||
|
|
@ -502,7 +506,7 @@ class LightArrayRevolverSchedulerSpec extends AkkaSpec(SchedulerSpec.testConfRev
|
|||
|
||||
val nums = 0 until numEvents
|
||||
nums.foreach(i => sched.scheduleOnce(start + step * i, testActor, "hello-" + i))
|
||||
expectNoMsg(step)
|
||||
expectNoMessage(step)
|
||||
wakeUp(step)
|
||||
expectWait(step)
|
||||
|
||||
|
|
@ -549,6 +553,7 @@ class LightArrayRevolverSchedulerSpec extends AkkaSpec(SchedulerSpec.testConfRev
|
|||
def reportFailure(t: Throwable): Unit = { t.printStackTrace() }
|
||||
}
|
||||
|
||||
@silent
|
||||
def withScheduler(start: Long = 0L, _startTick: Int = 0, config: Config = ConfigFactory.empty)(
|
||||
thunk: (Scheduler with Closeable, Driver) => Unit): Unit = {
|
||||
import akka.actor.{ LightArrayRevolverScheduler => LARS }
|
||||
|
|
|
|||
|
|
@ -6,6 +6,7 @@ package akka.actor
|
|||
|
||||
import language.postfixOps
|
||||
import java.util.concurrent.{ CountDownLatch, TimeUnit }
|
||||
|
||||
import scala.concurrent.Await
|
||||
import scala.concurrent.duration._
|
||||
import scala.util.Random
|
||||
|
|
@ -20,14 +21,16 @@ import akka.testkit.{ filterEvents, filterException, TestDuration, TestLatch }
|
|||
import akka.testkit.TestEvent.Mute
|
||||
import java.util.concurrent.ConcurrentHashMap
|
||||
import java.lang.ref.WeakReference
|
||||
|
||||
import akka.event.Logging
|
||||
import java.util.concurrent.atomic.AtomicInteger
|
||||
import java.lang.System.identityHashCode
|
||||
|
||||
import akka.util.Helpers.ConfigOps
|
||||
import akka.testkit.LongRunningTest
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
object SupervisorHierarchySpec {
|
||||
import FSM.`->`
|
||||
|
||||
class FireWorkerException(msg: String) extends Exception(msg)
|
||||
|
||||
|
|
@ -220,7 +223,7 @@ object SupervisorHierarchySpec {
|
|||
throw f.copy(depth = f.depth - 1)
|
||||
}
|
||||
val prefix = orig match {
|
||||
case f: Failure => "applying "
|
||||
case _: Failure => "applying "
|
||||
case _ => "re-applying "
|
||||
}
|
||||
log :+= Event(prefix + f + " to " + sender(), identityHashCode(this))
|
||||
|
|
@ -312,7 +315,7 @@ object SupervisorHierarchySpec {
|
|||
} else {
|
||||
// WARNING: The Terminated that is logged by this is logged by check() above, too. It is not
|
||||
// an indication of duplicate Terminate messages
|
||||
log :+= Event(sender() + " terminated while pongOfDeath", identityHashCode(Hierarchy.this))
|
||||
log :+= Event(s"${sender()} terminated while pongOfDeath", identityHashCode(Hierarchy.this))
|
||||
}
|
||||
case Abort => abort("terminating")
|
||||
case PingOfDeath =>
|
||||
|
|
@ -596,6 +599,7 @@ object SupervisorHierarchySpec {
|
|||
when(Stopping, stateTimeout = 5.seconds.dilated) {
|
||||
case Event(PongOfDeath, _) => stay
|
||||
case Event(Terminated(r), _) if r == hierarchy =>
|
||||
@silent
|
||||
val undead = children.filterNot(_.isTerminated)
|
||||
if (undead.nonEmpty) {
|
||||
log.info("undead:\n" + undead.mkString("\n"))
|
||||
|
|
@ -637,7 +641,6 @@ object SupervisorHierarchySpec {
|
|||
case Event(GCcheck(weak), _) =>
|
||||
val next = weak.filter(_.get ne null)
|
||||
if (next.nonEmpty) {
|
||||
println(next.size + " left")
|
||||
context.system.scheduler.scheduleOnce(workSchedule, self, GCcheck(next))(context.dispatcher)
|
||||
System.gc()
|
||||
stay
|
||||
|
|
@ -759,7 +762,7 @@ class SupervisorHierarchySpec extends AkkaSpec(SupervisorHierarchySpec.config) w
|
|||
val manager = Await.result((boss ? managerProps).mapTo[ActorRef], timeout.duration)
|
||||
|
||||
val workerProps = Props(new CountDownActor(countDown, SupervisorStrategy.defaultStrategy))
|
||||
val workerOne, workerTwo, workerThree = Await.result((manager ? workerProps).mapTo[ActorRef], timeout.duration)
|
||||
val workerOne = Await.result((manager ? workerProps).mapTo[ActorRef], timeout.duration)
|
||||
|
||||
filterException[ActorKilledException] {
|
||||
workerOne ! Kill
|
||||
|
|
@ -835,7 +838,7 @@ class SupervisorHierarchySpec extends AkkaSpec(SupervisorHierarchySpec.config) w
|
|||
boss ! "fail"
|
||||
awaitCond(worker.asInstanceOf[LocalActorRef].underlying.mailbox.isSuspended)
|
||||
worker ! "ping"
|
||||
expectNoMsg(2 seconds)
|
||||
expectNoMessage(2 seconds)
|
||||
latch.countDown()
|
||||
}
|
||||
expectMsg("pong")
|
||||
|
|
@ -856,7 +859,7 @@ class SupervisorHierarchySpec extends AkkaSpec(SupervisorHierarchySpec.config) w
|
|||
system.actorOf(
|
||||
Props(new Actor {
|
||||
override def supervisorStrategy = OneForOneStrategy() {
|
||||
case e: ActorInitializationException =>
|
||||
case _: ActorInitializationException =>
|
||||
if (createAttempt.get % 2 == 0) SupervisorStrategy.Resume else SupervisorStrategy.Restart
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -5,13 +5,16 @@
|
|||
package akka.actor
|
||||
|
||||
import language.postfixOps
|
||||
|
||||
import akka.testkit.{ filterEvents, EventFilter }
|
||||
|
||||
import scala.concurrent.Await
|
||||
import java.util.concurrent.{ CountDownLatch, TimeUnit }
|
||||
|
||||
import akka.testkit.AkkaSpec
|
||||
import akka.testkit.DefaultTimeout
|
||||
import akka.pattern.ask
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
import scala.concurrent.duration._
|
||||
import scala.util.control.NonFatal
|
||||
|
||||
|
|
@ -27,6 +30,7 @@ object SupervisorMiscSpec {
|
|||
"""
|
||||
}
|
||||
|
||||
@silent
|
||||
class SupervisorMiscSpec extends AkkaSpec(SupervisorMiscSpec.config) with DefaultTimeout {
|
||||
|
||||
"A Supervisor" must {
|
||||
|
|
@ -121,7 +125,7 @@ class SupervisorMiscSpec extends AkkaSpec(SupervisorMiscSpec.config) with Defaul
|
|||
context.actorOf(Props.empty, "foo")
|
||||
testActor ! "red"
|
||||
} catch {
|
||||
case e: InvalidActorNameException => testActor ! "green"
|
||||
case _: InvalidActorNameException => testActor ! "green"
|
||||
}
|
||||
}
|
||||
}))
|
||||
|
|
|
|||
|
|
@ -6,11 +6,13 @@ package akka.actor
|
|||
|
||||
import language.postfixOps
|
||||
import org.scalatest.BeforeAndAfterEach
|
||||
|
||||
import scala.concurrent.duration._
|
||||
import akka.{ Die, Ping }
|
||||
import akka.testkit.TestEvent._
|
||||
import akka.testkit._
|
||||
import java.util.concurrent.atomic.AtomicInteger
|
||||
|
||||
import scala.concurrent.Await
|
||||
import akka.pattern.ask
|
||||
import com.typesafe.config.ConfigFactory
|
||||
|
|
@ -19,6 +21,7 @@ import akka.dispatch.MessageQueue
|
|||
import com.typesafe.config.Config
|
||||
import akka.ConfigurationException
|
||||
import akka.routing.RoundRobinPool
|
||||
import akka.util.unused
|
||||
|
||||
object SupervisorSpec {
|
||||
val Timeout = 5.seconds
|
||||
|
|
@ -88,7 +91,7 @@ object SupervisorSpec {
|
|||
|
||||
val failure = new AssertionError("deliberate test failure")
|
||||
|
||||
class Mailbox(settings: ActorSystem.Settings, config: Config) extends MailboxType {
|
||||
class Mailbox(@unused settings: ActorSystem.Settings, @unused config: Config) extends MailboxType {
|
||||
override def create(owner: Option[ActorRef], system: Option[ActorSystem]): MessageQueue =
|
||||
throw failure
|
||||
}
|
||||
|
|
@ -193,14 +196,14 @@ class SupervisorSpec
|
|||
}
|
||||
|
||||
def kill(pingPongActor: ActorRef) = {
|
||||
val result = (pingPongActor.?(DieReply)(DilatedTimeout))
|
||||
val result = pingPongActor.?(DieReply)(DilatedTimeout)
|
||||
expectMsg(Timeout, ExceptionMessage) //this is sent from PingPongActor's postRestart()
|
||||
intercept[RuntimeException] { Await.result(result, DilatedTimeout) }
|
||||
}
|
||||
|
||||
def killExpectNoRestart(pingPongActor: ActorRef) = {
|
||||
val result = (pingPongActor.?(DieReply)(DilatedTimeout))
|
||||
expectNoMsg(500 milliseconds)
|
||||
val result = pingPongActor.?(DieReply)(DilatedTimeout)
|
||||
expectNoMessage(500 milliseconds)
|
||||
intercept[RuntimeException] { Await.result(result, DilatedTimeout) }
|
||||
}
|
||||
|
||||
|
|
@ -211,7 +214,7 @@ class SupervisorSpec
|
|||
|
||||
master ! Die
|
||||
expectMsg(3 seconds, "terminated")
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
}
|
||||
|
||||
"restart properly when same instance is returned" in {
|
||||
|
|
@ -263,7 +266,7 @@ class SupervisorSpec
|
|||
expectMsg("postStop1")
|
||||
}
|
||||
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
}
|
||||
|
||||
"not restart temporary actor" in {
|
||||
|
|
@ -271,13 +274,13 @@ class SupervisorSpec
|
|||
|
||||
intercept[RuntimeException] { Await.result(temporaryActor.?(DieReply)(DilatedTimeout), DilatedTimeout) }
|
||||
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
}
|
||||
|
||||
"start server for nested supervisor hierarchy" in {
|
||||
val (actor1, _, _, _) = nestedSupervisorsAllForOne
|
||||
ping(actor1)
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
}
|
||||
|
||||
"kill single actor OneForOne" in {
|
||||
|
|
@ -286,36 +289,36 @@ class SupervisorSpec
|
|||
}
|
||||
|
||||
"call-kill-call single actor OneForOne" in {
|
||||
val (actor, supervisor) = singleActorOneForOne
|
||||
val (actor, _) = singleActorOneForOne
|
||||
ping(actor)
|
||||
kill(actor)
|
||||
ping(actor)
|
||||
}
|
||||
|
||||
"kill single actor AllForOne" in {
|
||||
val (actor, supervisor) = singleActorAllForOne
|
||||
val (actor, _) = singleActorAllForOne
|
||||
kill(actor)
|
||||
}
|
||||
|
||||
"call-kill-call single actor AllForOne" in {
|
||||
val (actor, supervisor) = singleActorAllForOne
|
||||
val (actor, _) = singleActorAllForOne
|
||||
ping(actor)
|
||||
kill(actor)
|
||||
ping(actor)
|
||||
}
|
||||
|
||||
"kill multiple actors OneForOne 1" in {
|
||||
val (actor1, actor2, actor3, supervisor) = multipleActorsOneForOne
|
||||
val (actor1, _, _, _) = multipleActorsOneForOne
|
||||
kill(actor1)
|
||||
}
|
||||
|
||||
"kill multiple actors OneForOne 2" in {
|
||||
val (actor1, actor2, actor3, supervisor) = multipleActorsOneForOne
|
||||
val (_, _, actor3, _) = multipleActorsOneForOne
|
||||
kill(actor3)
|
||||
}
|
||||
|
||||
"call-kill-call multiple actors OneForOne" in {
|
||||
val (actor1, actor2, actor3, supervisor) = multipleActorsOneForOne
|
||||
val (actor1, actor2, actor3, _) = multipleActorsOneForOne
|
||||
|
||||
ping(actor1)
|
||||
ping(actor2)
|
||||
|
|
@ -329,7 +332,7 @@ class SupervisorSpec
|
|||
}
|
||||
|
||||
"kill multiple actors AllForOne" in {
|
||||
val (actor1, actor2, actor3, supervisor) = multipleActorsAllForOne
|
||||
val (_, actor2, _, _) = multipleActorsAllForOne
|
||||
|
||||
kill(actor2)
|
||||
|
||||
|
|
@ -339,7 +342,7 @@ class SupervisorSpec
|
|||
}
|
||||
|
||||
"call-kill-call multiple actors AllForOne" in {
|
||||
val (actor1, actor2, actor3, supervisor) = multipleActorsAllForOne
|
||||
val (actor1, actor2, actor3, _) = multipleActorsAllForOne
|
||||
|
||||
ping(actor1)
|
||||
ping(actor2)
|
||||
|
|
@ -457,7 +460,7 @@ class SupervisorSpec
|
|||
}
|
||||
|
||||
def receive = {
|
||||
case Terminated(a) if a.path == child.path => testActor ! "child terminated"
|
||||
case Terminated(t) if t.path == child.path => testActor ! "child terminated"
|
||||
case l: TestLatch => child ! l
|
||||
case "test" => sender() ! "green"
|
||||
case "testchild" => child.forward("test")
|
||||
|
|
@ -496,7 +499,7 @@ class SupervisorSpec
|
|||
val middle = expectMsgType[ActorRef]
|
||||
middle ! creator(testActor, fail = true)
|
||||
expectMsgPF(hint = "ConfigurationException") {
|
||||
case (top, middle, ex: ConfigurationException) =>
|
||||
case (_, _, ex: ConfigurationException) =>
|
||||
ex.getCause should ===(failure)
|
||||
}
|
||||
}
|
||||
|
|
@ -513,7 +516,7 @@ class SupervisorSpec
|
|||
val middle = expectMsgType[ActorRef]
|
||||
middle ! creator(testActor, fail = true).withRouter(RoundRobinPool(1))
|
||||
expectMsgPF(hint = "ConfigurationException") {
|
||||
case (top, middle, ex: ConfigurationException) =>
|
||||
case (_, _, ex: ConfigurationException) =>
|
||||
ex.getCause should ===(failure)
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -36,7 +36,7 @@ class SupervisorTreeSpec
|
|||
middleActor ! Kill
|
||||
expectMsg(middleActor.path)
|
||||
expectMsg(lastActor.path)
|
||||
expectNoMsg(2 seconds)
|
||||
expectNoMessage(2 seconds)
|
||||
system.stop(headActor)
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -53,7 +53,7 @@ class Ticket669Spec extends AkkaSpec with BeforeAndAfterAll with ImplicitSender
|
|||
object Ticket669Spec {
|
||||
class Supervised extends Actor {
|
||||
def receive = {
|
||||
case msg => throw new Exception("test")
|
||||
case _ => throw new Exception("test")
|
||||
}
|
||||
|
||||
override def preRestart(reason: scala.Throwable, msg: Option[Any]): Unit = {
|
||||
|
|
|
|||
|
|
@ -183,7 +183,7 @@ abstract class AbstractTimerSpec extends AkkaSpec {
|
|||
val ref = system.actorOf(target(probe.ref, 10.millis, repeat = false))
|
||||
|
||||
probe.expectMsg(Tock(1))
|
||||
probe.expectNoMsg(100.millis)
|
||||
probe.expectNoMessage(100.millis)
|
||||
|
||||
ref ! End
|
||||
probe.expectMsg(GotPostStop(false))
|
||||
|
|
@ -209,7 +209,7 @@ abstract class AbstractTimerSpec extends AkkaSpec {
|
|||
val latch = new TestLatch(1)
|
||||
// next Tock(1) enqueued in mailboxed, but should be discarded because of new timer
|
||||
ref ! SlowThenBump(latch)
|
||||
probe.expectNoMsg(interval + 100.millis)
|
||||
probe.expectNoMessage(interval + 100.millis)
|
||||
latch.countDown()
|
||||
probe.expectMsg(Tock(2))
|
||||
|
||||
|
|
@ -222,7 +222,7 @@ abstract class AbstractTimerSpec extends AkkaSpec {
|
|||
val ref = system.actorOf(target(probe.ref, dilatedInterval, repeat = true))
|
||||
probe.expectMsg(Tock(1))
|
||||
ref ! Cancel
|
||||
probe.expectNoMsg(dilatedInterval + 100.millis)
|
||||
probe.expectNoMessage(dilatedInterval + 100.millis)
|
||||
|
||||
ref ! End
|
||||
probe.expectMsg(GotPostStop(false))
|
||||
|
|
@ -248,10 +248,10 @@ abstract class AbstractTimerSpec extends AkkaSpec {
|
|||
val latch = new TestLatch(1)
|
||||
// next Tock(1) is enqueued in mailbox, but should be discarded by new incarnation
|
||||
ref ! SlowThenThrow(latch, new Exc)
|
||||
probe.expectNoMsg(interval + 100.millis)
|
||||
probe.expectNoMessage(interval + 100.millis)
|
||||
latch.countDown()
|
||||
probe.expectMsg(GotPreRestart(false))
|
||||
probe.expectNoMsg(interval / 2)
|
||||
probe.expectNoMessage(interval / 2)
|
||||
probe.expectMsg(Tock(2)) // this is from the startCounter increment
|
||||
|
||||
ref ! End
|
||||
|
|
@ -270,7 +270,7 @@ abstract class AbstractTimerSpec extends AkkaSpec {
|
|||
val latch = new TestLatch(1)
|
||||
// next Tock(2) is enqueued in mailbox, but should be discarded by new incarnation
|
||||
ref ! SlowThenThrow(latch, new Exc)
|
||||
probe.expectNoMsg(interval + 100.millis)
|
||||
probe.expectNoMessage(interval + 100.millis)
|
||||
latch.countDown()
|
||||
probe.expectMsg(GotPreRestart(false))
|
||||
probe.expectMsg(Tock(1))
|
||||
|
|
|
|||
|
|
@ -14,6 +14,7 @@ import akka.routing.RoundRobinGroup
|
|||
import akka.serialization.{ JavaSerializer, SerializerWithStringManifest }
|
||||
import akka.testkit.{ filterEvents, AkkaSpec, DefaultTimeout, EventFilter, TimingTest }
|
||||
import akka.util.Timeout
|
||||
import com.github.ghik.silencer.silent
|
||||
import org.scalatest.{ BeforeAndAfterAll, BeforeAndAfterEach }
|
||||
|
||||
import scala.annotation.tailrec
|
||||
|
|
@ -24,6 +25,7 @@ import scala.language.postfixOps
|
|||
|
||||
object TypedActorSpec {
|
||||
|
||||
@silent
|
||||
val config = """
|
||||
pooled-dispatcher {
|
||||
type = "akka.dispatch.BalancingDispatcherConfigurator"
|
||||
|
|
@ -109,7 +111,7 @@ object TypedActorSpec {
|
|||
def read(): Int
|
||||
|
||||
def testMethodCallSerialization(foo: Foo, s: String, i: Int, o: WithStringSerializedClass): Unit =
|
||||
throw new IllegalStateException("expected")
|
||||
throw new IllegalStateException(s"expected $foo $s $i $o")
|
||||
}
|
||||
|
||||
class Bar extends Foo with Serializable {
|
||||
|
|
@ -131,7 +133,6 @@ object TypedActorSpec {
|
|||
}
|
||||
|
||||
def futureComposePigdogFrom(foo: Foo): Future[String] = {
|
||||
implicit val timeout = TypedActor(TypedActor.context.system).DefaultReturnTimeout
|
||||
foo.futurePigdog(500 millis).map(_.toUpperCase)
|
||||
}
|
||||
|
||||
|
|
@ -188,19 +189,19 @@ object TypedActorSpec {
|
|||
|
||||
private def ensureContextAvailable[T](f: => T): T = TypedActor.context match {
|
||||
case null => throw new IllegalStateException("TypedActor.context is null!")
|
||||
case some => f
|
||||
case _ => f
|
||||
}
|
||||
|
||||
override def crash(): Unit = throw new IllegalStateException("Crash!")
|
||||
|
||||
override def preStart(): Unit = ensureContextAvailable(latch.countDown())
|
||||
|
||||
override def postStop(): Unit = ensureContextAvailable(for (i <- 1 to 3) latch.countDown())
|
||||
override def postStop(): Unit = ensureContextAvailable(for (_ <- 1 to 3) latch.countDown())
|
||||
|
||||
override def preRestart(reason: Throwable, message: Option[Any]): Unit =
|
||||
ensureContextAvailable(for (i <- 1 to 5) latch.countDown())
|
||||
ensureContextAvailable(for (_ <- 1 to 5) latch.countDown())
|
||||
|
||||
override def postRestart(reason: Throwable): Unit = ensureContextAvailable(for (i <- 1 to 7) latch.countDown())
|
||||
override def postRestart(reason: Throwable): Unit = ensureContextAvailable(for (_ <- 1 to 7) latch.countDown())
|
||||
|
||||
override def onReceive(msg: Any, sender: ActorRef): Unit = {
|
||||
ensureContextAvailable(msg match {
|
||||
|
|
@ -231,7 +232,7 @@ object TypedActorSpec {
|
|||
}
|
||||
|
||||
override def fromBinary(bytes: Array[Byte], manifest: String): AnyRef = manifest match {
|
||||
case manifest if bytes.length == 1 && bytes(0) == 255.toByte => WithStringSerializedClass()
|
||||
case _ if bytes.length == 1 && bytes(0) == 255.toByte => WithStringSerializedClass()
|
||||
case _ => throw new IllegalArgumentException(s"Cannot deserialize object with manifest $manifest")
|
||||
}
|
||||
}
|
||||
|
|
@ -282,9 +283,9 @@ class TypedActorSpec
|
|||
|
||||
"throw an IllegalStateException when TypedActor.self is called in the wrong scope" in {
|
||||
filterEvents(EventFilter[IllegalStateException]("Calling")) {
|
||||
(intercept[IllegalStateException] {
|
||||
intercept[IllegalStateException] {
|
||||
TypedActor.self[Foo]
|
||||
}).getMessage should ===("Calling TypedActor.self outside of a TypedActor implementation method!")
|
||||
}.getMessage should ===("Calling TypedActor.self outside of a TypedActor implementation method!")
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -405,10 +406,10 @@ class TypedActorSpec
|
|||
"expected")
|
||||
t.read() should ===(1) //Make sure state is not reset after failure
|
||||
|
||||
(intercept[IllegalStateException] { t.failingJOptionPigdog }).getMessage should ===("expected")
|
||||
intercept[IllegalStateException] { t.failingJOptionPigdog }.getMessage should ===("expected")
|
||||
t.read() should ===(1) //Make sure state is not reset after failure
|
||||
|
||||
(intercept[IllegalStateException] { t.failingOptionPigdog }).getMessage should ===("expected")
|
||||
intercept[IllegalStateException] { t.failingOptionPigdog }.getMessage should ===("expected")
|
||||
|
||||
t.read() should ===(1) //Make sure state is not reset after failure
|
||||
|
||||
|
|
@ -456,7 +457,7 @@ class TypedActorSpec
|
|||
}
|
||||
|
||||
"be able to use balancing dispatcher" in within(timeout.duration) {
|
||||
val thais = for (i <- 1 to 60) yield newFooBar("pooled-dispatcher", 6 seconds)
|
||||
val thais = for (_ <- 1 to 60) yield newFooBar("pooled-dispatcher", 6 seconds)
|
||||
val iterator = new CyclicIterator(thais)
|
||||
|
||||
val results = for (i <- 1 to 120) yield (i, iterator.next.futurePigdog(200 millis, i))
|
||||
|
|
|
|||
|
|
@ -41,8 +41,8 @@ object UidClashTest {
|
|||
|
||||
def receive = {
|
||||
case PleaseRestart => throw new Exception("restart")
|
||||
case Terminated(ref) => throw new TerminatedForNonWatchedActor
|
||||
// This is the tricky part to make this test a positive one (avoid expectNoMsg).
|
||||
case Terminated(_) => throw new TerminatedForNonWatchedActor
|
||||
// This is the tricky part to make this test a positive one (avoid expectNoMessage).
|
||||
// Since anything enqueued in postRestart will arrive before the Terminated
|
||||
// the bug triggers, there needs to be a bounce:
|
||||
// 1. Ping is sent from postRestart to self
|
||||
|
|
|
|||
|
|
@ -5,15 +5,12 @@
|
|||
package akka.actor.dispatch
|
||||
|
||||
import language.postfixOps
|
||||
|
||||
import java.rmi.RemoteException
|
||||
import java.util.concurrent.{ ConcurrentHashMap, CountDownLatch, TimeUnit }
|
||||
import java.util.concurrent.atomic.{ AtomicInteger, AtomicLong }
|
||||
|
||||
import org.scalatest.Assertions._
|
||||
|
||||
import com.typesafe.config.Config
|
||||
|
||||
import akka.actor._
|
||||
import akka.dispatch.sysmsg.SystemMessageList
|
||||
import akka.dispatch._
|
||||
|
|
@ -21,6 +18,8 @@ import akka.event.Logging.Error
|
|||
import akka.pattern.ask
|
||||
import akka.testkit._
|
||||
import akka.util.Switch
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
import scala.concurrent.duration._
|
||||
import scala.concurrent.{ Await, Future }
|
||||
import scala.annotation.tailrec
|
||||
|
|
@ -575,9 +574,12 @@ class DispatcherModelSpec extends ActorModelSpec(DispatcherModelSpec.config) {
|
|||
|
||||
"A " + dispatcherType must {
|
||||
"process messages in parallel" in {
|
||||
val probe = TestProbe()
|
||||
implicit val dispatcher = interceptedDispatcher()
|
||||
val aStart, aStop, bParallel = new CountDownLatch(1)
|
||||
val a, b = newTestActor(dispatcher.id)
|
||||
probe.watch(a)
|
||||
probe.watch(b)
|
||||
|
||||
a ! Meet(aStart, aStop)
|
||||
assertCountDown(aStart, 3.seconds.dilated.toMillis, "Should process first message within 3 seconds")
|
||||
|
|
@ -590,7 +592,8 @@ class DispatcherModelSpec extends ActorModelSpec(DispatcherModelSpec.config) {
|
|||
system.stop(a)
|
||||
system.stop(b)
|
||||
|
||||
while (!a.isTerminated && !b.isTerminated) {} //Busy wait for termination
|
||||
probe.expectTerminated(a)
|
||||
probe.expectTerminated(b)
|
||||
|
||||
assertRefDefaultZero(a)(registers = 1, unregisters = 1, msgsReceived = 1, msgsProcessed = 1)
|
||||
assertRefDefaultZero(b)(registers = 1, unregisters = 1, msgsReceived = 1, msgsProcessed = 1)
|
||||
|
|
@ -598,6 +601,7 @@ class DispatcherModelSpec extends ActorModelSpec(DispatcherModelSpec.config) {
|
|||
}
|
||||
}
|
||||
|
||||
@silent
|
||||
object BalancingDispatcherModelSpec {
|
||||
import ActorModelSpec._
|
||||
|
||||
|
|
@ -636,6 +640,7 @@ object BalancingDispatcherModelSpec {
|
|||
}
|
||||
}
|
||||
|
||||
@silent
|
||||
class BalancingDispatcherModelSpec extends ActorModelSpec(BalancingDispatcherModelSpec.config) {
|
||||
import ActorModelSpec._
|
||||
|
||||
|
|
|
|||
|
|
@ -7,7 +7,6 @@ package akka.actor.dispatch
|
|||
import java.util.concurrent.{ CountDownLatch, TimeUnit }
|
||||
|
||||
import akka.actor.{ Actor, ActorCell, ActorRefWithCell, Props }
|
||||
import akka.dispatch.Mailbox
|
||||
import akka.testkit.AkkaSpec
|
||||
|
||||
object BalancingDispatcherSpec {
|
||||
|
|
@ -28,7 +27,7 @@ class BalancingDispatcherSpec extends AkkaSpec(BalancingDispatcherSpec.config) {
|
|||
var invocationCount = 0
|
||||
|
||||
def receive = {
|
||||
case x: Int => {
|
||||
case _: Int => {
|
||||
Thread.sleep(delay)
|
||||
invocationCount += 1
|
||||
finishedCounter.countDown()
|
||||
|
|
@ -83,11 +82,11 @@ class BalancingDispatcherSpec extends AkkaSpec(BalancingDispatcherSpec.config) {
|
|||
}
|
||||
|
||||
finishedCounter.await(5, TimeUnit.SECONDS)
|
||||
fast.underlying.asInstanceOf[ActorCell].mailbox.asInstanceOf[Mailbox].hasMessages should ===(false)
|
||||
slow.underlying.asInstanceOf[ActorCell].mailbox.asInstanceOf[Mailbox].hasMessages should ===(false)
|
||||
fast.underlying.asInstanceOf[ActorCell].mailbox.hasMessages should ===(false)
|
||||
slow.underlying.asInstanceOf[ActorCell].mailbox.hasMessages should ===(false)
|
||||
fast.underlying.asInstanceOf[ActorCell].actor.asInstanceOf[DelayableActor].invocationCount should be > sentToFast
|
||||
fast.underlying.asInstanceOf[ActorCell].actor.asInstanceOf[DelayableActor].invocationCount should be >
|
||||
(slow.underlying.asInstanceOf[ActorCell].actor.asInstanceOf[DelayableActor].invocationCount)
|
||||
slow.underlying.asInstanceOf[ActorCell].actor.asInstanceOf[DelayableActor].invocationCount
|
||||
system.stop(slow)
|
||||
system.stop(fast)
|
||||
}
|
||||
|
|
|
|||
|
|
@ -7,8 +7,8 @@ package akka.actor.dispatch
|
|||
import language.postfixOps
|
||||
|
||||
import java.util.concurrent.{ CountDownLatch, TimeUnit }
|
||||
import java.util.concurrent.atomic.{ AtomicBoolean }
|
||||
import akka.testkit.{ AkkaSpec }
|
||||
import java.util.concurrent.atomic.AtomicBoolean
|
||||
import akka.testkit.AkkaSpec
|
||||
import akka.actor.{ Actor, Props }
|
||||
import scala.concurrent.Await
|
||||
import scala.concurrent.duration._
|
||||
|
|
@ -56,13 +56,11 @@ object DispatcherActorSpec {
|
|||
class DispatcherActorSpec extends AkkaSpec(DispatcherActorSpec.config) with DefaultTimeout {
|
||||
import DispatcherActorSpec._
|
||||
|
||||
private val unit = TimeUnit.MILLISECONDS
|
||||
|
||||
"A Dispatcher and an Actor" must {
|
||||
|
||||
"support tell" in {
|
||||
val actor = system.actorOf(Props[OneWayTestActor].withDispatcher("test-dispatcher"))
|
||||
val result = actor ! "OneWay"
|
||||
actor ! "OneWay"
|
||||
assert(OneWayTestActor.oneWay.await(1, TimeUnit.SECONDS))
|
||||
system.stop(actor)
|
||||
}
|
||||
|
|
@ -84,7 +82,7 @@ class DispatcherActorSpec extends AkkaSpec(DispatcherActorSpec.config) with Defa
|
|||
|
||||
val slowOne = system.actorOf(Props(new Actor {
|
||||
def receive = {
|
||||
case "hogexecutor" => { sender() ! "OK"; start.await }
|
||||
case "hogexecutor" => { sender() ! "OK"; start.await() }
|
||||
case "ping" => if (works.get) latch.countDown()
|
||||
}
|
||||
}).withDispatcher(throughputDispatcher))
|
||||
|
|
@ -118,7 +116,7 @@ class DispatcherActorSpec extends AkkaSpec(DispatcherActorSpec.config) with Defa
|
|||
|
||||
val slowOne = system.actorOf(Props(new Actor {
|
||||
def receive = {
|
||||
case "hogexecutor" => { ready.countDown(); start.await }
|
||||
case "hogexecutor" => { ready.countDown(); start.await() }
|
||||
case "ping" => { works.set(false); context.stop(self) }
|
||||
}
|
||||
}).withDispatcher(throughputDispatcher))
|
||||
|
|
|
|||
|
|
@ -15,7 +15,7 @@ class DispatcherActorsSpec extends AkkaSpec {
|
|||
class SlowActor(finishedCounter: CountDownLatch) extends Actor {
|
||||
|
||||
def receive = {
|
||||
case x: Int => {
|
||||
case _: Int => {
|
||||
Thread.sleep(50) // slow actor
|
||||
finishedCounter.countDown()
|
||||
}
|
||||
|
|
@ -24,7 +24,7 @@ class DispatcherActorsSpec extends AkkaSpec {
|
|||
|
||||
class FastActor(finishedCounter: CountDownLatch) extends Actor {
|
||||
def receive = {
|
||||
case x: Int => {
|
||||
case _: Int => {
|
||||
finishedCounter.countDown()
|
||||
}
|
||||
}
|
||||
|
|
@ -48,9 +48,9 @@ class DispatcherActorsSpec extends AkkaSpec {
|
|||
}
|
||||
|
||||
// now assert that f is finished while s is still busy
|
||||
fFinished.await
|
||||
fFinished.await()
|
||||
assert(sFinished.getCount > 0)
|
||||
sFinished.await
|
||||
sFinished.await()
|
||||
assert(sFinished.getCount === 0L)
|
||||
system.stop(f)
|
||||
system.stop(s)
|
||||
|
|
|
|||
|
|
@ -16,6 +16,8 @@ import akka.routing.FromConfig
|
|||
import java.util.concurrent.ConcurrentLinkedQueue
|
||||
import java.util.concurrent.atomic.AtomicBoolean
|
||||
|
||||
import akka.util.unused
|
||||
|
||||
object DispatchersSpec {
|
||||
val config = """
|
||||
myapp {
|
||||
|
|
@ -69,18 +71,18 @@ object DispatchersSpec {
|
|||
}
|
||||
}
|
||||
|
||||
class OneShotMailboxType(settings: ActorSystem.Settings, config: Config)
|
||||
class OneShotMailboxType(@unused settings: ActorSystem.Settings, @unused config: Config)
|
||||
extends MailboxType
|
||||
with ProducesMessageQueue[DoublingMailbox] {
|
||||
val created = new AtomicBoolean(false)
|
||||
override def create(owner: Option[ActorRef], system: Option[ActorSystem]) =
|
||||
if (created.compareAndSet(false, true)) {
|
||||
new DoublingMailbox(owner)
|
||||
new DoublingMailbox()
|
||||
} else
|
||||
throw new IllegalStateException("I've already created the mailbox.")
|
||||
}
|
||||
|
||||
class DoublingMailbox(owner: Option[ActorRef]) extends UnboundedQueueBasedMessageQueue {
|
||||
class DoublingMailbox() extends UnboundedQueueBasedMessageQueue {
|
||||
final val queue = new ConcurrentLinkedQueue[Envelope]()
|
||||
override def enqueue(receiver: ActorRef, handle: Envelope): Unit = {
|
||||
queue.add(handle)
|
||||
|
|
@ -108,11 +110,11 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
val throughput = "throughput"
|
||||
val id = "id"
|
||||
|
||||
def instance(dispatcher: MessageDispatcher): (MessageDispatcher) => Boolean = _ == dispatcher
|
||||
def ofType[T <: MessageDispatcher: ClassTag]: (MessageDispatcher) => Boolean =
|
||||
def instance(dispatcher: MessageDispatcher): MessageDispatcher => Boolean = _ == dispatcher
|
||||
def ofType[T <: MessageDispatcher: ClassTag]: MessageDispatcher => Boolean =
|
||||
_.getClass == implicitly[ClassTag[T]].runtimeClass
|
||||
|
||||
def typesAndValidators: Map[String, (MessageDispatcher) => Boolean] =
|
||||
def typesAndValidators: Map[String, MessageDispatcher => Boolean] =
|
||||
Map("PinnedDispatcher" -> ofType[PinnedDispatcher], "Dispatcher" -> ofType[Dispatcher])
|
||||
|
||||
def validTypes = typesAndValidators.keys.toList
|
||||
|
|
@ -129,7 +131,7 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
actor ! "what's the name?"
|
||||
val Expected = R("(DispatchersSpec-myapp.mydispatcher-[1-9][0-9]*)")
|
||||
expectMsgPF() {
|
||||
case Expected(x) =>
|
||||
case Expected(_) =>
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -185,7 +187,7 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
system.actorOf(Props[ThreadNameEcho].withDispatcher("myapp.thread-pool-dispatcher")) ! "what's the name?"
|
||||
val Expected = R("(DispatchersSpec-myapp.thread-pool-dispatcher-[1-9][0-9]*)")
|
||||
expectMsgPF() {
|
||||
case Expected(x) =>
|
||||
case Expected(_) =>
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -193,7 +195,7 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
system.actorOf(Props[ThreadNameEcho]) ! "what's the name?"
|
||||
val Expected = R("(DispatchersSpec-akka.actor.default-dispatcher-[1-9][0-9]*)")
|
||||
expectMsgPF() {
|
||||
case Expected(x) =>
|
||||
case Expected(_) =>
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -201,7 +203,7 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
system.actorOf(Props[ThreadNameEcho].withDispatcher("myapp.my-pinned-dispatcher")) ! "what's the name?"
|
||||
val Expected = R("(DispatchersSpec-myapp.my-pinned-dispatcher-[1-9][0-9]*)")
|
||||
expectMsgPF() {
|
||||
case Expected(x) =>
|
||||
case Expected(_) =>
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -209,7 +211,7 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
system.actorOf(Props[ThreadNameEcho].withDispatcher("myapp.balancing-dispatcher")) ! "what's the name?"
|
||||
val Expected = R("(DispatchersSpec-myapp.balancing-dispatcher-[1-9][0-9]*)")
|
||||
expectMsgPF() {
|
||||
case Expected(x) =>
|
||||
case Expected(_) =>
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -229,7 +231,7 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
routee ! "what's the name?"
|
||||
val Expected = R("""(DispatchersSpec-akka\.actor\.deployment\./pool1\.pool-dispatcher-[1-9][0-9]*)""")
|
||||
expectMsgPF() {
|
||||
case Expected(x) =>
|
||||
case Expected(_) =>
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -237,10 +239,10 @@ class DispatchersSpec extends AkkaSpec(DispatchersSpec.config) with ImplicitSend
|
|||
system.actorOf(FromConfig.props(Props[ThreadNameEcho]), name = "balanced") ! "what's the name?"
|
||||
val Expected = R("""(DispatchersSpec-BalancingPool-/balanced-[1-9][0-9]*)""")
|
||||
expectMsgPF() {
|
||||
case Expected(x) =>
|
||||
case Expected(_) =>
|
||||
}
|
||||
expectMsgPF() {
|
||||
case Expected(x) =>
|
||||
case Expected(_) =>
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -32,15 +32,13 @@ object PinnedActorSpec {
|
|||
class PinnedActorSpec extends AkkaSpec(PinnedActorSpec.config) with BeforeAndAfterEach with DefaultTimeout {
|
||||
import PinnedActorSpec._
|
||||
|
||||
private val unit = TimeUnit.MILLISECONDS
|
||||
|
||||
"A PinnedActor" must {
|
||||
|
||||
"support tell" in {
|
||||
var oneWay = new CountDownLatch(1)
|
||||
val oneWay = new CountDownLatch(1)
|
||||
val actor = system.actorOf(
|
||||
Props(new Actor { def receive = { case "OneWay" => oneWay.countDown() } }).withDispatcher("pinned-dispatcher"))
|
||||
val result = actor ! "OneWay"
|
||||
actor ! "OneWay"
|
||||
assert(oneWay.await(1, TimeUnit.SECONDS))
|
||||
system.stop(actor)
|
||||
}
|
||||
|
|
|
|||
|
|
@ -40,7 +40,7 @@ class ControlAwareDispatcherSpec extends AkkaSpec(ControlAwareDispatcherSpec.con
|
|||
// with RepointableActorRef, since messages might be queued in
|
||||
// UnstartedCell and the sent to the PriorityQueue and consumed immediately
|
||||
// without the ordering taking place.
|
||||
val actor = system.actorOf(Props(new Actor {
|
||||
system.actorOf(Props(new Actor {
|
||||
context.actorOf(Props(new Actor {
|
||||
|
||||
self ! "test"
|
||||
|
|
|
|||
|
|
@ -55,7 +55,7 @@ class ExecutionContextSpec extends AkkaSpec with DefaultTimeout {
|
|||
batchable {
|
||||
val lock, callingThreadLock, count = new AtomicInteger(0)
|
||||
callingThreadLock.compareAndSet(0, 1) // Enable the lock
|
||||
(1 to 100).foreach { i =>
|
||||
(1 to 100).foreach { _ =>
|
||||
batchable {
|
||||
if (callingThreadLock.get != 0) p.tryFailure(new IllegalStateException("Batch was executed inline!"))
|
||||
else if (count.incrementAndGet == 100) p.trySuccess(()) //Done
|
||||
|
|
@ -77,12 +77,12 @@ class ExecutionContextSpec extends AkkaSpec with DefaultTimeout {
|
|||
def batchable[T](f: => T)(implicit ec: ExecutionContext): Unit =
|
||||
ec.execute(new Batchable {
|
||||
override def isBatchable = true
|
||||
override def run: Unit = f
|
||||
override def run(): Unit = f
|
||||
})
|
||||
|
||||
val latch = TestLatch(101)
|
||||
batchable {
|
||||
(1 to 100).foreach { i =>
|
||||
(1 to 100).foreach { _ =>
|
||||
batchable {
|
||||
val deadlock = TestLatch(1)
|
||||
batchable { deadlock.open() }
|
||||
|
|
|
|||
|
|
@ -5,12 +5,14 @@
|
|||
package akka.dispatch
|
||||
|
||||
import language.postfixOps
|
||||
|
||||
import java.util.concurrent.{ BlockingQueue, ConcurrentLinkedQueue }
|
||||
|
||||
import org.scalatest.{ BeforeAndAfterAll, BeforeAndAfterEach }
|
||||
import com.typesafe.config.{ Config, ConfigFactory }
|
||||
import akka.actor._
|
||||
import akka.testkit.{ AkkaSpec, EventFilter }
|
||||
import akka.util.unused
|
||||
|
||||
import scala.concurrent.{ Await, ExecutionContext, Future }
|
||||
import scala.concurrent.duration._
|
||||
|
||||
|
|
@ -47,7 +49,7 @@ abstract class MailboxSpec extends AkkaSpec with BeforeAndAfterAll with BeforeAn
|
|||
val q = factory(config)
|
||||
ensureInitialMailboxState(config, q)
|
||||
|
||||
for (i <- 1 to config.capacity) q.enqueue(testActor, exampleMessage)
|
||||
for (_ <- 1 to config.capacity) q.enqueue(testActor, exampleMessage)
|
||||
|
||||
q.numberOfMessages should ===(config.capacity)
|
||||
q.hasMessages should ===(true)
|
||||
|
|
@ -225,14 +227,14 @@ object CustomMailboxSpec {
|
|||
}
|
||||
"""
|
||||
|
||||
class MyMailboxType(settings: ActorSystem.Settings, config: Config) extends MailboxType {
|
||||
class MyMailboxType(@unused settings: ActorSystem.Settings, @unused config: Config) extends MailboxType {
|
||||
override def create(owner: Option[ActorRef], system: Option[ActorSystem]) = owner match {
|
||||
case Some(o) => new MyMailbox(o)
|
||||
case None => throw new Exception("no mailbox owner given")
|
||||
}
|
||||
}
|
||||
|
||||
class MyMailbox(owner: ActorRef) extends UnboundedQueueBasedMessageQueue {
|
||||
class MyMailbox(@unused owner: ActorRef) extends UnboundedQueueBasedMessageQueue {
|
||||
final val queue = new ConcurrentLinkedQueue[Envelope]()
|
||||
}
|
||||
}
|
||||
|
|
@ -255,8 +257,8 @@ class SingleConsumerOnlyMailboxSpec extends MailboxSpec {
|
|||
lazy val name = "The single-consumer-only mailbox implementation"
|
||||
override def maxConsumers = 1
|
||||
def factory = {
|
||||
case u: UnboundedMailbox => SingleConsumerOnlyUnboundedMailbox().create(None, None)
|
||||
case b @ BoundedMailbox(capacity, _) => NonBlockingBoundedMailbox(capacity).create(None, None)
|
||||
case _: UnboundedMailbox => SingleConsumerOnlyUnboundedMailbox().create(None, None)
|
||||
case _ @BoundedMailbox(capacity, _) => NonBlockingBoundedMailbox(capacity).create(None, None)
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -5,11 +5,11 @@
|
|||
package akka.dispatch
|
||||
|
||||
import language.postfixOps
|
||||
|
||||
import com.typesafe.config.Config
|
||||
|
||||
import akka.actor.{ Actor, ActorSystem, Props }
|
||||
import akka.testkit.{ AkkaSpec, DefaultTimeout }
|
||||
import akka.util.unused
|
||||
|
||||
import scala.concurrent.duration._
|
||||
|
||||
object PriorityDispatcherSpec {
|
||||
|
|
@ -22,13 +22,13 @@ object PriorityDispatcherSpec {
|
|||
}
|
||||
"""
|
||||
|
||||
class Unbounded(settings: ActorSystem.Settings, config: Config)
|
||||
class Unbounded(@unused settings: ActorSystem.Settings, @unused config: Config)
|
||||
extends UnboundedPriorityMailbox(PriorityGenerator({
|
||||
case i: Int => i //Reverse order
|
||||
case 'Result => Int.MaxValue
|
||||
}: Any => Int))
|
||||
|
||||
class Bounded(settings: ActorSystem.Settings, config: Config)
|
||||
class Bounded(@unused settings: ActorSystem.Settings, @unused config: Config)
|
||||
extends BoundedPriorityMailbox(PriorityGenerator({
|
||||
case i: Int => i //Reverse order
|
||||
case 'Result => Int.MaxValue
|
||||
|
|
@ -57,7 +57,7 @@ class PriorityDispatcherSpec extends AkkaSpec(PriorityDispatcherSpec.config) wit
|
|||
// with RepointableActorRef, since messages might be queued in
|
||||
// UnstartedCell and the sent to the PriorityQueue and consumed immediately
|
||||
// without the ordering taking place.
|
||||
val actor = system.actorOf(Props(new Actor {
|
||||
system.actorOf(Props(new Actor {
|
||||
context.actorOf(Props(new Actor {
|
||||
|
||||
val acc = scala.collection.mutable.ListBuffer[Int]()
|
||||
|
|
|
|||
|
|
@ -5,11 +5,11 @@
|
|||
package akka.dispatch
|
||||
|
||||
import language.postfixOps
|
||||
|
||||
import com.typesafe.config.Config
|
||||
|
||||
import akka.actor.{ Actor, ActorSystem, Props }
|
||||
import akka.testkit.{ AkkaSpec, DefaultTimeout }
|
||||
import akka.util.unused
|
||||
|
||||
import scala.concurrent.duration._
|
||||
|
||||
object StablePriorityDispatcherSpec {
|
||||
|
|
@ -22,17 +22,17 @@ object StablePriorityDispatcherSpec {
|
|||
}
|
||||
"""
|
||||
|
||||
class Unbounded(settings: ActorSystem.Settings, config: Config)
|
||||
class Unbounded(@unused settings: ActorSystem.Settings, @unused config: Config)
|
||||
extends UnboundedStablePriorityMailbox(PriorityGenerator({
|
||||
case i: Int if i <= 100 => i // Small integers have high priority
|
||||
case i: Int => 101 // Don't care for other integers
|
||||
case _: Int => 101 // Don't care for other integers
|
||||
case 'Result => Int.MaxValue
|
||||
}: Any => Int))
|
||||
|
||||
class Bounded(settings: ActorSystem.Settings, config: Config)
|
||||
class Bounded(@unused settings: ActorSystem.Settings, @unused config: Config)
|
||||
extends BoundedStablePriorityMailbox(PriorityGenerator({
|
||||
case i: Int if i <= 100 => i // Small integers have high priority
|
||||
case i: Int => 101 // Don't care for other integers
|
||||
case _: Int => 101 // Don't care for other integers
|
||||
case 'Result => Int.MaxValue
|
||||
}: Any => Int), 1000, 10 seconds)
|
||||
|
||||
|
|
@ -61,7 +61,7 @@ class StablePriorityDispatcherSpec extends AkkaSpec(StablePriorityDispatcherSpec
|
|||
// with RepointableActorRef, since messages might be queued in
|
||||
// UnstartedCell and then sent to the StablePriorityQueue and consumed immediately
|
||||
// without the ordering taking place.
|
||||
val actor = system.actorOf(Props(new Actor {
|
||||
system.actorOf(Props(new Actor {
|
||||
context.actorOf(Props(new Actor {
|
||||
|
||||
val acc = scala.collection.mutable.ListBuffer[Int]()
|
||||
|
|
|
|||
|
|
@ -37,7 +37,7 @@ class AddressTerminatedTopicBenchSpec extends AkkaSpec("akka.loglevel=INFO") {
|
|||
|
||||
val t1 = System.nanoTime()
|
||||
val p = Props(classOf[Subscriber], testActor)
|
||||
val subscribers = Vector.fill(num)(sys.actorOf(p))
|
||||
Vector.fill(num)(sys.actorOf(p))
|
||||
receiveN(num, 10.seconds)
|
||||
log.info("Starting {} actors took {} ms", num, (System.nanoTime() - t1).nanos.toMillis)
|
||||
|
||||
|
|
|
|||
|
|
@ -96,7 +96,7 @@ abstract class EventBusSpec(busName: String, conf: Config = ConfigFactory.empty(
|
|||
bus.subscribe(subscriber, classifier)
|
||||
bus.publish(event)
|
||||
expectMsg(event)
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
bus.unsubscribe(subscriber, classifier)
|
||||
}
|
||||
|
||||
|
|
@ -108,7 +108,7 @@ abstract class EventBusSpec(busName: String, conf: Config = ConfigFactory.empty(
|
|||
expectMsg(event)
|
||||
expectMsg(event)
|
||||
expectMsg(event)
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
bus.unsubscribe(subscriber, classifier)
|
||||
}
|
||||
|
||||
|
|
@ -136,14 +136,14 @@ abstract class EventBusSpec(busName: String, conf: Config = ConfigFactory.empty(
|
|||
expectMsg(event)
|
||||
bus.unsubscribe(subscriber, classifier)
|
||||
bus.unsubscribe(otherSubscriber, otherClassifier)
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
}
|
||||
|
||||
"not publish the given event to a former subscriber" in {
|
||||
bus.subscribe(subscriber, classifier)
|
||||
bus.unsubscribe(subscriber, classifier)
|
||||
bus.publish(event)
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
}
|
||||
|
||||
"cleanup subscriber" in {
|
||||
|
|
@ -207,7 +207,7 @@ class ActorEventBusSpec(conf: Config) extends EventBusSpec("ActorEventBus", conf
|
|||
expectUnsubscribedByUnsubscriber(p, subs)
|
||||
|
||||
bus.publish(m(2))
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
|
||||
disposeSubscriber(system, subs)
|
||||
disposeSubscriber(system, a1)
|
||||
|
|
@ -256,7 +256,7 @@ class ActorEventBusSpec(conf: Config) extends EventBusSpec("ActorEventBus", conf
|
|||
|
||||
bus.unsubscribe(subs, a1)
|
||||
bus.publish(m1(2))
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
bus.publish(m2(2))
|
||||
expectMsg(m2(2))
|
||||
|
||||
|
|
@ -264,7 +264,7 @@ class ActorEventBusSpec(conf: Config) extends EventBusSpec("ActorEventBus", conf
|
|||
expectUnregisterFromUnsubscriber(p, subs)
|
||||
bus.publish(m1(3))
|
||||
bus.publish(m2(3))
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
|
||||
disposeSubscriber(system, subs)
|
||||
disposeSubscriber(system, a1)
|
||||
|
|
@ -275,7 +275,7 @@ class ActorEventBusSpec(conf: Config) extends EventBusSpec("ActorEventBus", conf
|
|||
val expectedMsg = s"actor $a has terminated, unsubscribing it from $bus"
|
||||
p.fishForMessage(1 second, hint = expectedMsg) {
|
||||
case Logging.Debug(_, _, msg) if msg.equals(expectedMsg) => true
|
||||
case other => false
|
||||
case _ => false
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -283,7 +283,7 @@ class ActorEventBusSpec(conf: Config) extends EventBusSpec("ActorEventBus", conf
|
|||
val expectedMsg = s"unregistered watch of $a in $bus"
|
||||
p.fishForMessage(1 second, hint = expectedMsg) {
|
||||
case Logging.Debug(_, _, msg) if msg.equals(expectedMsg) => true
|
||||
case other => false
|
||||
case _ => false
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -86,7 +86,7 @@ class EventStreamSpec extends AkkaSpec(EventStreamSpec.config) {
|
|||
expectMsg(M(42))
|
||||
bus.unsubscribe(testActor)
|
||||
bus.publish(M(13))
|
||||
expectNoMsg
|
||||
expectNoMessage
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -160,7 +160,7 @@ class EventStreamSpec extends AkkaSpec(EventStreamSpec.config) {
|
|||
bus.publish(a)
|
||||
expectMsg(b2)
|
||||
expectMsg(a)
|
||||
expectNoMsg
|
||||
expectNoMessage
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -202,7 +202,7 @@ class EventStreamSpec extends AkkaSpec(EventStreamSpec.config) {
|
|||
es.publish(tm2)
|
||||
a1.expectMsgType[AT] should ===(tm2)
|
||||
a2.expectMsgType[BT] should ===(tm2)
|
||||
a3.expectNoMsg(1 second)
|
||||
a3.expectNoMessage(1 second)
|
||||
a4.expectMsgType[CCATBT] should ===(tm2)
|
||||
es.unsubscribe(a1.ref, classOf[AT]) should ===(true)
|
||||
es.unsubscribe(a2.ref, classOf[BT]) should ===(true)
|
||||
|
|
@ -224,7 +224,7 @@ class EventStreamSpec extends AkkaSpec(EventStreamSpec.config) {
|
|||
es.publish(tm2)
|
||||
a1.expectMsgType[AT] should ===(tm2)
|
||||
a2.expectMsgType[BT] should ===(tm2)
|
||||
a3.expectNoMsg(1 second)
|
||||
a3.expectNoMessage(1 second)
|
||||
a4.expectMsgType[CCATBT] should ===(tm2)
|
||||
es.unsubscribe(a1.ref, classOf[AT]) should ===(true)
|
||||
es.unsubscribe(a2.ref, classOf[BT]) should ===(true)
|
||||
|
|
@ -277,7 +277,7 @@ class EventStreamSpec extends AkkaSpec(EventStreamSpec.config) {
|
|||
es.subscribe(a1.ref, classOf[AT]) should ===(true)
|
||||
es.publish(tm1)
|
||||
a1.expectMsgType[AT] should ===(tm1)
|
||||
a2.expectNoMsg(1 second)
|
||||
a2.expectNoMessage(1 second)
|
||||
es.subscribe(a2.ref, classOf[BTT]) should ===(true)
|
||||
es.publish(tm1)
|
||||
a1.expectMsgType[AT] should ===(tm1)
|
||||
|
|
@ -308,7 +308,7 @@ class EventStreamSpec extends AkkaSpec(EventStreamSpec.config) {
|
|||
|
||||
es.publish(tm)
|
||||
|
||||
a1.expectNoMsg(1 second)
|
||||
a1.expectNoMessage(1 second)
|
||||
a2.expectMsg(tm)
|
||||
} finally {
|
||||
shutdown(sys)
|
||||
|
|
@ -396,12 +396,12 @@ class EventStreamSpec extends AkkaSpec(EventStreamSpec.config) {
|
|||
|
||||
es.unsubscribe(a2.ref, classOf[A]) should equal(true)
|
||||
fishForDebugMessage(a1, s"unsubscribing ${a2.ref} from channel class akka.event.EventStreamSpec$$A")
|
||||
a1.expectNoMsg(1 second)
|
||||
a1.expectNoMessage(1 second)
|
||||
|
||||
es.unsubscribe(a2.ref, classOf[T]) should equal(true)
|
||||
fishForDebugMessage(a1, s"unsubscribing ${a2.ref} from channel interface akka.event.EventStreamSpec$$T")
|
||||
fishForDebugMessage(a1, s"unwatching ${a2.ref}, since has no subscriptions")
|
||||
a1.expectNoMsg(1 second)
|
||||
a1.expectNoMessage(1 second)
|
||||
|
||||
es.unsubscribe(a2.ref, classOf[T]) should equal(false)
|
||||
|
||||
|
|
@ -424,7 +424,7 @@ class EventStreamSpec extends AkkaSpec(EventStreamSpec.config) {
|
|||
private def fishForDebugMessage(a: TestProbe, messagePrefix: String, max: Duration = 3 seconds): Unit = {
|
||||
a.fishForMessage(max, hint = "expected debug message prefix: " + messagePrefix) {
|
||||
case Logging.Debug(_, _, msg: String) if msg.startsWith(messagePrefix) => true
|
||||
case other => false
|
||||
case _ => false
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -159,7 +159,7 @@ class LoggerSpec extends WordSpec with Matchers {
|
|||
case _ => false
|
||||
}
|
||||
} else {
|
||||
probe.expectNoMsg(0.5.seconds.dilated)
|
||||
probe.expectNoMessage(0.5.seconds.dilated)
|
||||
}
|
||||
} finally {
|
||||
TestKit.shutdownActorSystem(system)
|
||||
|
|
|
|||
|
|
@ -97,11 +97,10 @@ class LoggingReceiveSpec extends WordSpec with BeforeAndAfterAll {
|
|||
def switch: Actor.Receive = { case "becomenull" => context.become(r, false) }
|
||||
def receive =
|
||||
switch.orElse(LoggingReceive {
|
||||
case x => sender() ! "x"
|
||||
case _ => sender() ! "x"
|
||||
})
|
||||
})
|
||||
|
||||
val name = actor.path.toString
|
||||
actor ! "buh"
|
||||
expectMsg(
|
||||
Logging
|
||||
|
|
@ -112,7 +111,7 @@ class LoggingReceiveSpec extends WordSpec with BeforeAndAfterAll {
|
|||
|
||||
actor ! "bah"
|
||||
expectMsgPF() {
|
||||
case UnhandledMessage("bah", testActor, `actor`) => true
|
||||
case UnhandledMessage("bah", _, `actor`) => true
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
@ -227,7 +226,7 @@ class LoggingReceiveSpec extends WordSpec with BeforeAndAfterAll {
|
|||
supervisor.watch(actor)
|
||||
fishForMessage(hint = "now watched by") {
|
||||
case Logging.Debug(`aname`, `sclass`, msg: String) if msg.startsWith("now watched by") => true
|
||||
case m => false
|
||||
case _ => false
|
||||
}
|
||||
|
||||
supervisor.unwatch(actor)
|
||||
|
|
|
|||
|
|
@ -9,7 +9,9 @@ import java.util.concurrent.TimeUnit
|
|||
|
||||
import akka.actor.Props
|
||||
import akka.testkit.{ AkkaSpec, TestActorRef }
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
@silent
|
||||
class InetAddressDnsResolverSpec extends AkkaSpec("""
|
||||
akka.io.dns.inet-address.positive-ttl = default
|
||||
akka.io.dns.inet-address.negative-ttl = default
|
||||
|
|
@ -117,6 +119,8 @@ class InetAddressDnsResolverSpec extends AkkaSpec("""
|
|||
}
|
||||
|
||||
}
|
||||
|
||||
@silent
|
||||
class InetAddressDnsResolverConfigSpec extends AkkaSpec("""
|
||||
akka.io.dns.inet-address.positive-ttl = forever
|
||||
akka.io.dns.inet-address.negative-ttl = never
|
||||
|
|
|
|||
|
|
@ -638,7 +638,7 @@ class TcpConnectionSpec extends AkkaSpec("""
|
|||
sel.select(3000)
|
||||
|
||||
key.isConnectable should ===(true)
|
||||
val forceThisLazyVal = connectionActor.toString
|
||||
connectionActor.toString // force the lazy val
|
||||
Thread.sleep(300)
|
||||
selector.send(connectionActor, ChannelConnectable)
|
||||
userHandler.expectMsg(CommandFailed(Connect(UnboundAddress)))
|
||||
|
|
@ -720,7 +720,7 @@ class TcpConnectionSpec extends AkkaSpec("""
|
|||
}
|
||||
// dump the NACKs
|
||||
writer.receiveWhile(1.second) {
|
||||
case CommandFailed(write) => written -= 1
|
||||
case CommandFailed(_) => written -= 1
|
||||
}
|
||||
writer.msgAvailable should ===(false)
|
||||
|
||||
|
|
@ -758,7 +758,7 @@ class TcpConnectionSpec extends AkkaSpec("""
|
|||
}
|
||||
// dump the NACKs
|
||||
writer.receiveWhile(1.second) {
|
||||
case CommandFailed(write) => written -= 1
|
||||
case CommandFailed(_) => written -= 1
|
||||
}
|
||||
|
||||
// drain the queue until it works again
|
||||
|
|
@ -794,7 +794,7 @@ class TcpConnectionSpec extends AkkaSpec("""
|
|||
}
|
||||
// dump the NACKs
|
||||
writer.receiveWhile(1.second) {
|
||||
case CommandFailed(write) => written -= 1
|
||||
case CommandFailed(_) => written -= 1
|
||||
}
|
||||
writer.msgAvailable should ===(false)
|
||||
|
||||
|
|
@ -827,7 +827,7 @@ class TcpConnectionSpec extends AkkaSpec("""
|
|||
}
|
||||
// dump the NACKs
|
||||
writer.receiveWhile(1.second) {
|
||||
case CommandFailed(write) => written -= 1
|
||||
case CommandFailed(_) => written -= 1
|
||||
}
|
||||
|
||||
// drain the queue until it works again
|
||||
|
|
|
|||
|
|
@ -12,7 +12,7 @@ import java.io.IOException
|
|||
import java.net.{ InetSocketAddress, ServerSocket }
|
||||
|
||||
import akka.testkit.WithLogCapturing
|
||||
import org.scalatest.concurrent.Timeouts
|
||||
import org.scalatest.concurrent.TimeLimits
|
||||
|
||||
import scala.concurrent.duration._
|
||||
import scala.language.postfixOps
|
||||
|
|
@ -22,7 +22,7 @@ class TcpIntegrationSpec extends AkkaSpec("""
|
|||
akka.loggers = ["akka.testkit.SilenceAllTestEventListener"]
|
||||
akka.io.tcp.trace-logging = on
|
||||
akka.actor.serialize-creators = on
|
||||
""") with TcpIntegrationSpecSupport with Timeouts with WithLogCapturing {
|
||||
""") with TcpIntegrationSpecSupport with TimeLimits with WithLogCapturing {
|
||||
|
||||
def verifyActorTermination(actor: ActorRef): Unit = {
|
||||
watch(actor)
|
||||
|
|
@ -188,7 +188,7 @@ class TcpIntegrationSpec extends AkkaSpec("""
|
|||
try {
|
||||
accept.getInputStream.read() should ===(-1)
|
||||
} catch {
|
||||
case e: IOException => // this is also fine
|
||||
case _: IOException => // this is also fine
|
||||
}
|
||||
}
|
||||
verifyActorTermination(connectionActor)
|
||||
|
|
|
|||
|
|
@ -47,7 +47,7 @@ class TcpListenerSpec extends AkkaSpec("""
|
|||
|
||||
expectWorkerForCommand
|
||||
expectWorkerForCommand
|
||||
selectorRouter.expectNoMsg(100.millis)
|
||||
selectorRouter.expectNoMessage(100.millis)
|
||||
interestCallReceiver.expectMsg(OP_ACCEPT)
|
||||
|
||||
// and pick up the last remaining connection on the next ChannelAcceptable
|
||||
|
|
@ -61,13 +61,13 @@ class TcpListenerSpec extends AkkaSpec("""
|
|||
attemptConnectionToEndpoint()
|
||||
listener ! ChannelAcceptable
|
||||
expectWorkerForCommand
|
||||
selectorRouter.expectNoMsg(100.millis)
|
||||
selectorRouter.expectNoMessage(100.millis)
|
||||
interestCallReceiver.expectMsg(OP_ACCEPT)
|
||||
|
||||
attemptConnectionToEndpoint()
|
||||
listener ! ChannelAcceptable
|
||||
expectWorkerForCommand
|
||||
selectorRouter.expectNoMsg(100.millis)
|
||||
selectorRouter.expectNoMessage(100.millis)
|
||||
interestCallReceiver.expectMsg(OP_ACCEPT)
|
||||
}
|
||||
|
||||
|
|
@ -75,16 +75,16 @@ class TcpListenerSpec extends AkkaSpec("""
|
|||
bindListener()
|
||||
|
||||
attemptConnectionToEndpoint()
|
||||
expectNoMsg(100.millis)
|
||||
expectNoMessage(100.millis)
|
||||
|
||||
listener ! ResumeAccepting(batchSize = 1)
|
||||
listener ! ChannelAcceptable
|
||||
expectWorkerForCommand
|
||||
selectorRouter.expectNoMsg(100.millis)
|
||||
selectorRouter.expectNoMessage(100.millis)
|
||||
interestCallReceiver.expectMsg(OP_ACCEPT)
|
||||
|
||||
// No more accepts are allowed now
|
||||
interestCallReceiver.expectNoMsg(100.millis)
|
||||
interestCallReceiver.expectNoMessage(100.millis)
|
||||
|
||||
listener ! ResumeAccepting(batchSize = 2)
|
||||
interestCallReceiver.expectMsg(OP_ACCEPT)
|
||||
|
|
@ -92,17 +92,17 @@ class TcpListenerSpec extends AkkaSpec("""
|
|||
attemptConnectionToEndpoint()
|
||||
listener ! ChannelAcceptable
|
||||
expectWorkerForCommand
|
||||
selectorRouter.expectNoMsg(100.millis)
|
||||
selectorRouter.expectNoMessage(100.millis)
|
||||
// There is still one token remaining, accepting
|
||||
interestCallReceiver.expectMsg(OP_ACCEPT)
|
||||
|
||||
attemptConnectionToEndpoint()
|
||||
listener ! ChannelAcceptable
|
||||
expectWorkerForCommand
|
||||
selectorRouter.expectNoMsg(100.millis)
|
||||
selectorRouter.expectNoMessage(100.millis)
|
||||
|
||||
// Tokens are depleted now
|
||||
interestCallReceiver.expectNoMsg(100.millis)
|
||||
interestCallReceiver.expectNoMessage(100.millis)
|
||||
}
|
||||
|
||||
"react to Unbind commands by replying with Unbound and stopping itself" in new TestSetup(pullMode = false) {
|
||||
|
|
|
|||
|
|
@ -88,7 +88,7 @@ class UdpConnectedIntegrationSpec extends AkkaSpec("""
|
|||
expectMsg(Udp.Unbound)
|
||||
|
||||
// Reusing the address
|
||||
val server2 = bindUdp(serverAddress, testActor)
|
||||
bindUdp(serverAddress, testActor)
|
||||
|
||||
client ! UdpConnected.Send(data1)
|
||||
expectMsgType[Udp.Received].data should ===(data1)
|
||||
|
|
|
|||
|
|
@ -22,7 +22,6 @@ class DnsClientSpec extends AkkaSpec with ImplicitSender {
|
|||
val exampleResponseMessage = Message(42, MessageFlags(answer = true))
|
||||
val exampleResponse = Answer(42, Nil)
|
||||
val dnsServerAddress = InetSocketAddress.createUnresolved("foo", 53)
|
||||
val localAddress = InetSocketAddress.createUnresolved("localhost", 13441)
|
||||
|
||||
"not connect to the DNS server over TCP eagerly" in {
|
||||
val udpExtensionProbe = TestProbe()
|
||||
|
|
|
|||
|
|
@ -4,6 +4,6 @@
|
|||
|
||||
package akka.japi
|
||||
|
||||
import org.scalatest.junit.JUnitSuiteLike
|
||||
import org.scalatestplus.junit.JUnitSuiteLike
|
||||
|
||||
class JavaAPITest extends JavaAPITestBase with JUnitSuiteLike
|
||||
|
|
|
|||
|
|
@ -7,13 +7,14 @@ package akka.pattern
|
|||
import akka.actor._
|
||||
import akka.testkit.{ AkkaSpec, TestProbe }
|
||||
import akka.util.Timeout
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
import scala.concurrent.Await
|
||||
import scala.concurrent.duration._
|
||||
import scala.util.Failure
|
||||
|
||||
import language.postfixOps
|
||||
|
||||
@silent
|
||||
class AskSpec extends AkkaSpec {
|
||||
|
||||
"The “ask” pattern" must {
|
||||
|
|
@ -25,7 +26,6 @@ class AskSpec extends AkkaSpec {
|
|||
}
|
||||
|
||||
"return broken promises on DeadLetters" in {
|
||||
implicit val timeout = Timeout(5 seconds)
|
||||
val dead = system.actorFor("/system/deadLetters")
|
||||
val f = dead.ask(42)(1 second)
|
||||
f.isCompleted should ===(true)
|
||||
|
|
@ -135,7 +135,7 @@ class AskSpec extends AkkaSpec {
|
|||
|
||||
Await.result(f, 1 seconds) should ===("hi")
|
||||
|
||||
deadListener.expectNoMsg(200 milliseconds)
|
||||
deadListener.expectNoMessage(200 milliseconds)
|
||||
}
|
||||
|
||||
"throw AskTimeoutException on using *" in {
|
||||
|
|
@ -183,7 +183,6 @@ class AskSpec extends AkkaSpec {
|
|||
val echo = system.actorOf(Props(new Actor {
|
||||
def receive = {
|
||||
case x =>
|
||||
val name = sender.path.name
|
||||
val parent = sender.path.parent
|
||||
context.actorSelection(parent / "missing") ! x
|
||||
}
|
||||
|
|
|
|||
|
|
@ -8,8 +8,11 @@ import java.util.concurrent.{ CountDownLatch, TimeUnit }
|
|||
|
||||
import akka.pattern.TestActor.NormalException
|
||||
import akka.testkit.{ filterException, AkkaSpec, ImplicitSender, TestProbe }
|
||||
|
||||
import scala.concurrent.duration._
|
||||
import akka.actor._
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
import scala.language.postfixOps
|
||||
|
||||
object TestActor {
|
||||
|
|
@ -51,6 +54,7 @@ class TestParentActor(probe: ActorRef, supervisorProps: Props) extends Actor {
|
|||
|
||||
class BackoffOnRestartSupervisorSpec extends AkkaSpec with ImplicitSender {
|
||||
|
||||
@silent
|
||||
def supervisorProps(probeRef: ActorRef) = {
|
||||
val options = Backoff
|
||||
.onFailure(TestActor.props(probeRef), "someChildName", 200 millis, 10 seconds, 0.0, maxNrOfRetries = -1)
|
||||
|
|
@ -105,7 +109,7 @@ class BackoffOnRestartSupervisorSpec extends AkkaSpec with ImplicitSender {
|
|||
val supervisorChildSelection = system.actorSelection(supervisor.path / "*")
|
||||
supervisorChildSelection.tell("testmsg", probe.ref)
|
||||
probe.expectMsg("testmsg")
|
||||
probe.expectNoMsg
|
||||
probe.expectNoMessage
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -140,6 +144,7 @@ class BackoffOnRestartSupervisorSpec extends AkkaSpec with ImplicitSender {
|
|||
|
||||
"accept commands while child is terminating" in {
|
||||
val postStopLatch = new CountDownLatch(1)
|
||||
@silent
|
||||
val options = Backoff
|
||||
.onFailure(
|
||||
Props(new SlowlyFailingActor(postStopLatch)),
|
||||
|
|
@ -151,6 +156,7 @@ class BackoffOnRestartSupervisorSpec extends AkkaSpec with ImplicitSender {
|
|||
.withSupervisorStrategy(OneForOneStrategy(loggingEnabled = false) {
|
||||
case _: TestActor.StoppingException => SupervisorStrategy.Stop
|
||||
})
|
||||
@silent
|
||||
val supervisor = system.actorOf(BackoffSupervisor.props(options))
|
||||
|
||||
supervisor ! BackoffSupervisor.GetCurrentChild
|
||||
|
|
@ -164,7 +170,7 @@ class BackoffOnRestartSupervisorSpec extends AkkaSpec with ImplicitSender {
|
|||
expectMsg("THROWN")
|
||||
|
||||
child ! "PING"
|
||||
expectNoMsg(100.millis) // Child is in limbo due to latch in postStop. There is no Terminated message yet
|
||||
expectNoMessage(100.millis) // Child is in limbo due to latch in postStop. There is no Terminated message yet
|
||||
|
||||
supervisor ! BackoffSupervisor.GetCurrentChild
|
||||
expectMsgType[BackoffSupervisor.CurrentChild].ref should ===(Some(child))
|
||||
|
|
@ -205,11 +211,13 @@ class BackoffOnRestartSupervisorSpec extends AkkaSpec with ImplicitSender {
|
|||
// withinTimeRange indicates the time range in which maxNrOfRetries will cause the child to
|
||||
// stop. IE: If we restart more than maxNrOfRetries in a time range longer than withinTimeRange
|
||||
// that is acceptable.
|
||||
@silent
|
||||
val options = Backoff
|
||||
.onFailure(TestActor.props(probe.ref), "someChildName", 300 millis, 10 seconds, 0.0, maxNrOfRetries = -1)
|
||||
.withSupervisorStrategy(OneForOneStrategy(withinTimeRange = 1 seconds, maxNrOfRetries = 3) {
|
||||
case _: TestActor.StoppingException => SupervisorStrategy.Stop
|
||||
})
|
||||
@silent
|
||||
val supervisor = system.actorOf(BackoffSupervisor.props(options))
|
||||
probe.expectMsg("STARTED")
|
||||
filterException[TestActor.TestException] {
|
||||
|
|
|
|||
|
|
@ -6,6 +6,7 @@ package akka.pattern
|
|||
|
||||
import akka.actor._
|
||||
import akka.testkit._
|
||||
import com.github.ghik.silencer.silent
|
||||
import org.scalatest.concurrent.Eventually
|
||||
import org.scalatest.prop.TableDrivenPropertyChecks._
|
||||
|
||||
|
|
@ -46,10 +47,13 @@ object BackoffSupervisorSpec {
|
|||
class BackoffSupervisorSpec extends AkkaSpec with ImplicitSender with Eventually {
|
||||
import BackoffSupervisorSpec._
|
||||
|
||||
@silent
|
||||
def onStopOptions(props: Props = Child.props(testActor), maxNrOfRetries: Int = -1) =
|
||||
Backoff.onStop(props, "c1", 100.millis, 3.seconds, 0.2, maxNrOfRetries)
|
||||
@silent
|
||||
def onFailureOptions(props: Props = Child.props(testActor), maxNrOfRetries: Int = -1) =
|
||||
Backoff.onFailure(props, "c1", 100.millis, 3.seconds, 0.2, maxNrOfRetries)
|
||||
@silent
|
||||
def create(options: BackoffOptions) = system.actorOf(BackoffSupervisor.props(options))
|
||||
|
||||
"BackoffSupervisor" must {
|
||||
|
|
@ -174,6 +178,7 @@ class BackoffSupervisorSpec extends AkkaSpec with ImplicitSender with Eventually
|
|||
|
||||
"reply to sender if replyWhileStopped is specified" in {
|
||||
filterException[TestException] {
|
||||
@silent
|
||||
val supervisor = create(
|
||||
Backoff
|
||||
.onFailure(Child.props(testActor), "c1", 100.seconds, 300.seconds, 0.2, maxNrOfRetries = -1)
|
||||
|
|
@ -199,6 +204,7 @@ class BackoffSupervisorSpec extends AkkaSpec with ImplicitSender with Eventually
|
|||
|
||||
"not reply to sender if replyWhileStopped is NOT specified" in {
|
||||
filterException[TestException] {
|
||||
@silent
|
||||
val supervisor =
|
||||
create(Backoff.onFailure(Child.props(testActor), "c1", 100.seconds, 300.seconds, 0.2, maxNrOfRetries = -1))
|
||||
supervisor ! BackoffSupervisor.GetCurrentChild
|
||||
|
|
@ -216,7 +222,7 @@ class BackoffSupervisorSpec extends AkkaSpec with ImplicitSender with Eventually
|
|||
}
|
||||
|
||||
supervisor ! "boom" //this will be sent to deadLetters
|
||||
expectNoMsg(500.milliseconds)
|
||||
expectNoMessage(500.milliseconds)
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -12,7 +12,7 @@ import scala.util.{ Failure, Success, Try }
|
|||
import akka.testkit._
|
||||
import org.mockito.ArgumentCaptor
|
||||
import org.scalatest.BeforeAndAfter
|
||||
import org.scalatest.mockito.MockitoSugar
|
||||
import org.scalatestplus.mockito.MockitoSugar
|
||||
import org.mockito.Mockito._
|
||||
|
||||
object CircuitBreakerSpec {
|
||||
|
|
@ -603,7 +603,6 @@ class CircuitBreakerSpec extends AkkaSpec with BeforeAndAfter with MockitoSugar
|
|||
|
||||
"pass through next call and invoke onCallBreakerOpen while executing other" in {
|
||||
val breaker = CircuitBreakerSpec.shortResetTimeoutCb()
|
||||
val captor = timeCaptor
|
||||
|
||||
breaker().withCircuitBreaker(Future(throwException))
|
||||
checkLatch(breaker.halfOpenLatch)
|
||||
|
|
|
|||
|
|
@ -80,7 +80,7 @@ class BalancingSpec extends AkkaSpec("""
|
|||
|
||||
// all but one worker are blocked
|
||||
val replies1 = receiveN(iterationCount - poolSize + 1)
|
||||
expectNoMsg(1.second)
|
||||
expectNoMessage(1.second)
|
||||
// all replies from the unblocked worker so far
|
||||
replies1.toSet should be(Set(1))
|
||||
|
||||
|
|
@ -88,7 +88,7 @@ class BalancingSpec extends AkkaSpec("""
|
|||
val replies2 = receiveN(poolSize - 1)
|
||||
// the remaining replies come from the blocked
|
||||
replies2.toSet should be((2 to poolSize).toSet)
|
||||
expectNoMsg(500.millis)
|
||||
expectNoMessage(500.millis)
|
||||
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -20,6 +20,7 @@ import akka.testkit.TestProbe
|
|||
import akka.actor.ExtendedActorSystem
|
||||
import akka.testkit.TestActors.echoActorProps
|
||||
import akka.actor.ActorPath
|
||||
import com.github.ghik.silencer.silent
|
||||
|
||||
object ConfiguredLocalRoutingSpec {
|
||||
val config = """
|
||||
|
|
@ -168,6 +169,7 @@ class ConfiguredLocalRoutingSpec
|
|||
"not get confused when trying to wildcard-configure children" in {
|
||||
system.actorOf(FromConfig.props(routeeProps = Props(classOf[SendRefAtStartup], testActor)), "weird")
|
||||
val recv = Set() ++ (for (_ <- 1 to 3) yield expectMsgType[ActorRef])
|
||||
@silent
|
||||
val expc = Set('a', 'b', 'c').map(i => system.actorFor("/user/weird/$" + i))
|
||||
recv should ===(expc)
|
||||
expectNoMessage(1 second)
|
||||
|
|
|
|||
|
|
@ -40,7 +40,7 @@ class RouteeCreationSpec extends AkkaSpec {
|
|||
val gotit = receiveWhile(messages = N) {
|
||||
case "two" => lastSender.toString
|
||||
}
|
||||
expectNoMsg(100.millis)
|
||||
expectNoMessage(100.millis)
|
||||
if (gotit.size != N) {
|
||||
fail(s"got only ${gotit.size} from [${gotit.mkString(", ")}]")
|
||||
}
|
||||
|
|
|
|||
|
|
@ -51,7 +51,7 @@ class RoutingSpec extends AkkaSpec(RoutingSpec.config) with DefaultTimeout with
|
|||
implicit val ec = system.dispatcher
|
||||
import RoutingSpec._
|
||||
|
||||
muteDeadLetters(classOf[akka.dispatch.sysmsg.DeathWatchNotification])()
|
||||
muteDeadLetters(classOf[akka.dispatch.sysmsg.DeathWatchNotification])(system)
|
||||
|
||||
"routers in general" must {
|
||||
|
||||
|
|
@ -95,7 +95,7 @@ class RoutingSpec extends AkkaSpec(RoutingSpec.config) with DefaultTimeout with
|
|||
routees.size should ===(2)
|
||||
routees.foreach { _.send(PoisonPill, testActor) }
|
||||
// expect no Terminated
|
||||
expectNoMsg(2.seconds)
|
||||
expectNoMessage(2.seconds)
|
||||
}
|
||||
|
||||
"use configured nr-of-instances when FromConfig" in {
|
||||
|
|
|
|||
|
|
@ -32,7 +32,7 @@ class DisabledJavaSerializerWarningSpec extends AkkaSpec("""
|
|||
EventFilter.warning(start = "Outgoing message attempted to use Java Serialization", occurrences = 1).intercept {
|
||||
val echo = system.actorOf(TestActors.echoActorProps)
|
||||
echo ! List("a")
|
||||
expectNoMsg(300.millis)
|
||||
expectNoMessage(300.millis)
|
||||
}
|
||||
|
||||
}
|
||||
|
|
|
|||
|
|
@ -170,7 +170,7 @@ class SerializationSetupSpec
|
|||
p.ref ! new ProgrammaticJavaDummy
|
||||
SerializationExtension(system).findSerializerFor(new ProgrammaticJavaDummy).toBinary(new ProgrammaticJavaDummy)
|
||||
// should not receive this one, it would have been java serialization!
|
||||
p.expectNoMsg(100.millis)
|
||||
p.expectNoMessage(100.millis)
|
||||
|
||||
p.ref ! new ProgrammaticDummy
|
||||
p.expectMsgType[ProgrammaticDummy]
|
||||
|
|
|
|||
|
|
@ -8,15 +8,20 @@ import language.postfixOps
|
|||
import akka.testkit.{ AkkaSpec, EventFilter }
|
||||
import akka.actor._
|
||||
import java.io._
|
||||
|
||||
import scala.concurrent.Await
|
||||
import akka.util.Timeout
|
||||
import akka.util.{ unused, Timeout }
|
||||
|
||||
import scala.concurrent.duration._
|
||||
import com.typesafe.config._
|
||||
import akka.pattern.ask
|
||||
import java.nio.ByteOrder
|
||||
import java.nio.ByteBuffer
|
||||
|
||||
import akka.actor.dungeon.SerializationCheckFailedException
|
||||
import com.github.ghik.silencer.silent
|
||||
import test.akka.serialization.NoVerification
|
||||
import SerializationTests._
|
||||
|
||||
object SerializationTests {
|
||||
|
||||
|
|
@ -51,8 +56,10 @@ object SerializationTests {
|
|||
|
||||
final case class Record(id: Int, person: Person)
|
||||
|
||||
@silent // can't use unused otherwise case class below gets a deprecated
|
||||
class SimpleMessage(s: String) extends TestSerializable
|
||||
|
||||
@silent
|
||||
class ExtendedSimpleMessage(s: String, i: Int) extends SimpleMessage(s)
|
||||
|
||||
trait AnotherInterface extends TestSerializable
|
||||
|
|
@ -67,7 +74,7 @@ object SerializationTests {
|
|||
|
||||
class BothTestSerializableAndJavaSerializable(s: String) extends SimpleMessage(s) with Serializable
|
||||
|
||||
class BothTestSerializableAndTestSerializable2(s: String) extends TestSerializable with TestSerializable2
|
||||
class BothTestSerializableAndTestSerializable2(@unused s: String) extends TestSerializable with TestSerializable2
|
||||
|
||||
trait A
|
||||
trait B
|
||||
|
|
@ -101,7 +108,7 @@ object SerializationTests {
|
|||
receiveBuilder().build()
|
||||
}
|
||||
|
||||
class NonSerializableActor(system: ActorSystem) extends Actor {
|
||||
class NonSerializableActor(@unused system: ActorSystem) extends Actor {
|
||||
def receive = {
|
||||
case s: String => sender() ! s
|
||||
}
|
||||
|
|
@ -133,13 +140,15 @@ object SerializationTests {
|
|||
}
|
||||
|
||||
class SerializeSpec extends AkkaSpec(SerializationTests.serializeConf) {
|
||||
import SerializationTests._
|
||||
|
||||
val ser = SerializationExtension(system)
|
||||
import ser._
|
||||
|
||||
val address = Address("120", "Monroe Street", "Santa Clara", "95050")
|
||||
val person = Person("debasish ghosh", 25, Address("120", "Monroe Street", "Santa Clara", "95050"))
|
||||
val address = SerializationTests.Address("120", "Monroe Street", "Santa Clara", "95050")
|
||||
val person = SerializationTests.Person(
|
||||
"debasish ghosh",
|
||||
25,
|
||||
SerializationTests.Address("120", "Monroe Street", "Santa Clara", "95050"))
|
||||
|
||||
"Serialization" must {
|
||||
|
||||
|
|
@ -151,7 +160,7 @@ class SerializeSpec extends AkkaSpec(SerializationTests.serializeConf) {
|
|||
}
|
||||
|
||||
"serialize Address" in {
|
||||
assert(deserialize(serialize(address).get, classOf[Address]).get === address)
|
||||
assert(deserialize(serialize(address).get, classOf[SerializationTests.Address]).get === address)
|
||||
}
|
||||
|
||||
"serialize Person" in {
|
||||
|
|
@ -225,8 +234,8 @@ class SerializeSpec extends AkkaSpec(SerializationTests.serializeConf) {
|
|||
|
||||
"give warning for message with several bindings" in {
|
||||
EventFilter.warning(start = "Multiple serializers found", occurrences = 1).intercept {
|
||||
ser.serializerFor(classOf[BothTestSerializableAndTestSerializable2]).getClass should (be(
|
||||
classOf[NoopSerializer]).or(be(classOf[NoopSerializer2])))
|
||||
ser.serializerFor(classOf[BothTestSerializableAndTestSerializable2]).getClass should be(classOf[NoopSerializer])
|
||||
.or(be(classOf[NoopSerializer2]))
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -290,7 +299,6 @@ class SerializeSpec extends AkkaSpec(SerializationTests.serializeConf) {
|
|||
}
|
||||
|
||||
class VerifySerializabilitySpec extends AkkaSpec(SerializationTests.verifySerializabilityConf) {
|
||||
import SerializationTests._
|
||||
implicit val timeout = Timeout(5 seconds)
|
||||
|
||||
"verify config" in {
|
||||
|
|
@ -306,7 +314,7 @@ class VerifySerializabilitySpec extends AkkaSpec(SerializationTests.verifySerial
|
|||
system.stop(b)
|
||||
|
||||
intercept[IllegalArgumentException] {
|
||||
val d = system.actorOf(Props(new NonSerializableActor(system)))
|
||||
system.actorOf(Props(new NonSerializableActor(system)))
|
||||
}
|
||||
|
||||
}
|
||||
|
|
@ -318,14 +326,13 @@ class VerifySerializabilitySpec extends AkkaSpec(SerializationTests.verifySerial
|
|||
EventFilter[SerializationCheckFailedException](
|
||||
start = "Failed to serialize and deserialize message of type java.lang.Object",
|
||||
occurrences = 1).intercept {
|
||||
a ! (new AnyRef)
|
||||
a ! new AnyRef
|
||||
}
|
||||
system.stop(a)
|
||||
}
|
||||
}
|
||||
|
||||
class ReferenceSerializationSpec extends AkkaSpec(SerializationTests.mostlyReferenceSystem) {
|
||||
import SerializationTests._
|
||||
|
||||
val ser = SerializationExtension(system)
|
||||
def serializerMustBe(toSerialize: Class[_], expectedSerializer: Class[_]) =
|
||||
|
|
|
|||
|
|
@ -351,11 +351,11 @@ class BoundedBlockingQueueSpec
|
|||
|
||||
val f = Future(queue.poll(100, TimeUnit.MILLISECONDS))
|
||||
|
||||
after(10 milliseconds) {
|
||||
after(10.milliseconds) {
|
||||
f.isCompleted should be(false)
|
||||
notEmpty.advanceTime(99 milliseconds)
|
||||
notEmpty.advanceTime(99.milliseconds)
|
||||
}
|
||||
mustBlockFor(100 milliseconds, f)
|
||||
mustBlockFor(100.milliseconds, f)
|
||||
events should contain(awaitNotEmpty)
|
||||
}
|
||||
|
||||
|
|
@ -367,7 +367,7 @@ class BoundedBlockingQueueSpec
|
|||
}
|
||||
|
||||
"return null if the timeout is exceeded" in {
|
||||
val TestContext(queue, _, notEmpty, _, _, _) = newBoundedBlockingQueue(1)
|
||||
val TestContext(queue, _, _, _, _, _) = newBoundedBlockingQueue(1)
|
||||
|
||||
queue.poll(100, TimeUnit.MILLISECONDS) should equal(null)
|
||||
}
|
||||
|
|
@ -379,12 +379,12 @@ class BoundedBlockingQueueSpec
|
|||
|
||||
val f = Future(queue.poll(100, TimeUnit.MILLISECONDS))
|
||||
|
||||
notEmpty.advanceTime(99 milliseconds)
|
||||
after(50 milliseconds) {
|
||||
notEmpty.advanceTime(99.milliseconds)
|
||||
after(50.milliseconds) {
|
||||
f.isCompleted should be(false)
|
||||
queue.put("Hello")
|
||||
}
|
||||
Await.result(f, 3 seconds) should equal("Hello")
|
||||
Await.result(f, 3.seconds) should equal("Hello")
|
||||
(events should contain).inOrder(awaitNotEmpty, signalNotEmpty, poll)
|
||||
}
|
||||
}
|
||||
|
|
@ -620,7 +620,7 @@ trait CustomContainsMatcher {
|
|||
case (_, Nil) => matchResult(true)
|
||||
case (Nil, _) => matchResult(false)
|
||||
case (x :: xs, y :: ys) if x.equals(y) => attemptMatch(xs, ys)
|
||||
case (x :: xs, ys) => attemptMatch(xs, ys)
|
||||
case (_ :: xs, ys) => attemptMatch(xs, ys)
|
||||
}
|
||||
|
||||
def matchResult(success: Boolean): MatchResult =
|
||||
|
|
|
|||
|
|
@ -11,17 +11,18 @@ import java.nio.{ ByteBuffer, ByteOrder }
|
|||
import java.nio.ByteOrder.{ BIG_ENDIAN, LITTLE_ENDIAN }
|
||||
|
||||
import akka.util.ByteString.{ ByteString1, ByteString1C, ByteStrings }
|
||||
import com.github.ghik.silencer.silent
|
||||
import org.apache.commons.codec.binary.Hex.encodeHex
|
||||
import org.scalacheck.Arbitrary.arbitrary
|
||||
import org.scalacheck.{ Arbitrary, Gen }
|
||||
import org.scalatest.{ Matchers, WordSpec }
|
||||
import org.scalatest.prop.Checkers
|
||||
import org.scalatestplus.scalacheck.Checkers
|
||||
|
||||
import scala.collection.mutable.Builder
|
||||
|
||||
class ByteStringSpec extends WordSpec with Matchers with Checkers {
|
||||
|
||||
implicit val betterGeneratorDrivenConfig = PropertyCheckConfig().copy(minSuccessful = 1000)
|
||||
implicit val betterGeneratorDrivenConfig = PropertyCheckConfiguration().copy(minSuccessful = 1000)
|
||||
|
||||
def genSimpleByteString(min: Int, max: Int) =
|
||||
for {
|
||||
|
|
@ -157,6 +158,7 @@ class ByteStringSpec extends WordSpec with Matchers with Checkers {
|
|||
body(bsA, bsB) == body(vecA, vecB)
|
||||
}
|
||||
|
||||
@silent
|
||||
def likeVecIt(bs: ByteString)(body: BufferedIterator[Byte] => Any, strict: Boolean = true): Boolean = {
|
||||
val bsIterator = bs.iterator
|
||||
val vecIterator = Vector(bs: _*).iterator.buffered
|
||||
|
|
@ -164,6 +166,7 @@ class ByteStringSpec extends WordSpec with Matchers with Checkers {
|
|||
(!strict || (bsIterator.toSeq == vecIterator.toSeq))
|
||||
}
|
||||
|
||||
@silent
|
||||
def likeVecIts(a: ByteString, b: ByteString)(
|
||||
body: (BufferedIterator[Byte], BufferedIterator[Byte]) => Any,
|
||||
strict: Boolean = true): Boolean = {
|
||||
|
|
@ -1191,7 +1194,7 @@ class ByteStringSpec extends WordSpec with Matchers with Checkers {
|
|||
bs3.foreach { b =>
|
||||
builder += b
|
||||
}
|
||||
builder ++= Vector(array2: _*)
|
||||
builder ++= array2.toIndexedSeq
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -4,6 +4,6 @@
|
|||
|
||||
package akka.util
|
||||
|
||||
import org.scalatest.junit.JUnitSuiteLike
|
||||
import org.scalatestplus.junit.JUnitSuiteLike
|
||||
|
||||
class JavaDurationSpec extends JavaDuration with JUnitSuiteLike
|
||||
|
|
|
|||
|
|
@ -10,8 +10,10 @@ import language.postfixOps
|
|||
import akka.actor.Actor
|
||||
import akka.actor.ActorRef
|
||||
import akka.actor.Props
|
||||
import akka.util.unused
|
||||
import testkit.MultiNodeConfig
|
||||
import com.typesafe.config.ConfigFactory
|
||||
|
||||
import scala.concurrent.duration._
|
||||
|
||||
class NewRemoteActorMultiJvmSpec(artery: Boolean) extends MultiNodeConfig {
|
||||
|
|
@ -46,7 +48,7 @@ object NewRemoteActorSpec {
|
|||
}
|
||||
}
|
||||
|
||||
class SomeActorWithParam(ignored: String) extends Actor {
|
||||
class SomeActorWithParam(@unused ignored: String) extends Actor {
|
||||
def receive = {
|
||||
case "identify" => sender() ! self
|
||||
}
|
||||
|
|
|
|||
|
|
@ -66,7 +66,7 @@ abstract class PiercingShouldKeepQuarantineSpec(multiNodeConfig: PiercingShouldK
|
|||
// Quarantine is up -- Should not be able to communicate with remote system any more
|
||||
for (_ <- 1 to 4) {
|
||||
system.actorSelection(node(second) / "user" / "subject") ! "getuid"
|
||||
expectNoMsg(2.seconds)
|
||||
expectNoMessage(2.seconds)
|
||||
}
|
||||
|
||||
enterBarrier("quarantine-intact")
|
||||
|
|
|
|||
|
|
@ -102,7 +102,7 @@ abstract class RemoteNodeShutdownAndComesBackSpec extends RemotingMultiNodeSpec(
|
|||
val p = TestProbe()
|
||||
system.actorSelection(RootActorPath(secondAddress) / "user" / "subject").tell(Identify("subject"), p.ref)
|
||||
p.expectMsgPF(1 second) {
|
||||
case ActorIdentity("subject", Some(ref)) => true
|
||||
case ActorIdentity("subject", Some(_)) => true
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
@ -127,7 +127,6 @@ abstract class RemoteNodeShutdownAndComesBackSpec extends RemotingMultiNodeSpec(
|
|||
val address = system.asInstanceOf[ExtendedActorSystem].provider.getDefaultAddress
|
||||
system.actorOf(Props[Subject], "subject")
|
||||
system.actorOf(Props[Subject], "sysmsgBarrier")
|
||||
val path = node(first)
|
||||
enterBarrier("actors-started")
|
||||
|
||||
enterBarrier("watch-established")
|
||||
|
|
|
|||
|
|
@ -77,7 +77,7 @@ abstract class RemoteQuarantinePiercingSpec(multiNodeConfig: RemoteQuarantinePie
|
|||
|
||||
// Quarantine is up -- Cannot communicate with remote system any more
|
||||
system.actorSelection(RootActorPath(secondAddress) / "user" / "subject") ! "identify"
|
||||
expectNoMsg(2.seconds)
|
||||
expectNoMessage(2.seconds)
|
||||
|
||||
// Shut down the other system -- which results in restart (see runOn(second))
|
||||
Await.result(testConductor.shutdown(second), 30.seconds)
|
||||
|
|
|
|||
|
|
@ -124,7 +124,7 @@ abstract class RemoteReDeploymentMultiJvmSpec(multiNodeConfig: RemoteReDeploymen
|
|||
|
||||
"terminate the child when its parent system is replaced by a new one" in {
|
||||
// Any message sent to `echo` will be passed on to `testActor`
|
||||
val echo = system.actorOf(echoProps(testActor), "echo")
|
||||
system.actorOf(echoProps(testActor), "echo")
|
||||
enterBarrier("echo-started")
|
||||
|
||||
runOn(second) {
|
||||
|
|
@ -150,8 +150,8 @@ abstract class RemoteReDeploymentMultiJvmSpec(multiNodeConfig: RemoteReDeploymen
|
|||
within(sleepAfterKill) {
|
||||
// The quarantine of node 2, where the Parent lives, should cause the Hello child to be stopped:
|
||||
expectMsg("PostStop")
|
||||
expectNoMsg()
|
||||
} else expectNoMsg(sleepAfterKill)
|
||||
expectNoMessage()
|
||||
} else expectNoMessage(sleepAfterKill)
|
||||
awaitAssert(node(second), 10.seconds, 100.millis)
|
||||
}
|
||||
|
||||
|
|
@ -160,7 +160,7 @@ abstract class RemoteReDeploymentMultiJvmSpec(multiNodeConfig: RemoteReDeploymen
|
|||
// Start the second system again
|
||||
runOn(second) {
|
||||
Await.ready(system.whenTerminated, 30.seconds)
|
||||
expectNoMsg(sleepAfterKill)
|
||||
expectNoMessage(sleepAfterKill)
|
||||
sys = startNewSystem()
|
||||
}
|
||||
|
||||
|
|
@ -205,7 +205,7 @@ abstract class RemoteReDeploymentMultiJvmSpec(multiNodeConfig: RemoteReDeploymen
|
|||
enterBarrier("the-end")
|
||||
|
||||
// After this we expect no further messages
|
||||
expectNoMsg(1.second)
|
||||
expectNoMessage(1.second)
|
||||
|
||||
// Until we clean up after ourselves
|
||||
enterBarrier("stopping")
|
||||
|
|
|
|||
|
|
@ -40,7 +40,7 @@ object RemoteRestartedQuarantinedSpec extends MultiNodeConfig {
|
|||
class Subject extends Actor {
|
||||
def receive = {
|
||||
case "shutdown" => context.system.terminate()
|
||||
case "identify" => sender() ! (AddressUidExtension(context.system).addressUid -> self)
|
||||
case "identify" => sender() ! (AddressUidExtension(context.system).longAddressUid -> self)
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -70,7 +70,7 @@ abstract class RemoteRestartedQuarantinedSpec extends RemotingMultiNodeSpec(Remo
|
|||
runOn(first) {
|
||||
val secondAddress = node(second).address
|
||||
|
||||
val (uid, ref) = identifyWithUid(second, "subject")
|
||||
val (uid, _) = identifyWithUid(second, "subject")
|
||||
|
||||
RARP(system).provider.transport.quarantine(node(second).address, Some(uid), "test")
|
||||
|
||||
|
|
@ -108,7 +108,7 @@ abstract class RemoteRestartedQuarantinedSpec extends RemotingMultiNodeSpec(Remo
|
|||
}
|
||||
|
||||
expectMsgPF(10 seconds) {
|
||||
case ThisActorSystemQuarantinedEvent(local, remote) =>
|
||||
case ThisActorSystemQuarantinedEvent(_, _) =>
|
||||
}
|
||||
|
||||
enterBarrier("still-quarantined")
|
||||
|
|
|
|||
|
|
@ -53,7 +53,7 @@ abstract class Ticket15109Spec extends RemotingMultiNodeSpec(Ticket15109Spec) {
|
|||
|
||||
def identify(role: RoleName, actorName: String): ActorRef = {
|
||||
system.actorSelection(node(role) / "user" / actorName) ! Identify(0)
|
||||
expectMsgType[ActorIdentity](5.seconds).getRef
|
||||
expectMsgType[ActorIdentity](5.seconds).getActorRef.get
|
||||
}
|
||||
|
||||
def ping(ref: ActorRef) = {
|
||||
|
|
|
|||
|
|
@ -7,7 +7,6 @@ package akka.remote
|
|||
import java.util.concurrent.atomic.AtomicBoolean
|
||||
|
||||
import scala.concurrent.duration._
|
||||
|
||||
import akka.actor.Actor
|
||||
import akka.actor.ActorIdentity
|
||||
import akka.actor.ActorRef
|
||||
|
|
@ -18,6 +17,7 @@ import akka.event.EventStream
|
|||
import akka.remote.testconductor.RoleName
|
||||
import akka.remote.testkit.MultiNodeConfig
|
||||
import akka.testkit._
|
||||
import akka.util.unused
|
||||
import com.typesafe.config.Config
|
||||
import com.typesafe.config.ConfigFactory
|
||||
|
||||
|
|
@ -57,7 +57,7 @@ object TransportFailSpec {
|
|||
private val fdAvailable = new AtomicBoolean(true)
|
||||
|
||||
// FD that will fail when `fdAvailable` flag is false
|
||||
class TestFailureDetector(config: Config, ev: EventStream) extends FailureDetector {
|
||||
class TestFailureDetector(@unused config: Config, @unused ev: EventStream) extends FailureDetector {
|
||||
@volatile private var active = false
|
||||
|
||||
override def heartbeat(): Unit = {
|
||||
|
|
@ -110,7 +110,6 @@ abstract class TransportFailSpec extends RemotingMultiNodeSpec(TransportFailConf
|
|||
|
||||
"reconnect" taggedAs LongRunningTest in {
|
||||
runOn(first) {
|
||||
val secondAddress = node(second).address
|
||||
enterBarrier("actors-started")
|
||||
|
||||
val subject = identify(second, "subject")
|
||||
|
|
@ -144,7 +143,7 @@ abstract class TransportFailSpec extends RemotingMultiNodeSpec(TransportFailConf
|
|||
}
|
||||
}, max = 5.seconds)
|
||||
watch(subject2)
|
||||
quarantineProbe.expectNoMsg(1.seconds)
|
||||
quarantineProbe.expectNoMessage(1.seconds)
|
||||
subject2 ! "hello2"
|
||||
expectMsg("hello2")
|
||||
enterBarrier("watch-established2")
|
||||
|
|
|
|||
|
|
@ -141,7 +141,6 @@ abstract class FanInThroughputSpec extends RemotingMultiNodeSpec(FanInThroughput
|
|||
|
||||
runOn(sendingNodes: _*) {
|
||||
enterBarrier(receiverName + "-started")
|
||||
val ignore = TestProbe()
|
||||
val receivers = (1 to sendingNodes.size)
|
||||
.map { n =>
|
||||
identifyReceiver(receiverName + "-" + n, roles.head)
|
||||
|
|
|
|||
|
|
@ -138,9 +138,8 @@ abstract class FanOutThroughputSpec extends RemotingMultiNodeSpec(FanOutThroughp
|
|||
|
||||
runOn(roles.head) {
|
||||
enterBarrier(receiverName + "-started")
|
||||
val ignore = TestProbe()
|
||||
val receivers = targetNodes.map(target => identifyReceiver(receiverName, target)).toArray[Target]
|
||||
val senders = for ((target, i) <- targetNodes.zipWithIndex) yield {
|
||||
val senders = for ((_, i) <- targetNodes.zipWithIndex) yield {
|
||||
val receiver = receivers(i)
|
||||
val plotProbe = TestProbe()
|
||||
val snd = system.actorOf(
|
||||
|
|
|
|||
|
|
@ -121,7 +121,7 @@ object LatencySpec extends MultiNodeConfig {
|
|||
def receiveMessage(size: Int): Unit = {
|
||||
if (count == 0)
|
||||
startTime = System.nanoTime()
|
||||
reporter.onMessage(1, payloadSize)
|
||||
reporter.onMessage(1, size)
|
||||
count += 1
|
||||
val d = System.nanoTime() - sendTimes.get(count - 1)
|
||||
try {
|
||||
|
|
@ -135,14 +135,13 @@ object LatencySpec extends MultiNodeConfig {
|
|||
}
|
||||
}
|
||||
if (count == totalMessages) {
|
||||
printTotal(testName, size, histogram, System.nanoTime() - startTime, BenchmarkFileReporter)
|
||||
printTotal(testName, histogram, System.nanoTime() - startTime, BenchmarkFileReporter)
|
||||
context.stop(self)
|
||||
}
|
||||
}
|
||||
|
||||
def printTotal(
|
||||
testName: String,
|
||||
payloadSize: Long,
|
||||
histogram: Histogram,
|
||||
totalDurationNanos: Long,
|
||||
reporter: BenchmarkFileReporter): Unit = {
|
||||
|
|
@ -281,7 +280,7 @@ abstract class LatencySpec extends RemotingMultiNodeSpec(LatencySpec) {
|
|||
receiverProps(rep, testSettings, totalMessages, sendTimes, histogram, plotProbe.ref, BenchmarkFileReporter))
|
||||
|
||||
// warmup for 3 seconds to init compression
|
||||
val warmup = Source(1 to 30).throttle(10, 1.second, 10, ThrottleMode.Shaping).runForeach { n =>
|
||||
val warmup = Source(1 to 30).throttle(10, 1.second, 10, ThrottleMode.Shaping).runForeach { _ =>
|
||||
echo.tell(Array.emptyByteArray, receiver)
|
||||
}
|
||||
|
||||
|
|
@ -309,16 +308,16 @@ abstract class LatencySpec extends RemotingMultiNodeSpec(LatencySpec) {
|
|||
items = Vector(TestMessage.Item(1, "A"), TestMessage.Item(2, "B")))
|
||||
else payload
|
||||
|
||||
echo.tell(payload, receiver)
|
||||
echo.tell(msg, receiver)
|
||||
i += 1
|
||||
}
|
||||
|
||||
// measure rate and adjust for next repeat round
|
||||
val d = (sendTimes.get(totalMessages - 1) - sendTimes.get(0))
|
||||
val d = sendTimes.get(totalMessages - 1) - sendTimes.get(0)
|
||||
val measuredRate = totalMessages * SECONDS.toNanos(1) / math.max(1, d)
|
||||
val previousTargetRate = messageRate * adjustRateFactor
|
||||
adjustRateFactor = (previousTargetRate / math.max(1, measuredRate))
|
||||
println(s"Measured send rate $measuredRate msg/s (new adjustment facor: $adjustRateFactor)")
|
||||
adjustRateFactor = previousTargetRate / math.max(1, measuredRate)
|
||||
println(s"Measured send rate $measuredRate msg/s (new adjustment factor: $adjustRateFactor)")
|
||||
}
|
||||
|
||||
watch(receiver)
|
||||
|
|
|
|||
|
|
@ -124,7 +124,7 @@ object MaxThroughputSpec extends MultiNodeConfig {
|
|||
if (msg.length != payloadSize) throw new IllegalArgumentException("Invalid message")
|
||||
|
||||
report()
|
||||
case msg: TestMessage =>
|
||||
case _: TestMessage =>
|
||||
report()
|
||||
|
||||
case Start(corresponding) =>
|
||||
|
|
@ -236,7 +236,7 @@ object MaxThroughputSpec extends MultiNodeConfig {
|
|||
}
|
||||
|
||||
def active: Receive = {
|
||||
case c @ FlowControl(id, t0) =>
|
||||
case _ @FlowControl(id, t0) =>
|
||||
val targetCount = pendingFlowControl(id)
|
||||
if (targetCount - 1 == 0) {
|
||||
pendingFlowControl -= id
|
||||
|
|
@ -275,7 +275,7 @@ object MaxThroughputSpec extends MultiNodeConfig {
|
|||
plotRef ! PlotResult().add(testName, throughput * payloadSize * testSettings.senderReceiverPairs / 1024 / 1024)
|
||||
context.stop(self)
|
||||
|
||||
case c: ReceivedActorRefCompressionTable =>
|
||||
case _: ReceivedActorRefCompressionTable =>
|
||||
}
|
||||
|
||||
val sent = new Array[Long](targets.size)
|
||||
|
|
@ -352,7 +352,7 @@ object MaxThroughputSpec extends MultiNodeConfig {
|
|||
}
|
||||
|
||||
override def toBinary(o: AnyRef): Array[Byte] = o match {
|
||||
case FlowControl(id, burstStartTime) =>
|
||||
case FlowControl(_, _) =>
|
||||
val buf = ByteBuffer.allocate(12)
|
||||
toBinary(o, buf)
|
||||
buf.flip()
|
||||
|
|
@ -468,7 +468,6 @@ abstract class MaxThroughputSpec extends RemotingMultiNodeSpec(MaxThroughputSpec
|
|||
|
||||
runOn(first) {
|
||||
enterBarrier(receiverName + "-started")
|
||||
val ignore = TestProbe()
|
||||
val receivers = (for (n <- 1 to senderReceiverPairs) yield identifyReceiver(receiverName + n)).toArray
|
||||
val senders = for (n <- 1 to senderReceiverPairs) yield {
|
||||
val receiver = receivers(n - 1)
|
||||
|
|
|
|||
|
|
@ -64,7 +64,7 @@ abstract class RemoteRestartedQuarantinedSpec extends RemotingMultiNodeSpec(Remo
|
|||
runOn(first) {
|
||||
val secondAddress = node(second).address
|
||||
|
||||
val (uid, ref) = identifyWithUid(second, "subject", 5.seconds)
|
||||
val (uid, _) = identifyWithUid(second, "subject", 5.seconds)
|
||||
|
||||
enterBarrier("before-quarantined")
|
||||
RARP(system).provider.transport.quarantine(node(second).address, Some(uid), "test")
|
||||
|
|
@ -89,13 +89,13 @@ abstract class RemoteRestartedQuarantinedSpec extends RemotingMultiNodeSpec(Remo
|
|||
val firstAddress = node(first).address
|
||||
system.eventStream.subscribe(testActor, classOf[ThisActorSystemQuarantinedEvent])
|
||||
|
||||
val (firstUid, ref) = identifyWithUid(first, "subject", 5.seconds)
|
||||
val (firstUid, _) = identifyWithUid(first, "subject", 5.seconds)
|
||||
|
||||
enterBarrier("before-quarantined")
|
||||
enterBarrier("quarantined")
|
||||
|
||||
expectMsgPF(10 seconds) {
|
||||
case ThisActorSystemQuarantinedEvent(local, remote) =>
|
||||
case ThisActorSystemQuarantinedEvent(_, _) =>
|
||||
}
|
||||
|
||||
// check that we quarantine back
|
||||
|
|
|
|||
|
|
@ -10,7 +10,6 @@ import akka.actor.Identify
|
|||
import akka.remote.{ RARP, RemotingMultiNodeSpec }
|
||||
import akka.remote.testkit.MultiNodeConfig
|
||||
import akka.testkit._
|
||||
import akka.util.Timeout
|
||||
import com.typesafe.config.ConfigFactory
|
||||
import org.scalatest.concurrent.ScalaFutures
|
||||
|
||||
|
|
@ -76,7 +75,6 @@ abstract class SurviveInboundStreamRestartWithCompressionInFlightSpec
|
|||
"Decompression table" must {
|
||||
|
||||
import scala.concurrent.duration._
|
||||
implicit val timeout = Timeout(10.seconds)
|
||||
|
||||
"be kept even if inbound lane is restarted, and decode into correct actors still" taggedAs LongRunningTest in {
|
||||
val probeA = TestProbe()
|
||||
|
|
|
|||
|
|
@ -60,7 +60,7 @@ abstract class SurviveNetworkPartitionSpec extends RemotingMultiNodeSpec(Survive
|
|||
// send system message during network partition
|
||||
watch(ref)
|
||||
// keep the network partition for a while, but shorter than give-up-system-message-after
|
||||
expectNoMsg(RARP(system).provider.remoteSettings.Artery.Advanced.GiveUpSystemMessageAfter - 2.second)
|
||||
expectNoMessage(RARP(system).provider.remoteSettings.Artery.Advanced.GiveUpSystemMessageAfter - 2.second)
|
||||
|
||||
// heal the network partition
|
||||
testConductor.passThrough(first, second, Direction.Both).await
|
||||
|
|
@ -97,7 +97,7 @@ abstract class SurviveNetworkPartitionSpec extends RemotingMultiNodeSpec(Survive
|
|||
// send system message during network partition
|
||||
watch(ref)
|
||||
// keep the network partition for a while, longer than give-up-system-message-after
|
||||
expectNoMsg(RARP(system).provider.remoteSettings.Artery.Advanced.GiveUpSystemMessageAfter - 1.second)
|
||||
expectNoMessage(RARP(system).provider.remoteSettings.Artery.Advanced.GiveUpSystemMessageAfter - 1.second)
|
||||
qProbe.expectMsgType[QuarantinedEvent](5.seconds).address should ===(node(second).address)
|
||||
|
||||
expectTerminated(ref)
|
||||
|
|
|
|||
|
|
@ -10,7 +10,6 @@ import java.util.concurrent.atomic.AtomicInteger
|
|||
|
||||
import scala.concurrent.Await
|
||||
import scala.concurrent.duration._
|
||||
import scala.util.{ Failure, Success }
|
||||
|
||||
import akka.Done
|
||||
import akka.actor.ExtendedActorSystem
|
||||
|
|
|
|||
|
|
@ -142,12 +142,7 @@ abstract class AeronStreamLatencySpec
|
|||
super.afterAll()
|
||||
}
|
||||
|
||||
def printTotal(
|
||||
testName: String,
|
||||
payloadSize: Long,
|
||||
histogram: Histogram,
|
||||
totalDurationNanos: Long,
|
||||
lastRepeat: Boolean): Unit = {
|
||||
def printTotal(testName: String, histogram: Histogram, totalDurationNanos: Long, lastRepeat: Boolean): Unit = {
|
||||
def percentile(p: Double): Double = histogram.getValueAtPercentile(p) / 1000.0
|
||||
val throughput = 1000.0 * histogram.getTotalCount / totalDurationNanos.nanos.toMillis
|
||||
|
||||
|
|
@ -217,7 +212,7 @@ abstract class AeronStreamLatencySpec
|
|||
histogram.recordValue(d)
|
||||
if (c == totalMessages) {
|
||||
val totalDurationNanos = System.nanoTime() - startTime.get
|
||||
printTotal(testName, bytes.length, histogram, totalDurationNanos, lastRepeat.get)
|
||||
printTotal(testName, histogram, totalDurationNanos, lastRepeat.get)
|
||||
barrier.await() // this is always the last party
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -10,7 +10,6 @@ import java.util.concurrent.Executors
|
|||
import scala.collection.AbstractIterator
|
||||
import scala.concurrent.Await
|
||||
import scala.concurrent.duration._
|
||||
import scala.util.{ Failure, Success }
|
||||
|
||||
import akka.actor._
|
||||
import akka.remote.testconductor.RoleName
|
||||
|
|
@ -202,10 +201,9 @@ abstract class AeronStreamMaxThroughputSpec
|
|||
enterBarrier(receiverName + "-started")
|
||||
|
||||
val payload = ("0" * payloadSize).getBytes("utf-8")
|
||||
val t0 = System.nanoTime()
|
||||
Source
|
||||
.fromIterator(() => iterate(1, totalMessages))
|
||||
.map { n =>
|
||||
.map { _ =>
|
||||
val envelope = pool.acquire()
|
||||
envelope.byteBuffer.put(payload)
|
||||
envelope.byteBuffer.flip()
|
||||
|
|
|
|||
|
|
@ -79,7 +79,7 @@ class RemoteRandomSpec(multiNodeConfig: RemoteRandomConfig)
|
|||
val connectionCount = 3
|
||||
val iterationCount = 100
|
||||
|
||||
for (i <- 0 until iterationCount; k <- 0 until connectionCount) {
|
||||
for (_ <- 0 until iterationCount; _ <- 0 until connectionCount) {
|
||||
actor ! "hit"
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -96,7 +96,7 @@ class RemoteRoundRobinSpec(multiNodeConfig: RemoteRoundRobinConfig)
|
|||
val connectionCount = 3
|
||||
val iterationCount = 10
|
||||
|
||||
for (i <- 0 until iterationCount; k <- 0 until connectionCount) {
|
||||
for (_ <- 0 until iterationCount; _ <- 0 until connectionCount) {
|
||||
actor ! "hit"
|
||||
}
|
||||
|
||||
|
|
@ -184,7 +184,7 @@ class RemoteRoundRobinSpec(multiNodeConfig: RemoteRoundRobinConfig)
|
|||
val connectionCount = 3
|
||||
val iterationCount = 10
|
||||
|
||||
for (i <- 0 until iterationCount; k <- 0 until connectionCount) {
|
||||
for (_ <- 0 until iterationCount; _ <- 0 until connectionCount) {
|
||||
actor ! "hit"
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -91,7 +91,7 @@ class RemoteScatterGatherSpec(multiNodeConfig: RemoteScatterGatherConfig)
|
|||
// let them start
|
||||
Thread.sleep(2000)
|
||||
|
||||
for (i <- 0 until iterationCount; k <- 0 until connectionCount) {
|
||||
for (_ <- 0 until iterationCount; _ <- 0 until connectionCount) {
|
||||
actor ! "hit"
|
||||
}
|
||||
|
||||
|
|
|
|||
|
|
@ -48,9 +48,9 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
val b = getBarrier()
|
||||
b ! NodeInfo(A, AddressFromURIString("akka://sys"), system.deadLetters)
|
||||
b ! ClientDisconnected(B)
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
b ! ClientDisconnected(A)
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
}
|
||||
|
||||
"fail entering barrier when nobody registered" taggedAs TimingTest in {
|
||||
|
|
@ -106,7 +106,7 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
b.expectMsg(ToClient(BarrierResult("bar4", true)))
|
||||
}
|
||||
barrier ! ClientDisconnected(C)
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
}
|
||||
|
||||
"leave barrier when last “arrived” is removed" taggedAs TimingTest in {
|
||||
|
|
@ -132,8 +132,8 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
}
|
||||
val msg = expectMsgType[Failed]
|
||||
msg match {
|
||||
case Failed(barrier, thr: ClientLost)
|
||||
if (thr == ClientLost(Data(Set(nodeA), "bar6", a.ref :: Nil, thr.data.deadline), B)) =>
|
||||
case Failed(_, thr: ClientLost)
|
||||
if thr == ClientLost(Data(Set(nodeA), "bar6", a.ref :: Nil, thr.data.deadline), B) =>
|
||||
case x =>
|
||||
fail(
|
||||
"Expected " + Failed(barrier, ClientLost(Data(Set(nodeA), "bar6", a.ref :: Nil, null), B)) + " but got " + x)
|
||||
|
|
@ -155,8 +155,8 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
}
|
||||
val msg = expectMsgType[Failed]
|
||||
msg match {
|
||||
case Failed(barrier, thr: ClientLost)
|
||||
if (thr == ClientLost(Data(Set(nodeA, nodeC), "bar7", a.ref :: Nil, thr.data.deadline), B)) =>
|
||||
case Failed(_, thr: ClientLost)
|
||||
if thr == ClientLost(Data(Set(nodeA, nodeC), "bar7", a.ref :: Nil, thr.data.deadline), B) =>
|
||||
case x =>
|
||||
fail(
|
||||
"Expected " + Failed(barrier, ClientLost(Data(Set(nodeA, nodeC), "bar7", a.ref :: Nil, null), B)) + " but got " + x)
|
||||
|
|
@ -176,8 +176,8 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
}
|
||||
val msg = expectMsgType[Failed]
|
||||
msg match {
|
||||
case Failed(barrier, thr: WrongBarrier)
|
||||
if (thr == WrongBarrier("foo", b.ref, Data(Set(nodeA, nodeB), "bar8", a.ref :: Nil, thr.data.deadline))) =>
|
||||
case Failed(_, thr: WrongBarrier)
|
||||
if thr == WrongBarrier("foo", b.ref, Data(Set(nodeA, nodeB), "bar8", a.ref :: Nil, thr.data.deadline)) =>
|
||||
case x =>
|
||||
fail(
|
||||
"Expected " + Failed(
|
||||
|
|
@ -194,10 +194,10 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
}
|
||||
val msg = expectMsgType[Failed]
|
||||
msg match {
|
||||
case Failed(barrier, thr: BarrierEmpty)
|
||||
if (thr == BarrierEmpty(
|
||||
case Failed(_, thr: BarrierEmpty)
|
||||
if thr == BarrierEmpty(
|
||||
Data(Set(), "", Nil, thr.data.deadline),
|
||||
"cannot remove RoleName(a): no client to remove")) =>
|
||||
"cannot remove RoleName(a): no client to remove") =>
|
||||
case x =>
|
||||
fail("Expected " + Failed(
|
||||
barrier,
|
||||
|
|
@ -219,8 +219,8 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
EventFilter[BarrierTimeout](occurrences = 1).intercept {
|
||||
val msg = expectMsgType[Failed](7 seconds)
|
||||
msg match {
|
||||
case Failed(barrier, thr: BarrierTimeout)
|
||||
if (thr == BarrierTimeout(Data(Set(nodeA, nodeB), "bar10", a.ref :: Nil, thr.data.deadline))) =>
|
||||
case Failed(_, thr: BarrierTimeout)
|
||||
if thr == BarrierTimeout(Data(Set(nodeA, nodeB), "bar10", a.ref :: Nil, thr.data.deadline)) =>
|
||||
case x =>
|
||||
fail(
|
||||
"Expected " + Failed(barrier, BarrierTimeout(Data(Set(nodeA, nodeB), "bar10", a.ref :: Nil, null))) + " but got " + x)
|
||||
|
|
@ -239,15 +239,15 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
}
|
||||
val msg = expectMsgType[Failed]
|
||||
msg match {
|
||||
case Failed(barrier, thr: DuplicateNode)
|
||||
if (thr == DuplicateNode(Data(Set(nodeA), "", Nil, thr.data.deadline), nodeB)) =>
|
||||
case Failed(_, thr: DuplicateNode)
|
||||
if thr == DuplicateNode(Data(Set(nodeA), "", Nil, thr.data.deadline), nodeB) =>
|
||||
case x =>
|
||||
fail("Expected " + Failed(barrier, DuplicateNode(Data(Set(nodeA), "", Nil, null), nodeB)) + " but got " + x)
|
||||
}
|
||||
}
|
||||
|
||||
"finally have no failure messages left" taggedAs TimingTest in {
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
}
|
||||
|
||||
}
|
||||
|
|
@ -272,9 +272,9 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
b ! NodeInfo(A, AddressFromURIString("akka://sys"), testActor)
|
||||
expectMsg(ToClient(Done))
|
||||
b ! ClientDisconnected(B)
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
b ! ClientDisconnected(A)
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -342,7 +342,7 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
b.expectMsg(ToClient(BarrierResult("bar13", true)))
|
||||
}
|
||||
barrier ! ClientDisconnected(C)
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -478,7 +478,7 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
EventFilter[FailedBarrier](occurrences = 1).intercept {
|
||||
b.send(barrier, FailBarrier("bar20"))
|
||||
a.expectMsg(ToClient(BarrierResult("bar20", false)))
|
||||
b.expectNoMsg(1 second)
|
||||
b.expectNoMessage(1 second)
|
||||
}
|
||||
a.send(barrier, EnterBarrier("bar21", None))
|
||||
b.send(barrier, EnterBarrier("bar21", None))
|
||||
|
|
@ -536,12 +536,12 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
}
|
||||
|
||||
"finally have no failure messages left" taggedAs TimingTest in {
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
private def withController(participants: Int)(f: (ActorRef) => Unit): Unit = {
|
||||
private def withController(participants: Int)(f: ActorRef => Unit): Unit = {
|
||||
system.actorOf(Props(new Actor {
|
||||
val controller =
|
||||
context.actorOf(Props(classOf[Controller], participants, new InetSocketAddress(InetAddress.getLocalHost, 0)))
|
||||
|
|
@ -550,7 +550,7 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
case x => testActor ! Failed(controller, x); SupervisorStrategy.Restart
|
||||
}
|
||||
def receive = {
|
||||
case x: InetSocketAddress => testActor ! controller
|
||||
case _: InetSocketAddress => testActor ! controller
|
||||
}
|
||||
}).withDeploy(Deploy.local))
|
||||
val actor = expectMsgType[ActorRef]
|
||||
|
|
@ -576,15 +576,8 @@ class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender {
|
|||
}
|
||||
|
||||
private def noMsg(probes: TestProbe*): Unit = {
|
||||
expectNoMsg(1 second)
|
||||
expectNoMessage(1 second)
|
||||
probes.foreach(_.msgAvailable should ===(false))
|
||||
}
|
||||
|
||||
private def data(
|
||||
clients: Set[Controller.NodeInfo],
|
||||
barrier: String,
|
||||
arrived: List[ActorRef],
|
||||
previous: Data): Data = {
|
||||
Data(clients, barrier, arrived, previous.deadline)
|
||||
}
|
||||
}
|
||||
|
|
|
|||
|
|
@ -30,7 +30,9 @@ object AkkaDisciplinePlugin extends AutoPlugin with ScalafixSupport {
|
|||
"akka-cluster-sharding",
|
||||
"akka-stream",
|
||||
"akka-cluster-metrics",
|
||||
"akka-slf4j")
|
||||
"akka-slf4j",
|
||||
"akka-remote-tests",
|
||||
"akka-actor-tests")
|
||||
|
||||
val strictProjects = Set("akka-discovery", "akka-protobuf", "akka-coordination")
|
||||
|
||||
|
|
@ -116,5 +118,4 @@ object AkkaDisciplinePlugin extends AutoPlugin with ScalafixSupport {
|
|||
"-Ypartial-unification",
|
||||
"-Ywarn-extra-implicit")
|
||||
|
||||
|
||||
}
|
||||
|
|
|
|||
Loading…
Add table
Add a link
Reference in a new issue