Merging with latest master
This commit is contained in:
commit
ea1817b6d8
39 changed files with 7022 additions and 218 deletions
|
|
@ -0,0 +1,77 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.actor
|
||||
|
||||
import akka.testkit._
|
||||
import akka.testkit.DefaultTimeout
|
||||
import akka.testkit.TestEvent._
|
||||
import akka.util.duration._
|
||||
import akka.routing._
|
||||
import org.scalatest.BeforeAndAfterEach
|
||||
import akka.ConfigurationException
|
||||
|
||||
object ActorConfigurationVerificationSpec {
|
||||
|
||||
class TestActor extends Actor {
|
||||
def receive: Receive = {
|
||||
case _ ⇒
|
||||
}
|
||||
}
|
||||
|
||||
val config = """
|
||||
balancing-dispatcher {
|
||||
type = BalancingDispatcher
|
||||
throughput = 1
|
||||
}
|
||||
pinned-dispatcher {
|
||||
executor = "thread-pool-executor"
|
||||
type = PinnedDispatcher
|
||||
}
|
||||
"""
|
||||
}
|
||||
|
||||
@org.junit.runner.RunWith(classOf[org.scalatest.junit.JUnitRunner])
|
||||
class ActorConfigurationVerificationSpec extends AkkaSpec(ActorConfigurationVerificationSpec.config) with DefaultTimeout with BeforeAndAfterEach {
|
||||
import ActorConfigurationVerificationSpec._
|
||||
|
||||
override def atStartup {
|
||||
system.eventStream.publish(Mute(EventFilter[ConfigurationException]("")))
|
||||
}
|
||||
|
||||
"An Actor configured with a BalancingDispatcher" must {
|
||||
"fail verification with a ConfigurationException if also configured with a RoundRobinRouter" in {
|
||||
intercept[ConfigurationException] {
|
||||
system.actorOf(Props[TestActor].withDispatcher("balancing-dispatcher").withRouter(RoundRobinRouter(2)))
|
||||
}
|
||||
}
|
||||
"fail verification with a ConfigurationException if also configured with a BroadcastRouter" in {
|
||||
intercept[ConfigurationException] {
|
||||
system.actorOf(Props[TestActor].withDispatcher("balancing-dispatcher").withRouter(BroadcastRouter(2)))
|
||||
}
|
||||
}
|
||||
"fail verification with a ConfigurationException if also configured with a RandomRouter" in {
|
||||
intercept[ConfigurationException] {
|
||||
system.actorOf(Props[TestActor].withDispatcher("balancing-dispatcher").withRouter(RandomRouter(2)))
|
||||
}
|
||||
}
|
||||
"fail verification with a ConfigurationException if also configured with a SmallestMailboxRouter" in {
|
||||
intercept[ConfigurationException] {
|
||||
system.actorOf(Props[TestActor].withDispatcher("balancing-dispatcher").withRouter(SmallestMailboxRouter(2)))
|
||||
}
|
||||
}
|
||||
"fail verification with a ConfigurationException if also configured with a ScatterGatherFirstCompletedRouter" in {
|
||||
intercept[ConfigurationException] {
|
||||
system.actorOf(Props[TestActor].withDispatcher("balancing-dispatcher").withRouter(ScatterGatherFirstCompletedRouter(nrOfInstances = 2, within = 2 seconds)))
|
||||
}
|
||||
}
|
||||
"not fail verification with a ConfigurationException also not configured with a Router" in {
|
||||
system.actorOf(Props[TestActor].withDispatcher("balancing-dispatcher"))
|
||||
}
|
||||
}
|
||||
"An Actor configured with a non-balancing dispatcher" must {
|
||||
"not fail verification with a ConfigurationException if also configured with a Router" in {
|
||||
system.actorOf(Props[TestActor].withDispatcher("pinned-dispatcher").withRouter(RoundRobinRouter(2)))
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
@ -128,35 +128,6 @@ class ResizerSpec extends AkkaSpec(ResizerSpec.config) with DefaultTimeout with
|
|||
current.routees.size must be(2)
|
||||
}
|
||||
|
||||
"resize when busy" in {
|
||||
|
||||
val busy = new TestLatch(1)
|
||||
|
||||
val resizer = DefaultResizer(
|
||||
lowerBound = 1,
|
||||
upperBound = 3,
|
||||
pressureThreshold = 0,
|
||||
messagesPerResize = 1)
|
||||
|
||||
val router = system.actorOf(Props[BusyActor].withRouter(RoundRobinRouter(resizer = Some(resizer))).withDispatcher("bal-disp"))
|
||||
|
||||
val latch1 = new TestLatch(1)
|
||||
router ! (latch1, busy)
|
||||
Await.ready(latch1, 2 seconds)
|
||||
|
||||
val latch2 = new TestLatch(1)
|
||||
router ! (latch2, busy)
|
||||
Await.ready(latch2, 2 seconds)
|
||||
|
||||
val latch3 = new TestLatch(1)
|
||||
router ! (latch3, busy)
|
||||
Await.ready(latch3, 2 seconds)
|
||||
|
||||
Await.result(router ? CurrentRoutees, 5 seconds).asInstanceOf[RouterRoutees].routees.size must be(3)
|
||||
|
||||
busy.countDown()
|
||||
}
|
||||
|
||||
"grow as needed under pressure" in {
|
||||
// make sure the pool starts at the expected lower limit and grows to the upper as needed
|
||||
// as influenced by the backlog of blocking pooled actors
|
||||
|
|
|
|||
|
|
@ -127,6 +127,8 @@ class Dispatchers(val settings: ActorSystem.Settings, val prerequisites: Dispatc
|
|||
*/
|
||||
private[akka] def from(cfg: Config): MessageDispatcher = configuratorFrom(cfg).dispatcher()
|
||||
|
||||
private[akka] def isBalancingDispatcher(id: String): Boolean = settings.config.hasPath(id) && config(id).getString("type") == "BalancingDispatcher"
|
||||
|
||||
/**
|
||||
* Creates a MessageDispatcherConfigurator from a Config.
|
||||
*
|
||||
|
|
|
|||
|
|
@ -647,7 +647,7 @@ object Logging {
|
|||
import java.text.SimpleDateFormat
|
||||
import java.util.Date
|
||||
|
||||
private val dateFormat = new SimpleDateFormat("MM/dd/yyyy HH:mm:ss.S")
|
||||
private val dateFormat = new SimpleDateFormat("MM/dd/yyyy HH:mm:ss.SSS")
|
||||
private val errorFormat = "[ERROR] [%s] [%s] [%s] %s\n%s".intern
|
||||
private val errorFormatWithoutCause = "[ERROR] [%s] [%s] [%s] %s".intern
|
||||
private val warningFormat = "[WARN] [%s] [%s] [%s] %s".intern
|
||||
|
|
|
|||
|
|
@ -29,6 +29,12 @@ private[akka] class RoutedActorRef(_system: ActorSystemImpl, _props: Props, _sup
|
|||
_supervisor,
|
||||
_path) {
|
||||
|
||||
// verify that a BalancingDispatcher is not used with a Router
|
||||
if (_system.dispatchers.isBalancingDispatcher(_props.dispatcher) && _props.routerConfig != NoRouter)
|
||||
throw new ConfigurationException(
|
||||
"Configuration for actor [" + _path.toString +
|
||||
"] is invalid - you can not use a 'BalancingDispatcher' together with any type of 'Router'")
|
||||
|
||||
/*
|
||||
* CAUTION: RoutedActorRef is PROBLEMATIC
|
||||
* ======================================
|
||||
|
|
|
|||
|
|
@ -0,0 +1,91 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.cluster
|
||||
|
||||
import com.typesafe.config.ConfigFactory
|
||||
import org.scalatest.BeforeAndAfter
|
||||
import akka.remote.testkit.MultiNodeConfig
|
||||
import akka.remote.testkit.MultiNodeSpec
|
||||
import akka.testkit._
|
||||
|
||||
object NodeStartupMultiJvmSpec extends MultiNodeConfig {
|
||||
val first = role("first")
|
||||
val second = role("second")
|
||||
|
||||
commonConfig(debugConfig(on = false))
|
||||
|
||||
nodeConfig(first, ConfigFactory.parseString("""
|
||||
# FIXME get rid of this hardcoded port
|
||||
akka.remote.netty.port=2601
|
||||
"""))
|
||||
|
||||
nodeConfig(second, ConfigFactory.parseString("""
|
||||
# FIXME get rid of this hardcoded host:port
|
||||
akka.cluster.node-to-join = "akka://MultiNodeSpec@localhost:2601"
|
||||
"""))
|
||||
|
||||
}
|
||||
|
||||
class NodeStartupMultiJvmNode1 extends NodeStartupSpec
|
||||
class NodeStartupMultiJvmNode2 extends NodeStartupSpec
|
||||
|
||||
class NodeStartupSpec extends MultiNodeSpec(NodeStartupMultiJvmSpec) with ImplicitSender with BeforeAndAfter {
|
||||
import NodeStartupMultiJvmSpec._
|
||||
|
||||
override def initialParticipants = 2
|
||||
|
||||
var firstNode: Cluster = _
|
||||
|
||||
after {
|
||||
testConductor.enter("after")
|
||||
}
|
||||
|
||||
runOn(first) {
|
||||
firstNode = Cluster(system)
|
||||
}
|
||||
|
||||
"A first cluster node with a 'node-to-join' config set to empty string (singleton cluster)" must {
|
||||
|
||||
"be a singleton cluster when started up" in {
|
||||
runOn(first) {
|
||||
awaitCond(firstNode.isSingletonCluster)
|
||||
// FIXME #2117 singletonCluster should reach convergence
|
||||
//awaitCond(firstNode.convergence.isDefined)
|
||||
}
|
||||
}
|
||||
|
||||
"be in 'Joining' phase when started up" in {
|
||||
runOn(first) {
|
||||
val members = firstNode.latestGossip.members
|
||||
members.size must be(1)
|
||||
val firstAddress = testConductor.getAddressFor(first).await
|
||||
val joiningMember = members find (_.address == firstAddress)
|
||||
joiningMember must not be (None)
|
||||
joiningMember.get.status must be(MemberStatus.Joining)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
"A second cluster node with a 'node-to-join' config defined" must {
|
||||
"join the other node cluster when sending a Join command" in {
|
||||
runOn(second) {
|
||||
// start cluster on second node, and join
|
||||
val secondNode = Cluster(system)
|
||||
awaitCond(secondNode.convergence.isDefined)
|
||||
}
|
||||
|
||||
runOn(first) {
|
||||
val secondAddress = testConductor.getAddressFor(second).await
|
||||
awaitCond {
|
||||
firstNode.latestGossip.members.exists { member ⇒
|
||||
member.address == secondAddress && member.status == MemberStatus.Up
|
||||
}
|
||||
}
|
||||
firstNode.latestGossip.members.size must be(2)
|
||||
awaitCond(firstNode.convergence.isDefined)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
}
|
||||
|
|
@ -1,84 +0,0 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.cluster
|
||||
|
||||
import java.net.InetSocketAddress
|
||||
|
||||
import akka.testkit._
|
||||
import akka.dispatch._
|
||||
import akka.actor._
|
||||
import akka.remote._
|
||||
import akka.util.duration._
|
||||
|
||||
import com.typesafe.config._
|
||||
|
||||
class NodeStartupSpec extends ClusterSpec with ImplicitSender {
|
||||
val portPrefix = 8
|
||||
|
||||
var node0: Cluster = _
|
||||
var node1: Cluster = _
|
||||
var system0: ActorSystemImpl = _
|
||||
var system1: ActorSystemImpl = _
|
||||
|
||||
try {
|
||||
"A first cluster node with a 'node-to-join' config set to empty string (singleton cluster)" must {
|
||||
system0 = ActorSystem("system0", ConfigFactory
|
||||
.parseString("""
|
||||
akka {
|
||||
actor.provider = "akka.remote.RemoteActorRefProvider"
|
||||
remote.netty.port=%d550
|
||||
}""".format(portPrefix))
|
||||
.withFallback(system.settings.config))
|
||||
.asInstanceOf[ActorSystemImpl]
|
||||
val remote0 = system0.provider.asInstanceOf[RemoteActorRefProvider]
|
||||
node0 = Cluster(system0)
|
||||
|
||||
"be a singleton cluster when started up" taggedAs LongRunningTest in {
|
||||
Thread.sleep(1.seconds.dilated.toMillis)
|
||||
node0.isSingletonCluster must be(true)
|
||||
}
|
||||
|
||||
"be in 'Joining' phase when started up" taggedAs LongRunningTest in {
|
||||
val members = node0.latestGossip.members
|
||||
val joiningMember = members find (_.address.port.get == 550.withPortPrefix)
|
||||
joiningMember must be('defined)
|
||||
joiningMember.get.status must be(MemberStatus.Joining)
|
||||
}
|
||||
}
|
||||
|
||||
"A second cluster node with a 'node-to-join' config defined" must {
|
||||
"join the other node cluster when sending a Join command" taggedAs LongRunningTest in {
|
||||
system1 = ActorSystem("system1", ConfigFactory
|
||||
.parseString("""
|
||||
akka {
|
||||
actor.provider = "akka.remote.RemoteActorRefProvider"
|
||||
remote.netty.port=%d551
|
||||
cluster.node-to-join = "akka://system0@localhost:%d550"
|
||||
}""".format(portPrefix, portPrefix))
|
||||
.withFallback(system.settings.config))
|
||||
.asInstanceOf[ActorSystemImpl]
|
||||
val remote1 = system1.provider.asInstanceOf[RemoteActorRefProvider]
|
||||
node1 = Cluster(system1)
|
||||
|
||||
Thread.sleep(10.seconds.dilated.toMillis) // give enough time for node1 to JOIN node0 and leader to move him to UP
|
||||
val members = node0.latestGossip.members
|
||||
val joiningMember = members find (_.address.port.get == 551.withPortPrefix)
|
||||
joiningMember must be('defined)
|
||||
joiningMember.get.status must be(MemberStatus.Up)
|
||||
}
|
||||
}
|
||||
} catch {
|
||||
case e: Exception ⇒
|
||||
e.printStackTrace
|
||||
fail(e.toString)
|
||||
}
|
||||
|
||||
override def atTermination() {
|
||||
if (node0 ne null) node0.shutdown()
|
||||
if (system0 ne null) system0.shutdown()
|
||||
|
||||
if (node1 ne null) node1.shutdown()
|
||||
if (system1 ne null) system1.shutdown()
|
||||
}
|
||||
}
|
||||
|
|
@ -334,3 +334,10 @@ same machine at the same time.
|
|||
The machines that are used for testing (slaves) should have ssh access to the outside world and be able to talk
|
||||
to each other with the internal addresses given. On the master machine ssh client is required. Obviosly git
|
||||
and sbt should be installed on both master and slave machines.
|
||||
|
||||
The Test Conductor Extension
|
||||
============================
|
||||
|
||||
The Test Conductor Extension is aimed at enhancing the multi JVM and multi node testing facilities.
|
||||
|
||||
.. image:: ../images/akka-remote-testconductor.png
|
||||
|
|
|
|||
BIN
akka-docs/images/akka-remote-testconductor.png
Normal file
BIN
akka-docs/images/akka-remote-testconductor.png
Normal file
Binary file not shown.
|
After Width: | Height: | Size: 18 KiB |
|
|
@ -85,6 +85,8 @@ There are 4 different types of message dispatchers:
|
|||
"thread-pool-executor" or the FQCN of
|
||||
an ``akka.dispatcher.ExecutorServiceConfigurator``
|
||||
|
||||
- Note that you can **not** use a ``BalancingDispatcher`` together with any kind of ``Router``, trying to do so will make your actor fail verification.
|
||||
|
||||
* CallingThreadDispatcher
|
||||
|
||||
- This dispatcher runs invocations on the current thread only. This dispatcher does not create any new threads,
|
||||
|
|
|
|||
|
|
@ -380,7 +380,8 @@ The dispatcher for created children of the router will be taken from
|
|||
makes sense to configure the :class:`BalancingDispatcher` if the precise
|
||||
routing is not so important (i.e. no consistent hashing or round-robin is
|
||||
required); this enables newly created routees to pick up work immediately by
|
||||
stealing it from their siblings.
|
||||
stealing it from their siblings. Note that you can **not** use a ``BalancingDispatcher``
|
||||
together with any kind of ``Router``, trying to do so will make your actor fail verification.
|
||||
|
||||
The “head” router, of course, cannot run on the same balancing dispatcher,
|
||||
because it does not process the same messages, hence this special actor does
|
||||
|
|
|
|||
|
|
@ -86,6 +86,8 @@ There are 4 different types of message dispatchers:
|
|||
"thread-pool-executor" or the FQCN of
|
||||
an ``akka.dispatcher.ExecutorServiceConfigurator``
|
||||
|
||||
- Note that you can **not** use a ``BalancingDispatcher`` together with any kind of ``Router``, trying to do so will make your actor fail verification.
|
||||
|
||||
* CallingThreadDispatcher
|
||||
|
||||
- This dispatcher runs invocations on the current thread only. This dispatcher does not create any new threads,
|
||||
|
|
|
|||
|
|
@ -380,7 +380,9 @@ The dispatcher for created children of the router will be taken from
|
|||
makes sense to configure the :class:`BalancingDispatcher` if the precise
|
||||
routing is not so important (i.e. no consistent hashing or round-robin is
|
||||
required); this enables newly created routees to pick up work immediately by
|
||||
stealing it from their siblings.
|
||||
stealing it from their siblings. Note that you can **not** use a ``BalancingDispatcher``
|
||||
together with any kind of ``Router``, trying to do so will make your actor fail verification.
|
||||
|
||||
|
||||
.. note::
|
||||
|
||||
|
|
|
|||
File diff suppressed because it is too large
Load diff
|
|
@ -0,0 +1,62 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
option java_package = "akka.remote.testconductor";
|
||||
option optimize_for = SPEED;
|
||||
|
||||
/******************************************
|
||||
Compile with:
|
||||
cd ./akka-remote/src/main/protocol
|
||||
protoc TestConductorProtocol.proto --java_out ../java
|
||||
*******************************************/
|
||||
|
||||
message Wrapper {
|
||||
optional Hello hello = 1;
|
||||
optional EnterBarrier barrier = 2;
|
||||
optional InjectFailure failure = 3;
|
||||
optional string done = 4;
|
||||
optional AddressRequest addr = 5;
|
||||
}
|
||||
|
||||
message Hello {
|
||||
required string name = 1;
|
||||
required Address address = 2;
|
||||
}
|
||||
|
||||
message EnterBarrier {
|
||||
required string name = 1;
|
||||
optional bool status = 2;
|
||||
}
|
||||
|
||||
message AddressRequest {
|
||||
required string node = 1;
|
||||
optional Address addr = 2;
|
||||
}
|
||||
|
||||
message Address {
|
||||
required string protocol = 1;
|
||||
required string system = 2;
|
||||
required string host = 3;
|
||||
required int32 port = 4;
|
||||
}
|
||||
|
||||
enum FailType {
|
||||
Throttle = 1;
|
||||
Disconnect = 2;
|
||||
Abort = 3;
|
||||
Shutdown = 4;
|
||||
}
|
||||
enum Direction {
|
||||
Send = 1;
|
||||
Receive = 2;
|
||||
Both = 3;
|
||||
}
|
||||
message InjectFailure {
|
||||
required FailType failure = 1;
|
||||
optional Direction direction = 2;
|
||||
optional Address address = 3;
|
||||
optional float rateMBit = 6;
|
||||
optional int32 exitValue = 7;
|
||||
}
|
||||
|
||||
33
akka-remote-tests/src/main/resources/reference.conf
Normal file
33
akka-remote-tests/src/main/resources/reference.conf
Normal file
|
|
@ -0,0 +1,33 @@
|
|||
#############################################
|
||||
# Akka Remote Testing Reference Config File #
|
||||
#############################################
|
||||
|
||||
# This is the reference config file that contains all the default settings.
|
||||
# Make your edits/overrides in your application.conf.
|
||||
|
||||
akka {
|
||||
testconductor {
|
||||
|
||||
# Timeout for joining a barrier: this is the maximum time any participants
|
||||
# waits for everybody else to join a named barrier.
|
||||
barrier-timeout = 30s
|
||||
|
||||
# Timeout for interrogation of TestConductor’s Controller actor
|
||||
query-timeout = 5s
|
||||
|
||||
# Threshold for packet size in time unit above which the failure injector will
|
||||
# split the packet and deliver in smaller portions; do not give value smaller
|
||||
# than HashedWheelTimer resolution (would not make sense)
|
||||
packet-split-threshold = 100ms
|
||||
|
||||
# amount of time for the ClientFSM to wait for the connection to the conductor
|
||||
# to be successful
|
||||
connect-timeout = 20s
|
||||
|
||||
# Number of connect attempts to be made to the conductor controller
|
||||
client-reconnects = 10
|
||||
|
||||
# minimum time interval which is to be inserted between reconnect attempts
|
||||
reconnect-backoff = 1s
|
||||
}
|
||||
}
|
||||
|
|
@ -0,0 +1,566 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.remote.testconductor
|
||||
|
||||
import akka.actor.{ Actor, ActorRef, ActorSystem, LoggingFSM, Props }
|
||||
import RemoteConnection.getAddrString
|
||||
import TestConductorProtocol._
|
||||
import org.jboss.netty.channel.{ Channel, SimpleChannelUpstreamHandler, ChannelHandlerContext, ChannelStateEvent, MessageEvent }
|
||||
import com.typesafe.config.ConfigFactory
|
||||
import akka.util.Timeout
|
||||
import akka.util.Duration
|
||||
import akka.util.duration._
|
||||
import akka.pattern.ask
|
||||
import java.util.concurrent.TimeUnit.MILLISECONDS
|
||||
import akka.dispatch.Await
|
||||
import akka.event.LoggingAdapter
|
||||
import akka.actor.PoisonPill
|
||||
import akka.event.Logging
|
||||
import scala.util.control.NoStackTrace
|
||||
import akka.event.LoggingReceive
|
||||
import akka.actor.Address
|
||||
import java.net.InetSocketAddress
|
||||
import akka.dispatch.Future
|
||||
import akka.actor.OneForOneStrategy
|
||||
import akka.actor.SupervisorStrategy
|
||||
import java.util.concurrent.ConcurrentHashMap
|
||||
import akka.actor.Status
|
||||
|
||||
sealed trait Direction {
|
||||
def includes(other: Direction): Boolean
|
||||
}
|
||||
|
||||
object Direction {
|
||||
case object Send extends Direction {
|
||||
override def includes(other: Direction): Boolean = other match {
|
||||
case Send ⇒ true
|
||||
case _ ⇒ false
|
||||
}
|
||||
}
|
||||
case object Receive extends Direction {
|
||||
override def includes(other: Direction): Boolean = other match {
|
||||
case Receive ⇒ true
|
||||
case _ ⇒ false
|
||||
}
|
||||
}
|
||||
case object Both extends Direction {
|
||||
override def includes(other: Direction): Boolean = true
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* The conductor is the one orchestrating the test: it governs the
|
||||
* [[akka.remote.testconductor.Controller]]’s port to which all
|
||||
* [[akka.remote.testconductor.Player]]s connect, it issues commands to their
|
||||
* [[akka.remote.testconductor.NetworkFailureInjector]] and provides support
|
||||
* for barriers using the [[akka.remote.testconductor.BarrierCoordinator]].
|
||||
* All of this is bundled inside the [[akka.remote.testconductor.TestConductorExt]]
|
||||
* extension.
|
||||
*/
|
||||
trait Conductor { this: TestConductorExt ⇒
|
||||
|
||||
import Controller._
|
||||
|
||||
private var _controller: ActorRef = _
|
||||
private def controller: ActorRef = _controller match {
|
||||
case null ⇒ throw new IllegalStateException("TestConductorServer was not started")
|
||||
case x ⇒ x
|
||||
}
|
||||
|
||||
/**
|
||||
* Start the [[akka.remote.testconductor.Controller]], which in turn will
|
||||
* bind to a TCP port as specified in the `akka.testconductor.port` config
|
||||
* property, where 0 denotes automatic allocation. Since the latter is
|
||||
* actually preferred, a `Future[Int]` is returned which will be completed
|
||||
* with the port number actually chosen, so that this can then be communicated
|
||||
* to the players for their proper start-up.
|
||||
*
|
||||
* This method also invokes [[akka.remote.testconductor.Player]].startClient,
|
||||
* since it is expected that the conductor participates in barriers for
|
||||
* overall coordination. The returned Future will only be completed once the
|
||||
* client’s start-up finishes, which in fact waits for all other players to
|
||||
* connect.
|
||||
*
|
||||
* @param participants gives the number of participants which shall connect
|
||||
* before any of their startClient() operations complete.
|
||||
*/
|
||||
def startController(participants: Int, name: RoleName, controllerPort: InetSocketAddress): Future[InetSocketAddress] = {
|
||||
if (_controller ne null) throw new RuntimeException("TestConductorServer was already started")
|
||||
_controller = system.actorOf(Props(new Controller(participants, controllerPort)), "controller")
|
||||
import Settings.BarrierTimeout
|
||||
controller ? GetSockAddr flatMap { case sockAddr: InetSocketAddress ⇒ startClient(name, sockAddr) map (_ ⇒ sockAddr) }
|
||||
}
|
||||
|
||||
/**
|
||||
* Obtain the port to which the controller’s socket is actually bound. This
|
||||
* will deviate from the configuration in `akka.testconductor.port` in case
|
||||
* that was given as zero.
|
||||
*/
|
||||
def sockAddr: Future[InetSocketAddress] = {
|
||||
import Settings.QueryTimeout
|
||||
controller ? GetSockAddr mapTo
|
||||
}
|
||||
|
||||
/**
|
||||
* Make the remoting pipeline on the node throttle data sent to or received
|
||||
* from the given remote peer. Throttling works by delaying packet submission
|
||||
* within the netty pipeline until the packet would have been completely sent
|
||||
* according to the given rate, the previous packet completion and the current
|
||||
* packet length. In case of large packets they are split up if the calculated
|
||||
* send pause would exceed `akka.testconductor.packet-split-threshold`
|
||||
* (roughly). All of this uses the system’s HashedWheelTimer, which is not
|
||||
* terribly precise and will execute tasks later than they are schedule (even
|
||||
* on average), but that is countered by using the actual execution time for
|
||||
* determining how much to send, leading to the correct output rate, but with
|
||||
* increased latency.
|
||||
*
|
||||
* @param node is the symbolic name of the node which is to be affected
|
||||
* @param target is the symbolic name of the other node to which connectivity shall be throttled
|
||||
* @param direction can be either `Direction.Send`, `Direction.Receive` or `Direction.Both`
|
||||
* @param rateMBit is the maximum data rate in MBit
|
||||
*/
|
||||
def throttle(node: RoleName, target: RoleName, direction: Direction, rateMBit: Double): Future[Done] = {
|
||||
import Settings.QueryTimeout
|
||||
controller ? Throttle(node, target, direction, rateMBit.toFloat) mapTo
|
||||
}
|
||||
|
||||
/**
|
||||
* Switch the Netty pipeline of the remote support into blackhole mode for
|
||||
* sending and/or receiving: it will just drop all messages right before
|
||||
* submitting them to the Socket or right after receiving them from the
|
||||
* Socket.
|
||||
*
|
||||
* @param node is the symbolic name of the node which is to be affected
|
||||
* @param target is the symbolic name of the other node to which connectivity shall be impeded
|
||||
* @param direction can be either `Direction.Send`, `Direction.Receive` or `Direction.Both`
|
||||
*/
|
||||
def blackhole(node: RoleName, target: RoleName, direction: Direction): Future[Done] = {
|
||||
import Settings.QueryTimeout
|
||||
controller ? Throttle(node, target, direction, 0f) mapTo
|
||||
}
|
||||
|
||||
/**
|
||||
* Tell the remote support to shutdown the connection to the given remote
|
||||
* peer. It works regardless of whether the recipient was initiator or
|
||||
* responder.
|
||||
*
|
||||
* @param node is the symbolic name of the node which is to be affected
|
||||
* @param target is the symbolic name of the other node to which connectivity shall be impeded
|
||||
*/
|
||||
def disconnect(node: RoleName, target: RoleName): Future[Done] = {
|
||||
import Settings.QueryTimeout
|
||||
controller ? Disconnect(node, target, false) mapTo
|
||||
}
|
||||
|
||||
/**
|
||||
* Tell the remote support to TCP_RESET the connection to the given remote
|
||||
* peer. It works regardless of whether the recipient was initiator or
|
||||
* responder.
|
||||
*
|
||||
* @param node is the symbolic name of the node which is to be affected
|
||||
* @param target is the symbolic name of the other node to which connectivity shall be impeded
|
||||
*/
|
||||
def abort(node: RoleName, target: RoleName): Future[Done] = {
|
||||
import Settings.QueryTimeout
|
||||
controller ? Disconnect(node, target, true) mapTo
|
||||
}
|
||||
|
||||
/**
|
||||
* Tell the remote node to shut itself down using System.exit with the given
|
||||
* exitValue.
|
||||
*
|
||||
* @param node is the symbolic name of the node which is to be affected
|
||||
* @param exitValue is the return code which shall be given to System.exit
|
||||
*/
|
||||
def shutdown(node: RoleName, exitValue: Int): Future[Done] = {
|
||||
import Settings.QueryTimeout
|
||||
controller ? Terminate(node, exitValue) mapTo
|
||||
}
|
||||
|
||||
/**
|
||||
* Tell the SBT plugin to forcibly terminate the given remote node using Process.destroy.
|
||||
*
|
||||
* @param node is the symbolic name of the node which is to be affected
|
||||
*/
|
||||
// TODO: uncomment (and implement in Controller) if really needed
|
||||
// def kill(node: RoleName): Future[Done] = {
|
||||
// import Settings.QueryTimeout
|
||||
// controller ? Terminate(node, -1) mapTo
|
||||
// }
|
||||
|
||||
/**
|
||||
* Obtain the list of remote host names currently registered.
|
||||
*/
|
||||
def getNodes: Future[Iterable[RoleName]] = {
|
||||
import Settings.QueryTimeout
|
||||
controller ? GetNodes mapTo
|
||||
}
|
||||
|
||||
/**
|
||||
* Remove a remote host from the list, so that the remaining nodes may still
|
||||
* pass subsequent barriers. This must be done before the client connection
|
||||
* breaks down in order to affect an “orderly” removal (i.e. without failing
|
||||
* present and future barriers).
|
||||
*
|
||||
* @param node is the symbolic name of the node which is to be removed
|
||||
*/
|
||||
def removeNode(node: RoleName): Future[Done] = {
|
||||
import Settings.QueryTimeout
|
||||
controller ? Remove(node) mapTo
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
/**
|
||||
* This handler is installed at the end of the controller’s netty pipeline. Its only
|
||||
* purpose is to dispatch incoming messages to the right ServerFSM actor. There is
|
||||
* one shared instance of this class for all connections accepted by one Controller.
|
||||
*
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] class ConductorHandler(_createTimeout: Timeout, controller: ActorRef, log: LoggingAdapter) extends SimpleChannelUpstreamHandler {
|
||||
|
||||
implicit val createTimeout = _createTimeout
|
||||
val clients = new ConcurrentHashMap[Channel, ActorRef]()
|
||||
|
||||
override def channelConnected(ctx: ChannelHandlerContext, event: ChannelStateEvent) = {
|
||||
val channel = event.getChannel
|
||||
log.debug("connection from {}", getAddrString(channel))
|
||||
val fsm: ActorRef = Await.result(controller ? Controller.CreateServerFSM(channel) mapTo, Duration.Inf)
|
||||
clients.put(channel, fsm)
|
||||
}
|
||||
|
||||
override def channelDisconnected(ctx: ChannelHandlerContext, event: ChannelStateEvent) = {
|
||||
val channel = event.getChannel
|
||||
log.debug("disconnect from {}", getAddrString(channel))
|
||||
val fsm = clients.get(channel)
|
||||
fsm ! Controller.ClientDisconnected
|
||||
clients.remove(channel)
|
||||
}
|
||||
|
||||
override def messageReceived(ctx: ChannelHandlerContext, event: MessageEvent) = {
|
||||
val channel = event.getChannel
|
||||
log.debug("message from {}: {}", getAddrString(channel), event.getMessage)
|
||||
event.getMessage match {
|
||||
case msg: NetworkOp ⇒
|
||||
clients.get(channel) ! msg
|
||||
case msg ⇒
|
||||
log.info("client {} sent garbage '{}', disconnecting", getAddrString(channel), msg)
|
||||
channel.close()
|
||||
}
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] object ServerFSM {
|
||||
sealed trait State
|
||||
case object Initial extends State
|
||||
case object Ready extends State
|
||||
}
|
||||
|
||||
/**
|
||||
* The server part of each client connection is represented by a ServerFSM.
|
||||
* The Initial state handles reception of the new client’s
|
||||
* [[akka.remote.testconductor.Hello]] message (which is needed for all subsequent
|
||||
* node name translations).
|
||||
*
|
||||
* In the Ready state, messages from the client are forwarded to the controller
|
||||
* and [[akka.remote.testconductor.Send]] requests are sent, but the latter is
|
||||
* treated specially: all client operations are to be confirmed by a
|
||||
* [[akka.remote.testconductor.Done]] message, and there can be only one such
|
||||
* request outstanding at a given time (i.e. a Send fails if the previous has
|
||||
* not yet been acknowledged).
|
||||
*
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] class ServerFSM(val controller: ActorRef, val channel: Channel) extends Actor with LoggingFSM[ServerFSM.State, Option[ActorRef]] {
|
||||
import ServerFSM._
|
||||
import akka.actor.FSM._
|
||||
import Controller._
|
||||
|
||||
startWith(Initial, None)
|
||||
|
||||
whenUnhandled {
|
||||
case Event(ClientDisconnected, Some(s)) ⇒
|
||||
s ! Status.Failure(new RuntimeException("client disconnected in state " + stateName + ": " + channel))
|
||||
stop()
|
||||
case Event(ClientDisconnected, None) ⇒ stop()
|
||||
}
|
||||
|
||||
onTermination {
|
||||
case _ ⇒ controller ! ClientDisconnected
|
||||
}
|
||||
|
||||
when(Initial, stateTimeout = 10 seconds) {
|
||||
case Event(Hello(name, addr), _) ⇒
|
||||
controller ! NodeInfo(RoleName(name), addr, self)
|
||||
goto(Ready)
|
||||
case Event(x: NetworkOp, _) ⇒
|
||||
log.warning("client {} sent no Hello in first message (instead {}), disconnecting", getAddrString(channel), x)
|
||||
channel.close()
|
||||
stop()
|
||||
case Event(ToClient(msg), _) ⇒
|
||||
log.warning("cannot send {} in state Initial", msg)
|
||||
stay
|
||||
case Event(StateTimeout, _) ⇒
|
||||
log.info("closing channel to {} because of Hello timeout", getAddrString(channel))
|
||||
channel.close()
|
||||
stop()
|
||||
}
|
||||
|
||||
when(Ready) {
|
||||
case Event(d: Done, Some(s)) ⇒
|
||||
s ! d
|
||||
stay using None
|
||||
case Event(op: ServerOp, _) ⇒
|
||||
controller ! op
|
||||
stay
|
||||
case Event(msg: NetworkOp, _) ⇒
|
||||
log.warning("client {} sent unsupported message {}", getAddrString(channel), msg)
|
||||
stop()
|
||||
case Event(ToClient(msg: UnconfirmedClientOp), _) ⇒
|
||||
channel.write(msg)
|
||||
stay
|
||||
case Event(ToClient(msg), None) ⇒
|
||||
channel.write(msg)
|
||||
stay using Some(sender)
|
||||
case Event(ToClient(msg), _) ⇒
|
||||
log.warning("cannot send {} while waiting for previous ACK", msg)
|
||||
stay
|
||||
}
|
||||
|
||||
initialize
|
||||
|
||||
onTermination {
|
||||
case _ ⇒ channel.close()
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] object Controller {
|
||||
case class ClientDisconnected(name: RoleName)
|
||||
case object GetNodes
|
||||
case object GetSockAddr
|
||||
case class CreateServerFSM(channel: Channel)
|
||||
|
||||
case class NodeInfo(name: RoleName, addr: Address, fsm: ActorRef)
|
||||
}
|
||||
|
||||
/**
|
||||
* This controls test execution by managing barriers (delegated to
|
||||
* [[akka.remote.testconductor.BarrierCoordinator]], its child) and allowing
|
||||
* network and other failures to be injected at the test nodes.
|
||||
*
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] class Controller(private var initialParticipants: Int, controllerPort: InetSocketAddress) extends Actor {
|
||||
import Controller._
|
||||
import BarrierCoordinator._
|
||||
|
||||
val settings = TestConductor().Settings
|
||||
val connection = RemoteConnection(Server, controllerPort,
|
||||
new ConductorHandler(settings.QueryTimeout, self, Logging(context.system, "ConductorHandler")))
|
||||
|
||||
/*
|
||||
* Supervision of the BarrierCoordinator means to catch all his bad emotions
|
||||
* and sometimes console him (BarrierEmpty, BarrierTimeout), sometimes tell
|
||||
* him to hate the world (WrongBarrier, DuplicateNode, ClientLost). The latter shall help
|
||||
* terminate broken tests as quickly as possible (i.e. without awaiting
|
||||
* BarrierTimeouts in the players).
|
||||
*/
|
||||
override def supervisorStrategy = OneForOneStrategy() {
|
||||
case BarrierTimeout(data) ⇒ SupervisorStrategy.Resume
|
||||
case BarrierEmpty(data, msg) ⇒ SupervisorStrategy.Resume
|
||||
case WrongBarrier(name, client, data) ⇒ client ! ToClient(BarrierResult(name, false)); failBarrier(data)
|
||||
case ClientLost(data, node) ⇒ failBarrier(data)
|
||||
case DuplicateNode(data, node) ⇒ failBarrier(data)
|
||||
}
|
||||
|
||||
def failBarrier(data: Data): SupervisorStrategy.Directive = {
|
||||
for (c ← data.arrived) c ! ToClient(BarrierResult(data.barrier, false))
|
||||
SupervisorStrategy.Restart
|
||||
}
|
||||
|
||||
val barrier = context.actorOf(Props[BarrierCoordinator], "barriers")
|
||||
var nodes = Map[RoleName, NodeInfo]()
|
||||
|
||||
// map keeping unanswered queries for node addresses (enqueued upon GetAddress, serviced upon NodeInfo)
|
||||
var addrInterest = Map[RoleName, Set[ActorRef]]()
|
||||
val generation = Iterator from 1
|
||||
|
||||
override def receive = LoggingReceive {
|
||||
case CreateServerFSM(channel) ⇒
|
||||
val (ip, port) = channel.getRemoteAddress match { case s: InetSocketAddress ⇒ (s.getAddress.getHostAddress, s.getPort) }
|
||||
val name = ip + ":" + port + "-server" + generation.next
|
||||
sender ! context.actorOf(Props(new ServerFSM(self, channel)), name)
|
||||
case c @ NodeInfo(name, addr, fsm) ⇒
|
||||
barrier forward c
|
||||
if (nodes contains name) {
|
||||
if (initialParticipants > 0) {
|
||||
for (NodeInfo(_, _, client) ← nodes.values) client ! ToClient(BarrierResult("initial startup", false))
|
||||
initialParticipants = 0
|
||||
}
|
||||
fsm ! ToClient(BarrierResult("initial startup", false))
|
||||
} else {
|
||||
nodes += name -> c
|
||||
if (initialParticipants <= 0) fsm ! ToClient(Done)
|
||||
else if (nodes.size == initialParticipants) {
|
||||
for (NodeInfo(_, _, client) ← nodes.values) client ! ToClient(Done)
|
||||
initialParticipants = 0
|
||||
}
|
||||
if (addrInterest contains name) {
|
||||
addrInterest(name) foreach (_ ! ToClient(AddressReply(name, addr)))
|
||||
addrInterest -= name
|
||||
}
|
||||
}
|
||||
case c @ ClientDisconnected(name) ⇒
|
||||
nodes -= name
|
||||
barrier forward c
|
||||
case op: ServerOp ⇒
|
||||
op match {
|
||||
case _: EnterBarrier ⇒ barrier forward op
|
||||
case GetAddress(node) ⇒
|
||||
if (nodes contains node) sender ! ToClient(AddressReply(node, nodes(node).addr))
|
||||
else addrInterest += node -> ((addrInterest get node getOrElse Set()) + sender)
|
||||
}
|
||||
case op: CommandOp ⇒
|
||||
op match {
|
||||
case Throttle(node, target, direction, rateMBit) ⇒
|
||||
val t = nodes(target)
|
||||
nodes(node).fsm forward ToClient(ThrottleMsg(t.addr, direction, rateMBit))
|
||||
case Disconnect(node, target, abort) ⇒
|
||||
val t = nodes(target)
|
||||
nodes(node).fsm forward ToClient(DisconnectMsg(t.addr, abort))
|
||||
case Terminate(node, exitValueOrKill) ⇒
|
||||
if (exitValueOrKill < 0) {
|
||||
// TODO: kill via SBT
|
||||
} else {
|
||||
nodes(node).fsm forward ToClient(TerminateMsg(exitValueOrKill))
|
||||
}
|
||||
case Remove(node) ⇒
|
||||
nodes -= node
|
||||
barrier ! BarrierCoordinator.RemoveClient(node)
|
||||
}
|
||||
case GetNodes ⇒ sender ! nodes.keys
|
||||
case GetSockAddr ⇒ sender ! connection.getLocalAddress
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] object BarrierCoordinator {
|
||||
sealed trait State
|
||||
case object Idle extends State
|
||||
case object Waiting extends State
|
||||
|
||||
case class RemoveClient(name: RoleName)
|
||||
|
||||
case class Data(clients: Set[Controller.NodeInfo], barrier: String, arrived: List[ActorRef])
|
||||
|
||||
trait Printer { this: Product with Throwable with NoStackTrace ⇒
|
||||
override def toString = productPrefix + productIterator.mkString("(", ", ", ")")
|
||||
}
|
||||
|
||||
case class BarrierTimeout(data: Data) extends RuntimeException(data.barrier) with NoStackTrace with Printer
|
||||
case class DuplicateNode(data: Data, node: Controller.NodeInfo) extends RuntimeException with NoStackTrace with Printer
|
||||
case class WrongBarrier(barrier: String, client: ActorRef, data: Data) extends RuntimeException(barrier) with NoStackTrace with Printer
|
||||
case class BarrierEmpty(data: Data, msg: String) extends RuntimeException(msg) with NoStackTrace with Printer
|
||||
case class ClientLost(data: Data, client: RoleName) extends RuntimeException with NoStackTrace with Printer
|
||||
}
|
||||
|
||||
/**
|
||||
* This barrier coordinator gets informed of players connecting (NodeInfo),
|
||||
* players being deliberately removed (RemoveClient) or failing (ClientDisconnected)
|
||||
* by the controller. It also receives EnterBarrier requests, where upon the first
|
||||
* one received the name of the current barrier is set and all other known clients
|
||||
* are expected to join the barrier, whereupon all of the will be sent the successful
|
||||
* EnterBarrier return message. In case of planned removals, this may just happen
|
||||
* earlier, in case of failures the current barrier (and all subsequent ones) will
|
||||
* be failed by sending BarrierFailed responses.
|
||||
*
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] class BarrierCoordinator extends Actor with LoggingFSM[BarrierCoordinator.State, BarrierCoordinator.Data] {
|
||||
import BarrierCoordinator._
|
||||
import akka.actor.FSM._
|
||||
import Controller._
|
||||
|
||||
// this shall be set to false if all subsequent barriers shall fail
|
||||
var failed = false
|
||||
override def preRestart(reason: Throwable, message: Option[Any]) {}
|
||||
override def postRestart(reason: Throwable) { failed = true }
|
||||
|
||||
// TODO what happens with the other waiting players in case of a test failure?
|
||||
|
||||
startWith(Idle, Data(Set(), "", Nil))
|
||||
|
||||
whenUnhandled {
|
||||
case Event(n: NodeInfo, d @ Data(clients, _, _)) ⇒
|
||||
if (clients.find(_.name == n.name).isDefined) throw new DuplicateNode(d, n)
|
||||
stay using d.copy(clients = clients + n)
|
||||
case Event(ClientDisconnected(name), d @ Data(clients, _, arrived)) ⇒
|
||||
if (clients.isEmpty) throw BarrierEmpty(d, "no client to disconnect")
|
||||
(clients find (_.name == name)) match {
|
||||
case None ⇒ stay
|
||||
case Some(c) ⇒ throw ClientLost(d.copy(clients = clients - c, arrived = arrived filterNot (_ == c.fsm)), name)
|
||||
}
|
||||
}
|
||||
|
||||
when(Idle) {
|
||||
case Event(EnterBarrier(name), d @ Data(clients, _, _)) ⇒
|
||||
if (failed)
|
||||
stay replying ToClient(BarrierResult(name, false))
|
||||
else if (clients.map(_.fsm) == Set(sender))
|
||||
stay replying ToClient(BarrierResult(name, true))
|
||||
else if (clients.find(_.fsm == sender).isEmpty)
|
||||
stay replying ToClient(BarrierResult(name, false))
|
||||
else
|
||||
goto(Waiting) using d.copy(barrier = name, arrived = sender :: Nil)
|
||||
case Event(RemoveClient(name), d @ Data(clients, _, _)) ⇒
|
||||
if (clients.isEmpty) throw BarrierEmpty(d, "no client to remove")
|
||||
stay using d.copy(clients = clients filterNot (_.name == name))
|
||||
}
|
||||
|
||||
onTransition {
|
||||
case Idle -> Waiting ⇒ setTimer("Timeout", StateTimeout, TestConductor().Settings.BarrierTimeout.duration, false)
|
||||
case Waiting -> Idle ⇒ cancelTimer("Timeout")
|
||||
}
|
||||
|
||||
when(Waiting) {
|
||||
case Event(EnterBarrier(name), d @ Data(clients, barrier, arrived)) ⇒
|
||||
if (name != barrier || clients.find(_.fsm == sender).isEmpty) throw WrongBarrier(name, sender, d)
|
||||
val together = sender :: arrived
|
||||
handleBarrier(d.copy(arrived = together))
|
||||
case Event(RemoveClient(name), d @ Data(clients, barrier, arrived)) ⇒
|
||||
clients find (_.name == name) match {
|
||||
case None ⇒ stay
|
||||
case Some(client) ⇒
|
||||
handleBarrier(d.copy(clients = clients - client, arrived = arrived filterNot (_ == client.fsm)))
|
||||
}
|
||||
case Event(StateTimeout, data) ⇒
|
||||
throw BarrierTimeout(data)
|
||||
}
|
||||
|
||||
initialize
|
||||
|
||||
def handleBarrier(data: Data): State = {
|
||||
log.debug("handleBarrier({})", data)
|
||||
if (data.arrived.isEmpty) {
|
||||
goto(Idle) using data.copy(barrier = "")
|
||||
} else if ((data.clients.map(_.fsm) -- data.arrived).isEmpty) {
|
||||
data.arrived foreach (_ ! ToClient(BarrierResult(data.barrier, true)))
|
||||
goto(Idle) using data.copy(barrier = "", arrived = Nil)
|
||||
} else {
|
||||
stay using data
|
||||
}
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
|
|
@ -0,0 +1,139 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.remote.testconductor
|
||||
|
||||
import org.jboss.netty.handler.codec.oneone.OneToOneEncoder
|
||||
import org.jboss.netty.channel.ChannelHandlerContext
|
||||
import org.jboss.netty.channel.Channel
|
||||
import akka.remote.testconductor.{ TestConductorProtocol ⇒ TCP }
|
||||
import com.google.protobuf.Message
|
||||
import akka.actor.Address
|
||||
import org.jboss.netty.handler.codec.oneone.OneToOneDecoder
|
||||
|
||||
case class RoleName(name: String)
|
||||
|
||||
private[akka] case class ToClient(msg: ClientOp with NetworkOp)
|
||||
private[akka] case class ToServer(msg: ServerOp with NetworkOp)
|
||||
|
||||
private[akka] sealed trait ClientOp // messages sent to from Conductor to Player
|
||||
private[akka] sealed trait ServerOp // messages sent to from Player to Conductor
|
||||
private[akka] sealed trait CommandOp // messages sent from TestConductorExt to Conductor
|
||||
private[akka] sealed trait NetworkOp // messages sent over the wire
|
||||
private[akka] sealed trait UnconfirmedClientOp extends ClientOp // unconfirmed messages going to the Player
|
||||
private[akka] sealed trait ConfirmedClientOp extends ClientOp
|
||||
|
||||
/**
|
||||
* First message of connection sets names straight.
|
||||
*/
|
||||
private[akka] case class Hello(name: String, addr: Address) extends NetworkOp
|
||||
|
||||
private[akka] case class EnterBarrier(name: String) extends ServerOp with NetworkOp
|
||||
private[akka] case class BarrierResult(name: String, success: Boolean) extends UnconfirmedClientOp with NetworkOp
|
||||
|
||||
private[akka] case class Throttle(node: RoleName, target: RoleName, direction: Direction, rateMBit: Float) extends CommandOp
|
||||
private[akka] case class ThrottleMsg(target: Address, direction: Direction, rateMBit: Float) extends ConfirmedClientOp with NetworkOp
|
||||
|
||||
private[akka] case class Disconnect(node: RoleName, target: RoleName, abort: Boolean) extends CommandOp
|
||||
private[akka] case class DisconnectMsg(target: Address, abort: Boolean) extends ConfirmedClientOp with NetworkOp
|
||||
|
||||
private[akka] case class Terminate(node: RoleName, exitValueOrKill: Int) extends CommandOp
|
||||
private[akka] case class TerminateMsg(exitValue: Int) extends ConfirmedClientOp with NetworkOp
|
||||
|
||||
private[akka] case class GetAddress(node: RoleName) extends ServerOp with NetworkOp
|
||||
private[akka] case class AddressReply(node: RoleName, addr: Address) extends UnconfirmedClientOp with NetworkOp
|
||||
|
||||
private[akka] abstract class Done extends ServerOp with UnconfirmedClientOp with NetworkOp
|
||||
private[akka] case object Done extends Done {
|
||||
def getInstance: Done = this
|
||||
}
|
||||
|
||||
private[akka] case class Remove(node: RoleName) extends CommandOp
|
||||
|
||||
private[akka] class MsgEncoder extends OneToOneEncoder {
|
||||
|
||||
implicit def address2proto(addr: Address): TCP.Address =
|
||||
TCP.Address.newBuilder
|
||||
.setProtocol(addr.protocol)
|
||||
.setSystem(addr.system)
|
||||
.setHost(addr.host.get)
|
||||
.setPort(addr.port.get)
|
||||
.build
|
||||
|
||||
implicit def direction2proto(dir: Direction): TCP.Direction = dir match {
|
||||
case Direction.Send ⇒ TCP.Direction.Send
|
||||
case Direction.Receive ⇒ TCP.Direction.Receive
|
||||
case Direction.Both ⇒ TCP.Direction.Both
|
||||
}
|
||||
|
||||
def encode(ctx: ChannelHandlerContext, ch: Channel, msg: AnyRef): AnyRef = msg match {
|
||||
case x: NetworkOp ⇒
|
||||
val w = TCP.Wrapper.newBuilder
|
||||
x match {
|
||||
case Hello(name, addr) ⇒
|
||||
w.setHello(TCP.Hello.newBuilder.setName(name).setAddress(addr))
|
||||
case EnterBarrier(name) ⇒
|
||||
w.setBarrier(TCP.EnterBarrier.newBuilder.setName(name))
|
||||
case BarrierResult(name, success) ⇒
|
||||
w.setBarrier(TCP.EnterBarrier.newBuilder.setName(name).setStatus(success))
|
||||
case ThrottleMsg(target, dir, rate) ⇒
|
||||
w.setFailure(TCP.InjectFailure.newBuilder.setAddress(target)
|
||||
.setFailure(TCP.FailType.Throttle).setDirection(dir).setRateMBit(rate))
|
||||
case DisconnectMsg(target, abort) ⇒
|
||||
w.setFailure(TCP.InjectFailure.newBuilder.setAddress(target)
|
||||
.setFailure(if (abort) TCP.FailType.Abort else TCP.FailType.Disconnect))
|
||||
case TerminateMsg(exitValue) ⇒
|
||||
w.setFailure(TCP.InjectFailure.newBuilder.setFailure(TCP.FailType.Shutdown).setExitValue(exitValue))
|
||||
case GetAddress(node) ⇒
|
||||
w.setAddr(TCP.AddressRequest.newBuilder.setNode(node.name))
|
||||
case AddressReply(node, addr) ⇒
|
||||
w.setAddr(TCP.AddressRequest.newBuilder.setNode(node.name).setAddr(addr))
|
||||
case _: Done ⇒
|
||||
w.setDone("")
|
||||
}
|
||||
w.build
|
||||
case _ ⇒ throw new IllegalArgumentException("wrong message " + msg)
|
||||
}
|
||||
}
|
||||
|
||||
private[akka] class MsgDecoder extends OneToOneDecoder {
|
||||
|
||||
implicit def address2scala(addr: TCP.Address): Address =
|
||||
Address(addr.getProtocol, addr.getSystem, addr.getHost, addr.getPort)
|
||||
|
||||
implicit def direction2scala(dir: TCP.Direction): Direction = dir match {
|
||||
case TCP.Direction.Send ⇒ Direction.Send
|
||||
case TCP.Direction.Receive ⇒ Direction.Receive
|
||||
case TCP.Direction.Both ⇒ Direction.Both
|
||||
}
|
||||
|
||||
def decode(ctx: ChannelHandlerContext, ch: Channel, msg: AnyRef): AnyRef = msg match {
|
||||
case w: TCP.Wrapper if w.getAllFields.size == 1 ⇒
|
||||
if (w.hasHello) {
|
||||
val h = w.getHello
|
||||
Hello(h.getName, h.getAddress)
|
||||
} else if (w.hasBarrier) {
|
||||
val barrier = w.getBarrier
|
||||
if (barrier.hasStatus) BarrierResult(barrier.getName, barrier.getStatus)
|
||||
else EnterBarrier(w.getBarrier.getName)
|
||||
} else if (w.hasFailure) {
|
||||
val f = w.getFailure
|
||||
import TCP.{ FailType ⇒ FT }
|
||||
f.getFailure match {
|
||||
case FT.Throttle ⇒ ThrottleMsg(f.getAddress, f.getDirection, f.getRateMBit)
|
||||
case FT.Abort ⇒ DisconnectMsg(f.getAddress, true)
|
||||
case FT.Disconnect ⇒ DisconnectMsg(f.getAddress, false)
|
||||
case FT.Shutdown ⇒ TerminateMsg(f.getExitValue)
|
||||
}
|
||||
} else if (w.hasAddr) {
|
||||
val a = w.getAddr
|
||||
if (a.hasAddr) AddressReply(RoleName(a.getNode), a.getAddr)
|
||||
else GetAddress(RoleName(a.getNode))
|
||||
} else if (w.hasDone) {
|
||||
Done
|
||||
} else {
|
||||
throw new IllegalArgumentException("unknown message " + msg)
|
||||
}
|
||||
case _ ⇒ throw new IllegalArgumentException("wrong message " + msg)
|
||||
}
|
||||
}
|
||||
|
|
@ -0,0 +1,73 @@
|
|||
package akka.remote.testconductor
|
||||
|
||||
import akka.actor.ExtensionKey
|
||||
import akka.actor.Extension
|
||||
import akka.actor.ExtendedActorSystem
|
||||
import akka.remote.RemoteActorRefProvider
|
||||
import akka.actor.ActorContext
|
||||
import akka.util.{ Duration, Timeout }
|
||||
import java.util.concurrent.TimeUnit.MILLISECONDS
|
||||
import akka.actor.ActorRef
|
||||
import java.util.concurrent.ConcurrentHashMap
|
||||
import akka.actor.Address
|
||||
import akka.actor.ActorSystemImpl
|
||||
import akka.actor.Props
|
||||
|
||||
/**
|
||||
* Access to the [[akka.remote.testconductor.TestConductorExt]] extension:
|
||||
*
|
||||
* {{{
|
||||
* val tc = TestConductor(system)
|
||||
* tc.startController(numPlayers)
|
||||
* // OR
|
||||
* tc.startClient(conductorPort)
|
||||
* }}}
|
||||
*/
|
||||
object TestConductor extends ExtensionKey[TestConductorExt] {
|
||||
|
||||
def apply()(implicit ctx: ActorContext): TestConductorExt = apply(ctx.system)
|
||||
|
||||
}
|
||||
|
||||
/**
|
||||
* This binds together the [[akka.remote.testconductor.Conductor]] and
|
||||
* [[akka.remote.testconductor.Player]] roles inside an Akka
|
||||
* [[akka.actor.Extension]]. Please follow the aforementioned links for
|
||||
* more information.
|
||||
*
|
||||
* <b>This extension requires the `akka.actor.provider`
|
||||
* to be a [[akka.remote.RemoteActorRefProvider]].</b>
|
||||
*/
|
||||
class TestConductorExt(val system: ExtendedActorSystem) extends Extension with Conductor with Player {
|
||||
|
||||
object Settings {
|
||||
val config = system.settings.config
|
||||
|
||||
val ConnectTimeout = Duration(config.getMilliseconds("akka.testconductor.connect-timeout"), MILLISECONDS)
|
||||
val ClientReconnects = config.getInt("akka.testconductor.client-reconnects")
|
||||
val ReconnectBackoff = Duration(config.getMilliseconds("akka.testconductor.reconnect-backoff"), MILLISECONDS)
|
||||
|
||||
implicit val BarrierTimeout = Timeout(Duration(config.getMilliseconds("akka.testconductor.barrier-timeout"), MILLISECONDS))
|
||||
implicit val QueryTimeout = Timeout(Duration(config.getMilliseconds("akka.testconductor.query-timeout"), MILLISECONDS))
|
||||
val PacketSplitThreshold = Duration(config.getMilliseconds("akka.testconductor.packet-split-threshold"), MILLISECONDS)
|
||||
}
|
||||
|
||||
/**
|
||||
* Remote transport used by the actor ref provider.
|
||||
*/
|
||||
val transport = system.provider.asInstanceOf[RemoteActorRefProvider].transport
|
||||
|
||||
/**
|
||||
* Transport address of this Netty-like remote transport.
|
||||
*/
|
||||
val address = transport.address
|
||||
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*
|
||||
* [[akka.remote.testconductor.NetworkFailureInjector]]s register themselves here so that
|
||||
* failures can be injected.
|
||||
*/
|
||||
private[akka] val failureInjector = system.asInstanceOf[ActorSystemImpl].systemActorOf(Props[FailureInjector], "FailureInjector")
|
||||
|
||||
}
|
||||
|
|
@ -0,0 +1,328 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.remote.testconductor
|
||||
|
||||
import java.net.InetSocketAddress
|
||||
|
||||
import scala.annotation.tailrec
|
||||
import scala.collection.immutable.Queue
|
||||
|
||||
import org.jboss.netty.buffer.ChannelBuffer
|
||||
import org.jboss.netty.channel.{ SimpleChannelHandler, MessageEvent, Channels, ChannelStateEvent, ChannelHandlerContext, ChannelFutureListener, ChannelFuture }
|
||||
|
||||
import akka.actor.{ Props, LoggingFSM, Address, ActorSystem, ActorRef, ActorLogging, Actor, FSM }
|
||||
import akka.event.Logging
|
||||
import akka.remote.netty.ChannelAddress
|
||||
import akka.util.Duration
|
||||
import akka.util.duration._
|
||||
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] class FailureInjector extends Actor with ActorLogging {
|
||||
import ThrottleActor._
|
||||
import NetworkFailureInjector._
|
||||
|
||||
case class ChannelSettings(
|
||||
ctx: Option[ChannelHandlerContext] = None,
|
||||
throttleSend: Option[SetRate] = None,
|
||||
throttleReceive: Option[SetRate] = None)
|
||||
case class Injectors(sender: ActorRef, receiver: ActorRef)
|
||||
|
||||
var channels = Map[ChannelHandlerContext, Injectors]()
|
||||
var settings = Map[Address, ChannelSettings]()
|
||||
var generation = Iterator from 1
|
||||
|
||||
/**
|
||||
* Only for a NEW ctx, start ThrottleActors, prime them and update all maps.
|
||||
*/
|
||||
def ingestContextAddress(ctx: ChannelHandlerContext, addr: Address): Injectors = {
|
||||
val gen = generation.next
|
||||
val name = addr.host.get + ":" + addr.port.get
|
||||
val thrSend = context.actorOf(Props(new ThrottleActor(ctx)), name + "-snd" + gen)
|
||||
val thrRecv = context.actorOf(Props(new ThrottleActor(ctx)), name + "-rcv" + gen)
|
||||
val injectors = Injectors(thrSend, thrRecv)
|
||||
channels += ctx -> injectors
|
||||
settings += addr -> (settings get addr map {
|
||||
case c @ ChannelSettings(prevCtx, ts, tr) ⇒
|
||||
ts foreach (thrSend ! _)
|
||||
tr foreach (thrRecv ! _)
|
||||
prevCtx match {
|
||||
case Some(p) ⇒ log.warning("installing context {} instead of {} for address {}", ctx, p, addr)
|
||||
case None ⇒ // okay
|
||||
}
|
||||
c.copy(ctx = Some(ctx))
|
||||
} getOrElse ChannelSettings(Some(ctx)))
|
||||
injectors
|
||||
}
|
||||
|
||||
/**
|
||||
* Retrieve target settings, also if they were sketchy before (i.e. no system name)
|
||||
*/
|
||||
def retrieveTargetSettings(target: Address): Option[ChannelSettings] = {
|
||||
settings get target orElse {
|
||||
val host = target.host
|
||||
val port = target.port
|
||||
settings find {
|
||||
case (Address("akka", "", `host`, `port`), s) ⇒ true
|
||||
case _ ⇒ false
|
||||
} map {
|
||||
case (_, s) ⇒ settings += target -> s; s
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
def receive = {
|
||||
case RemoveContext(ctx) ⇒
|
||||
channels get ctx foreach { inj ⇒
|
||||
context stop inj.sender
|
||||
context stop inj.receiver
|
||||
}
|
||||
channels -= ctx
|
||||
settings ++= settings collect { case (addr, c @ ChannelSettings(Some(`ctx`), _, _)) ⇒ (addr, c.copy(ctx = None)) }
|
||||
case ThrottleMsg(target, dir, rateMBit) ⇒
|
||||
val setting = retrieveTargetSettings(target)
|
||||
settings += target -> ((setting getOrElse ChannelSettings() match {
|
||||
case cs @ ChannelSettings(ctx, _, _) if dir includes Direction.Send ⇒
|
||||
ctx foreach (c ⇒ channels get c foreach (_.sender ! SetRate(rateMBit)))
|
||||
cs.copy(throttleSend = Some(SetRate(rateMBit)))
|
||||
case x ⇒ x
|
||||
}) match {
|
||||
case cs @ ChannelSettings(ctx, _, _) if dir includes Direction.Receive ⇒
|
||||
ctx foreach (c ⇒ channels get c foreach (_.receiver ! SetRate(rateMBit)))
|
||||
cs.copy(throttleReceive = Some(SetRate(rateMBit)))
|
||||
case x ⇒ x
|
||||
})
|
||||
sender ! "ok"
|
||||
case DisconnectMsg(target, abort) ⇒
|
||||
retrieveTargetSettings(target) foreach {
|
||||
case ChannelSettings(Some(ctx), _, _) ⇒
|
||||
val ch = ctx.getChannel
|
||||
if (abort) {
|
||||
ch.getConfig.setOption("soLinger", 0)
|
||||
log.info("aborting connection {}", ch)
|
||||
} else log.info("closing connection {}", ch)
|
||||
ch.close
|
||||
case _ ⇒ log.debug("no connection to {} to close or abort", target)
|
||||
}
|
||||
sender ! "ok"
|
||||
case s @ Send(ctx, direction, future, msg) ⇒
|
||||
channels get ctx match {
|
||||
case Some(Injectors(snd, rcv)) ⇒
|
||||
if (direction includes Direction.Send) snd ! s
|
||||
if (direction includes Direction.Receive) rcv ! s
|
||||
case None ⇒
|
||||
val (ipaddr, ip, port) = ctx.getChannel.getRemoteAddress match {
|
||||
case s: InetSocketAddress ⇒ (s.getAddress, s.getAddress.getHostAddress, s.getPort)
|
||||
}
|
||||
val addr = ChannelAddress.get(ctx.getChannel) orElse {
|
||||
settings collect { case (a @ Address("akka", _, Some(`ip`), Some(`port`)), _) ⇒ a } headOption
|
||||
} orElse {
|
||||
val name = ipaddr.getHostName
|
||||
if (name == ip) None
|
||||
else settings collect { case (a @ Address("akka", _, Some(`name`), Some(`port`)), _) ⇒ a } headOption
|
||||
} getOrElse Address("akka", "", ip, port) // this will not match later requests directly, but be picked up by retrieveTargetSettings
|
||||
val inj = ingestContextAddress(ctx, addr)
|
||||
if (direction includes Direction.Send) inj.sender ! s
|
||||
if (direction includes Direction.Receive) inj.receiver ! s
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
private[akka] object NetworkFailureInjector {
|
||||
case class RemoveContext(ctx: ChannelHandlerContext)
|
||||
}
|
||||
|
||||
/**
|
||||
* Brief overview: all network traffic passes through the `sender`/`receiver` FSMs managed
|
||||
* by the FailureInjector of the TestConductor extension. These can
|
||||
* pass through requests immediately, drop them or throttle to a desired rate. The FSMs are
|
||||
* registered in the TestConductorExt.failureInjector so that settings can be applied from
|
||||
* the ClientFSMs.
|
||||
*
|
||||
* I found that simply forwarding events using ctx.sendUpstream/sendDownstream does not work,
|
||||
* it deadlocks and gives strange errors; in the end I just trusted the Netty docs which
|
||||
* recommend to prefer `Channels.write()` and `Channels.fireMessageReceived()`.
|
||||
*
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] class NetworkFailureInjector(system: ActorSystem) extends SimpleChannelHandler {
|
||||
import NetworkFailureInjector._
|
||||
|
||||
private val log = Logging(system, "FailureInjector")
|
||||
|
||||
private val conductor = TestConductor(system)
|
||||
private var announced = false
|
||||
|
||||
override def channelConnected(ctx: ChannelHandlerContext, state: ChannelStateEvent) {
|
||||
state.getValue match {
|
||||
case a: InetSocketAddress ⇒
|
||||
val addr = Address("akka", "", a.getHostName, a.getPort)
|
||||
log.debug("connected to {}", addr)
|
||||
case x ⇒ throw new IllegalArgumentException("unknown address type: " + x)
|
||||
}
|
||||
}
|
||||
|
||||
override def channelDisconnected(ctx: ChannelHandlerContext, state: ChannelStateEvent) {
|
||||
log.debug("disconnected from {}", state.getChannel)
|
||||
conductor.failureInjector ! RemoveContext(ctx)
|
||||
}
|
||||
|
||||
override def messageReceived(ctx: ChannelHandlerContext, msg: MessageEvent) {
|
||||
log.debug("upstream(queued): {}", msg)
|
||||
conductor.failureInjector ! ThrottleActor.Send(ctx, Direction.Receive, Option(msg.getFuture), msg.getMessage)
|
||||
}
|
||||
|
||||
override def writeRequested(ctx: ChannelHandlerContext, msg: MessageEvent) {
|
||||
log.debug("downstream(queued): {}", msg)
|
||||
conductor.failureInjector ! ThrottleActor.Send(ctx, Direction.Send, Option(msg.getFuture), msg.getMessage)
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] object ThrottleActor {
|
||||
sealed trait State
|
||||
case object PassThrough extends State
|
||||
case object Throttle extends State
|
||||
case object Blackhole extends State
|
||||
|
||||
case class Data(lastSent: Long, rateMBit: Float, queue: Queue[Send])
|
||||
|
||||
case class Send(ctx: ChannelHandlerContext, direction: Direction, future: Option[ChannelFuture], msg: AnyRef)
|
||||
case class SetRate(rateMBit: Float)
|
||||
case object Tick
|
||||
}
|
||||
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] class ThrottleActor(channelContext: ChannelHandlerContext)
|
||||
extends Actor with LoggingFSM[ThrottleActor.State, ThrottleActor.Data] {
|
||||
|
||||
import ThrottleActor._
|
||||
import FSM._
|
||||
|
||||
private val packetSplitThreshold = TestConductor(context.system).Settings.PacketSplitThreshold
|
||||
|
||||
startWith(PassThrough, Data(0, -1, Queue()))
|
||||
|
||||
when(PassThrough) {
|
||||
case Event(s @ Send(_, _, _, msg), _) ⇒
|
||||
log.debug("sending msg (PassThrough): {}", msg)
|
||||
send(s)
|
||||
stay
|
||||
}
|
||||
|
||||
when(Throttle) {
|
||||
case Event(s: Send, data @ Data(_, _, Queue())) ⇒
|
||||
stay using sendThrottled(data.copy(lastSent = System.nanoTime, queue = Queue(s)))
|
||||
case Event(s: Send, data) ⇒
|
||||
stay using sendThrottled(data.copy(queue = data.queue.enqueue(s)))
|
||||
case Event(Tick, data) ⇒
|
||||
stay using sendThrottled(data)
|
||||
}
|
||||
|
||||
onTransition {
|
||||
case Throttle -> PassThrough ⇒
|
||||
for (s ← stateData.queue) {
|
||||
log.debug("sending msg (Transition): {}", s.msg)
|
||||
send(s)
|
||||
}
|
||||
cancelTimer("send")
|
||||
case Throttle -> Blackhole ⇒
|
||||
cancelTimer("send")
|
||||
}
|
||||
|
||||
when(Blackhole) {
|
||||
case Event(Send(_, _, _, msg), _) ⇒
|
||||
log.debug("dropping msg {}", msg)
|
||||
stay
|
||||
}
|
||||
|
||||
whenUnhandled {
|
||||
case Event(SetRate(rate), d) ⇒
|
||||
if (rate > 0) {
|
||||
goto(Throttle) using d.copy(lastSent = System.nanoTime, rateMBit = rate, queue = Queue())
|
||||
} else if (rate == 0) {
|
||||
goto(Blackhole)
|
||||
} else {
|
||||
goto(PassThrough)
|
||||
}
|
||||
}
|
||||
|
||||
initialize
|
||||
|
||||
private def sendThrottled(d: Data): Data = {
|
||||
val (data, toSend, toTick) = schedule(d)
|
||||
for (s ← toSend) {
|
||||
log.debug("sending msg (Tick): {}", s.msg)
|
||||
send(s)
|
||||
}
|
||||
if (!timerActive_?("send"))
|
||||
for (time ← toTick) {
|
||||
log.debug("scheduling next Tick in {}", time)
|
||||
setTimer("send", Tick, time, false)
|
||||
}
|
||||
data
|
||||
}
|
||||
|
||||
private def send(s: Send): Unit = s.direction match {
|
||||
case Direction.Send ⇒ Channels.write(s.ctx, s.future getOrElse Channels.future(s.ctx.getChannel), s.msg)
|
||||
case Direction.Receive ⇒ Channels.fireMessageReceived(s.ctx, s.msg)
|
||||
case _ ⇒
|
||||
}
|
||||
|
||||
private def schedule(d: Data): (Data, Seq[Send], Option[Duration]) = {
|
||||
val now = System.nanoTime
|
||||
@tailrec def rec(d: Data, toSend: Seq[Send]): (Data, Seq[Send], Option[Duration]) = {
|
||||
if (d.queue.isEmpty) (d, toSend, None)
|
||||
else {
|
||||
val timeForPacket = d.lastSent + (1000 * size(d.queue.head.msg) / d.rateMBit).toLong
|
||||
if (timeForPacket <= now) rec(Data(timeForPacket, d.rateMBit, d.queue.tail), toSend :+ d.queue.head)
|
||||
else {
|
||||
val splitThreshold = d.lastSent + packetSplitThreshold.toNanos
|
||||
if (now < splitThreshold) (d, toSend, Some((timeForPacket - now).nanos min (splitThreshold - now).nanos))
|
||||
else {
|
||||
val microsToSend = (now - d.lastSent) / 1000
|
||||
val (s1, s2) = split(d.queue.head, (microsToSend * d.rateMBit / 8).toInt)
|
||||
(d.copy(queue = s2 +: d.queue.tail), toSend :+ s1, Some((timeForPacket - now).nanos min packetSplitThreshold))
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
rec(d, Seq())
|
||||
}
|
||||
|
||||
private def split(s: Send, bytes: Int): (Send, Send) = {
|
||||
s.msg match {
|
||||
case buf: ChannelBuffer ⇒
|
||||
val f = s.future map { f ⇒
|
||||
val newF = Channels.future(s.ctx.getChannel)
|
||||
newF.addListener(new ChannelFutureListener {
|
||||
def operationComplete(future: ChannelFuture) {
|
||||
if (future.isCancelled) f.cancel()
|
||||
else future.getCause match {
|
||||
case null ⇒
|
||||
case thr ⇒ f.setFailure(thr)
|
||||
}
|
||||
}
|
||||
})
|
||||
newF
|
||||
}
|
||||
val b = buf.slice()
|
||||
b.writerIndex(b.readerIndex + bytes)
|
||||
buf.readerIndex(buf.readerIndex + bytes)
|
||||
(Send(s.ctx, s.direction, f, b), Send(s.ctx, s.direction, s.future, buf))
|
||||
}
|
||||
}
|
||||
|
||||
private def size(msg: AnyRef) = msg match {
|
||||
case b: ChannelBuffer ⇒ b.readableBytes() * 8
|
||||
case _ ⇒ throw new UnsupportedOperationException("NetworkFailureInjector only supports ChannelBuffer messages")
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -0,0 +1,298 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.remote.testconductor
|
||||
|
||||
import akka.actor.{ Actor, ActorRef, ActorSystem, LoggingFSM, Props }
|
||||
import RemoteConnection.getAddrString
|
||||
import akka.util.duration._
|
||||
import org.jboss.netty.channel.{ Channel, SimpleChannelUpstreamHandler, ChannelHandlerContext, ChannelStateEvent, MessageEvent }
|
||||
import com.typesafe.config.ConfigFactory
|
||||
import akka.util.Timeout
|
||||
import akka.util.Duration
|
||||
import java.util.concurrent.TimeUnit.MILLISECONDS
|
||||
import akka.pattern.{ ask, pipe }
|
||||
import akka.dispatch.Await
|
||||
import scala.util.control.NoStackTrace
|
||||
import akka.actor.Status
|
||||
import akka.event.LoggingAdapter
|
||||
import akka.actor.PoisonPill
|
||||
import akka.event.Logging
|
||||
import akka.dispatch.Future
|
||||
import java.net.InetSocketAddress
|
||||
import akka.actor.Address
|
||||
import org.jboss.netty.channel.ExceptionEvent
|
||||
import org.jboss.netty.channel.WriteCompletionEvent
|
||||
import java.net.ConnectException
|
||||
import akka.util.Deadline
|
||||
import akka.actor.Scheduler
|
||||
|
||||
/**
|
||||
* The Player is the client component of the
|
||||
* [[akka.remote.testconductor.TestConductorExt]] extension. It registers with
|
||||
* the [[akka.remote.testconductor.Conductor]]’s [[akka.remote.testconductor.Controller]]
|
||||
* in order to participate in barriers and enable network failure injection.
|
||||
*/
|
||||
trait Player { this: TestConductorExt ⇒
|
||||
|
||||
private var _client: ActorRef = _
|
||||
private def client = _client match {
|
||||
case null ⇒ throw new IllegalStateException("TestConductor client not yet started")
|
||||
case x ⇒ x
|
||||
}
|
||||
|
||||
/**
|
||||
* Connect to the conductor on the given port (the host is taken from setting
|
||||
* `akka.testconductor.host`). The connection is made asynchronously, but you
|
||||
* should await completion of the returned Future because that implies that
|
||||
* all expected participants of this test have successfully connected (i.e.
|
||||
* this is a first barrier in itself). The number of expected participants is
|
||||
* set in [[akka.remote.testconductor.Conductor]]`.startController()`.
|
||||
*/
|
||||
def startClient(name: RoleName, controllerAddr: InetSocketAddress): Future[Done] = {
|
||||
import ClientFSM._
|
||||
import akka.actor.FSM._
|
||||
import Settings.BarrierTimeout
|
||||
|
||||
if (_client ne null) throw new IllegalStateException("TestConductorClient already started")
|
||||
_client = system.actorOf(Props(new ClientFSM(name, controllerAddr)), "TestConductorClient")
|
||||
val a = system.actorOf(Props(new Actor {
|
||||
var waiting: ActorRef = _
|
||||
def receive = {
|
||||
case fsm: ActorRef ⇒ waiting = sender; fsm ! SubscribeTransitionCallBack(self)
|
||||
case Transition(_, Connecting, AwaitDone) ⇒ // step 1, not there yet
|
||||
case Transition(_, AwaitDone, Connected) ⇒ waiting ! Done; context stop self
|
||||
case t: Transition[_] ⇒ waiting ! Status.Failure(new RuntimeException("unexpected transition: " + t)); context stop self
|
||||
case CurrentState(_, Connected) ⇒ waiting ! Done; context stop self
|
||||
case _: CurrentState[_] ⇒
|
||||
}
|
||||
}))
|
||||
|
||||
a ? client mapTo
|
||||
}
|
||||
|
||||
/**
|
||||
* Enter the named barriers, one after the other, in the order given. Will
|
||||
* throw an exception in case of timeouts or other errors.
|
||||
*/
|
||||
def enter(name: String*) {
|
||||
system.log.debug("entering barriers " + name.mkString("(", ", ", ")"))
|
||||
name foreach { b ⇒
|
||||
import Settings.BarrierTimeout
|
||||
Await.result(client ? ToServer(EnterBarrier(b)), Duration.Inf)
|
||||
system.log.debug("passed barrier {}", b)
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* Query remote transport address of named node.
|
||||
*/
|
||||
def getAddressFor(name: RoleName): Future[Address] = {
|
||||
import Settings.BarrierTimeout
|
||||
client ? ToServer(GetAddress(name)) mapTo
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] object ClientFSM {
|
||||
sealed trait State
|
||||
case object Connecting extends State
|
||||
case object AwaitDone extends State
|
||||
case object Connected extends State
|
||||
case object Failed extends State
|
||||
|
||||
case class Data(channel: Option[Channel], runningOp: Option[(String, ActorRef)])
|
||||
|
||||
case class Connected(channel: Channel)
|
||||
case class ConnectionFailure(msg: String) extends RuntimeException(msg) with NoStackTrace
|
||||
case object Disconnected
|
||||
}
|
||||
|
||||
/**
|
||||
* This is the controlling entity on the [[akka.remote.testconductor.Player]]
|
||||
* side: in a first step it registers itself with a symbolic name and its remote
|
||||
* address at the [[akka.remote.testconductor.Controller]], then waits for the
|
||||
* `Done` message which signals that all other expected test participants have
|
||||
* done the same. After that, it will pass barrier requests to and from the
|
||||
* coordinator and react to the [[akka.remote.testconductor.Conductor]]’s
|
||||
* requests for failure injection.
|
||||
*
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] class ClientFSM(name: RoleName, controllerAddr: InetSocketAddress) extends Actor with LoggingFSM[ClientFSM.State, ClientFSM.Data] {
|
||||
import ClientFSM._
|
||||
|
||||
val settings = TestConductor().Settings
|
||||
|
||||
val handler = new PlayerHandler(controllerAddr, settings.ClientReconnects, settings.ReconnectBackoff,
|
||||
self, Logging(context.system, "PlayerHandler"), context.system.scheduler)
|
||||
|
||||
startWith(Connecting, Data(None, None))
|
||||
|
||||
when(Connecting, stateTimeout = settings.ConnectTimeout) {
|
||||
case Event(msg: ClientOp, _) ⇒
|
||||
stay replying Status.Failure(new IllegalStateException("not connected yet"))
|
||||
case Event(Connected(channel), _) ⇒
|
||||
channel.write(Hello(name.name, TestConductor().address))
|
||||
goto(AwaitDone) using Data(Some(channel), None)
|
||||
case Event(_: ConnectionFailure, _) ⇒
|
||||
goto(Failed)
|
||||
case Event(StateTimeout, _) ⇒
|
||||
log.error("connect timeout to TestConductor")
|
||||
goto(Failed)
|
||||
}
|
||||
|
||||
when(AwaitDone, stateTimeout = settings.BarrierTimeout.duration) {
|
||||
case Event(Done, _) ⇒
|
||||
log.debug("received Done: starting test")
|
||||
goto(Connected)
|
||||
case Event(msg: NetworkOp, _) ⇒
|
||||
log.error("received {} instead of Done", msg)
|
||||
goto(Failed)
|
||||
case Event(msg: ServerOp, _) ⇒
|
||||
stay replying Status.Failure(new IllegalStateException("not connected yet"))
|
||||
case Event(StateTimeout, _) ⇒
|
||||
log.error("connect timeout to TestConductor")
|
||||
goto(Failed)
|
||||
}
|
||||
|
||||
when(Connected) {
|
||||
case Event(Disconnected, _) ⇒
|
||||
log.info("disconnected from TestConductor")
|
||||
throw new ConnectionFailure("disconnect")
|
||||
case Event(ToServer(Done), Data(Some(channel), _)) ⇒
|
||||
channel.write(Done)
|
||||
stay
|
||||
case Event(ToServer(msg), d @ Data(Some(channel), None)) ⇒
|
||||
channel.write(msg)
|
||||
val token = msg match {
|
||||
case EnterBarrier(barrier) ⇒ barrier
|
||||
case GetAddress(node) ⇒ node.name
|
||||
}
|
||||
stay using d.copy(runningOp = Some(token, sender))
|
||||
case Event(ToServer(op), Data(channel, Some((token, _)))) ⇒
|
||||
log.error("cannot write {} while waiting for {}", op, token)
|
||||
stay
|
||||
case Event(op: ClientOp, d @ Data(Some(channel), runningOp)) ⇒
|
||||
op match {
|
||||
case BarrierResult(b, success) ⇒
|
||||
runningOp match {
|
||||
case Some((barrier, requester)) ⇒
|
||||
if (b != barrier) {
|
||||
requester ! Status.Failure(new RuntimeException("wrong barrier " + b + " received while waiting for " + barrier))
|
||||
} else if (!success) {
|
||||
requester ! Status.Failure(new RuntimeException("barrier failed: " + b))
|
||||
} else {
|
||||
requester ! b
|
||||
}
|
||||
case None ⇒
|
||||
log.warning("did not expect {}", op)
|
||||
}
|
||||
stay using d.copy(runningOp = None)
|
||||
case AddressReply(node, addr) ⇒
|
||||
runningOp match {
|
||||
case Some((_, requester)) ⇒
|
||||
requester ! addr
|
||||
case None ⇒
|
||||
log.warning("did not expect {}", op)
|
||||
}
|
||||
stay using d.copy(runningOp = None)
|
||||
case t: ThrottleMsg ⇒
|
||||
import settings.QueryTimeout
|
||||
TestConductor().failureInjector ? t map (_ ⇒ ToServer(Done)) pipeTo self
|
||||
stay
|
||||
case d: DisconnectMsg ⇒
|
||||
import settings.QueryTimeout
|
||||
TestConductor().failureInjector ? d map (_ ⇒ ToServer(Done)) pipeTo self
|
||||
stay
|
||||
case TerminateMsg(exit) ⇒
|
||||
System.exit(exit)
|
||||
stay // needed because Java doesn’t have Nothing
|
||||
}
|
||||
}
|
||||
|
||||
when(Failed) {
|
||||
case Event(msg: ClientOp, _) ⇒
|
||||
stay replying Status.Failure(new RuntimeException("cannot do " + msg + " while Failed"))
|
||||
case Event(msg: NetworkOp, _) ⇒
|
||||
log.warning("ignoring network message {} while Failed", msg)
|
||||
stay
|
||||
}
|
||||
|
||||
onTermination {
|
||||
case StopEvent(_, _, Data(Some(channel), _)) ⇒
|
||||
channel.close()
|
||||
}
|
||||
|
||||
initialize
|
||||
|
||||
}
|
||||
|
||||
/**
|
||||
* This handler only forwards messages received from the conductor to the [[akka.remote.testconductor.ClientFSM]].
|
||||
*
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] class PlayerHandler(
|
||||
server: InetSocketAddress,
|
||||
private var reconnects: Int,
|
||||
backoff: Duration,
|
||||
fsm: ActorRef,
|
||||
log: LoggingAdapter,
|
||||
scheduler: Scheduler)
|
||||
extends SimpleChannelUpstreamHandler {
|
||||
|
||||
import ClientFSM._
|
||||
|
||||
reconnect()
|
||||
|
||||
var nextAttempt: Deadline = _
|
||||
|
||||
override def channelOpen(ctx: ChannelHandlerContext, event: ChannelStateEvent) = log.debug("channel {} open", event.getChannel)
|
||||
override def channelClosed(ctx: ChannelHandlerContext, event: ChannelStateEvent) = log.debug("channel {} closed", event.getChannel)
|
||||
override def channelBound(ctx: ChannelHandlerContext, event: ChannelStateEvent) = log.debug("channel {} bound", event.getChannel)
|
||||
override def channelUnbound(ctx: ChannelHandlerContext, event: ChannelStateEvent) = log.debug("channel {} unbound", event.getChannel)
|
||||
override def writeComplete(ctx: ChannelHandlerContext, event: WriteCompletionEvent) = log.debug("channel {} written {}", event.getChannel, event.getWrittenAmount)
|
||||
|
||||
override def exceptionCaught(ctx: ChannelHandlerContext, event: ExceptionEvent) = {
|
||||
log.debug("channel {} exception {}", event.getChannel, event.getCause)
|
||||
event.getCause match {
|
||||
case c: ConnectException if reconnects > 0 ⇒
|
||||
reconnects -= 1
|
||||
scheduler.scheduleOnce(nextAttempt.timeLeft)(reconnect())
|
||||
case e ⇒ fsm ! ConnectionFailure(e.getMessage)
|
||||
}
|
||||
}
|
||||
|
||||
private def reconnect(): Unit = {
|
||||
nextAttempt = Deadline.now + backoff
|
||||
RemoteConnection(Client, server, this)
|
||||
}
|
||||
|
||||
override def channelConnected(ctx: ChannelHandlerContext, event: ChannelStateEvent) = {
|
||||
val ch = event.getChannel
|
||||
log.debug("connected to {}", getAddrString(ch))
|
||||
fsm ! Connected(ch)
|
||||
}
|
||||
|
||||
override def channelDisconnected(ctx: ChannelHandlerContext, event: ChannelStateEvent) = {
|
||||
val channel = event.getChannel
|
||||
log.debug("disconnected from {}", getAddrString(channel))
|
||||
fsm ! PoisonPill
|
||||
}
|
||||
|
||||
override def messageReceived(ctx: ChannelHandlerContext, event: MessageEvent) = {
|
||||
val channel = event.getChannel
|
||||
log.debug("message from {}: {}", getAddrString(channel), event.getMessage)
|
||||
event.getMessage match {
|
||||
case msg: NetworkOp ⇒
|
||||
fsm ! msg
|
||||
case msg ⇒
|
||||
log.info("server {} sent garbage '{}', disconnecting", getAddrString(channel), msg)
|
||||
channel.close()
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -0,0 +1,67 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2011 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.remote.testconductor
|
||||
|
||||
import org.jboss.netty.channel.{ Channel, ChannelPipeline, ChannelPipelineFactory, ChannelUpstreamHandler, SimpleChannelUpstreamHandler, StaticChannelPipeline }
|
||||
import org.jboss.netty.channel.socket.nio.{ NioClientSocketChannelFactory, NioServerSocketChannelFactory }
|
||||
import org.jboss.netty.bootstrap.{ ClientBootstrap, ServerBootstrap }
|
||||
import org.jboss.netty.handler.codec.frame.{ LengthFieldBasedFrameDecoder, LengthFieldPrepender }
|
||||
import org.jboss.netty.handler.codec.compression.{ ZlibDecoder, ZlibEncoder }
|
||||
import org.jboss.netty.handler.codec.protobuf.{ ProtobufDecoder, ProtobufEncoder }
|
||||
import org.jboss.netty.handler.timeout.{ ReadTimeoutHandler, ReadTimeoutException }
|
||||
import java.net.InetSocketAddress
|
||||
import java.util.concurrent.Executors
|
||||
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] class TestConductorPipelineFactory(handler: ChannelUpstreamHandler) extends ChannelPipelineFactory {
|
||||
def getPipeline: ChannelPipeline = {
|
||||
val encap = List(new LengthFieldPrepender(4), new LengthFieldBasedFrameDecoder(10000, 0, 4, 0, 4))
|
||||
val proto = List(new ProtobufEncoder, new ProtobufDecoder(TestConductorProtocol.Wrapper.getDefaultInstance))
|
||||
val msg = List(new MsgEncoder, new MsgDecoder)
|
||||
new StaticChannelPipeline(encap ::: proto ::: msg ::: handler :: Nil: _*)
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] sealed trait Role
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] case object Client extends Role
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] case object Server extends Role
|
||||
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] object RemoteConnection {
|
||||
def apply(role: Role, sockaddr: InetSocketAddress, handler: ChannelUpstreamHandler): Channel = {
|
||||
role match {
|
||||
case Client ⇒
|
||||
val socketfactory = new NioClientSocketChannelFactory(Executors.newCachedThreadPool, Executors.newCachedThreadPool)
|
||||
val bootstrap = new ClientBootstrap(socketfactory)
|
||||
bootstrap.setPipelineFactory(new TestConductorPipelineFactory(handler))
|
||||
bootstrap.setOption("tcpNoDelay", true)
|
||||
bootstrap.connect(sockaddr).getChannel
|
||||
case Server ⇒
|
||||
val socketfactory = new NioServerSocketChannelFactory(Executors.newCachedThreadPool, Executors.newCachedThreadPool)
|
||||
val bootstrap = new ServerBootstrap(socketfactory)
|
||||
bootstrap.setPipelineFactory(new TestConductorPipelineFactory(handler))
|
||||
bootstrap.setOption("reuseAddress", true)
|
||||
bootstrap.setOption("child.tcpNoDelay", true)
|
||||
bootstrap.bind(sockaddr)
|
||||
}
|
||||
}
|
||||
|
||||
def getAddrString(channel: Channel) = channel.getRemoteAddress match {
|
||||
case i: InetSocketAddress ⇒ i.toString
|
||||
case _ ⇒ "[unknown]"
|
||||
}
|
||||
}
|
||||
|
|
@ -0,0 +1,24 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.remote.testconductor
|
||||
|
||||
import akka.remote.netty.NettyRemoteTransport
|
||||
import akka.remote.RemoteSettings
|
||||
import akka.actor.ExtendedActorSystem
|
||||
import akka.remote.RemoteActorRefProvider
|
||||
import org.jboss.netty.channel.ChannelHandler
|
||||
import org.jboss.netty.channel.ChannelPipelineFactory
|
||||
|
||||
/**
|
||||
* INTERNAL API.
|
||||
*/
|
||||
private[akka] class TestConductorTransport(_system: ExtendedActorSystem, _provider: RemoteActorRefProvider)
|
||||
extends NettyRemoteTransport(_system, _provider) {
|
||||
|
||||
override def createPipeline(endpoint: ⇒ ChannelHandler, withTimeout: Boolean): ChannelPipelineFactory =
|
||||
new ChannelPipelineFactory {
|
||||
def getPipeline = PipelineFactory(new NetworkFailureInjector(system) +: PipelineFactory.defaultStack(withTimeout) :+ endpoint)
|
||||
}
|
||||
|
||||
}
|
||||
|
|
@ -0,0 +1,55 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.remote
|
||||
|
||||
import akka.actor.Actor
|
||||
import akka.actor.ActorRef
|
||||
import akka.actor.Props
|
||||
import akka.pattern.ask
|
||||
import akka.remote.testkit.MultiNodeConfig
|
||||
import akka.remote.testkit.MultiNodeSpec
|
||||
import akka.testkit._
|
||||
|
||||
object SimpleRemoteMultiJvmSpec extends MultiNodeConfig {
|
||||
|
||||
class SomeActor extends Actor with Serializable {
|
||||
def receive = {
|
||||
case "identify" ⇒ sender ! self
|
||||
}
|
||||
}
|
||||
|
||||
commonConfig(debugConfig(on = false))
|
||||
|
||||
val master = role("master")
|
||||
val slave = role("slave")
|
||||
|
||||
}
|
||||
|
||||
class SimpleRemoteMultiJvmNode1 extends SimpleRemoteSpec
|
||||
class SimpleRemoteMultiJvmNode2 extends SimpleRemoteSpec
|
||||
|
||||
class SimpleRemoteSpec extends MultiNodeSpec(SimpleRemoteMultiJvmSpec)
|
||||
with ImplicitSender with DefaultTimeout {
|
||||
import SimpleRemoteMultiJvmSpec._
|
||||
|
||||
def initialParticipants = 2
|
||||
|
||||
runOn(master) {
|
||||
system.actorOf(Props[SomeActor], "service-hello")
|
||||
}
|
||||
|
||||
"Remoting" must {
|
||||
"lookup remote actor" in {
|
||||
runOn(slave) {
|
||||
val hello = system.actorFor(node(master) / "user" / "service-hello")
|
||||
hello.isInstanceOf[RemoteActorRef] must be(true)
|
||||
val masterAddress = testConductor.getAddressFor(master).await
|
||||
(hello ? "identify").await.asInstanceOf[ActorRef].path.address must equal(masterAddress)
|
||||
}
|
||||
testConductor.enter("done")
|
||||
}
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
|
|
@ -0,0 +1,73 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.remote.router
|
||||
|
||||
import com.typesafe.config.ConfigFactory
|
||||
|
||||
import akka.actor.Actor
|
||||
import akka.actor.ActorRef
|
||||
import akka.actor.Props
|
||||
import akka.pattern.ask
|
||||
import akka.remote.RemoteActorRef
|
||||
import akka.remote.testkit.MultiNodeConfig
|
||||
import akka.remote.testkit.MultiNodeSpec
|
||||
import akka.testkit._
|
||||
|
||||
object DirectRoutedRemoteActorMultiJvmSpec extends MultiNodeConfig {
|
||||
|
||||
class SomeActor extends Actor with Serializable {
|
||||
def receive = {
|
||||
case "identify" ⇒ sender ! self
|
||||
}
|
||||
}
|
||||
|
||||
commonConfig(debugConfig(on = false))
|
||||
|
||||
val master = role("master")
|
||||
val slave = role("slave")
|
||||
|
||||
nodeConfig(master, ConfigFactory.parseString("""
|
||||
akka.actor {
|
||||
deployment {
|
||||
/service-hello.remote = "akka://MultiNodeSpec@%s"
|
||||
}
|
||||
}
|
||||
# FIXME When using NettyRemoteTransport instead of TestConductorTransport it works
|
||||
# akka.remote.transport = "akka.remote.netty.NettyRemoteTransport"
|
||||
""".format("localhost:2553"))) // FIXME is there a way to avoid hardcoding the host:port here?
|
||||
|
||||
nodeConfig(slave, ConfigFactory.parseString("""
|
||||
akka.remote.netty.port = 2553
|
||||
"""))
|
||||
|
||||
}
|
||||
|
||||
class DirectRoutedRemoteActorMultiJvmNode1 extends DirectRoutedRemoteActorSpec
|
||||
class DirectRoutedRemoteActorMultiJvmNode2 extends DirectRoutedRemoteActorSpec
|
||||
|
||||
class DirectRoutedRemoteActorSpec extends MultiNodeSpec(DirectRoutedRemoteActorMultiJvmSpec)
|
||||
with ImplicitSender with DefaultTimeout {
|
||||
import DirectRoutedRemoteActorMultiJvmSpec._
|
||||
|
||||
def initialParticipants = 2
|
||||
|
||||
"A new remote actor configured with a Direct router" must {
|
||||
"be locally instantiated on a remote node and be able to communicate through its RemoteActorRef" in {
|
||||
|
||||
runOn(master) {
|
||||
val actor = system.actorOf(Props[SomeActor], "service-hello")
|
||||
actor.isInstanceOf[RemoteActorRef] must be(true)
|
||||
|
||||
val slaveAddress = testConductor.getAddressFor(slave).await
|
||||
(actor ? "identify").await.asInstanceOf[ActorRef].path.address must equal(slaveAddress)
|
||||
|
||||
// shut down the actor before we let the other node(s) shut down so we don't try to send
|
||||
// "Terminate" to a shut down node
|
||||
system.stop(actor)
|
||||
}
|
||||
|
||||
testConductor.enter("done")
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
@ -0,0 +1,111 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.remote.testconductor
|
||||
|
||||
import akka.remote.AkkaRemoteSpec
|
||||
import com.typesafe.config.ConfigFactory
|
||||
import akka.remote.AbstractRemoteActorMultiJvmSpec
|
||||
import akka.actor.Props
|
||||
import akka.actor.Actor
|
||||
import akka.dispatch.Await
|
||||
import akka.dispatch.Await.Awaitable
|
||||
import akka.util.Duration
|
||||
import akka.util.duration._
|
||||
import akka.testkit.ImplicitSender
|
||||
import java.net.InetSocketAddress
|
||||
import java.net.InetAddress
|
||||
import akka.remote.testkit.MultiNodeSpec
|
||||
import akka.remote.testkit.MultiNodeConfig
|
||||
|
||||
object TestConductorMultiJvmSpec extends MultiNodeConfig {
|
||||
commonConfig(debugConfig(on = false))
|
||||
|
||||
val master = role("master")
|
||||
val slave = role("slave")
|
||||
}
|
||||
|
||||
class TestConductorMultiJvmNode1 extends TestConductorSpec
|
||||
class TestConductorMultiJvmNode2 extends TestConductorSpec
|
||||
|
||||
class TestConductorSpec extends MultiNodeSpec(TestConductorMultiJvmSpec) with ImplicitSender {
|
||||
|
||||
import TestConductorMultiJvmSpec._
|
||||
|
||||
def initialParticipants = 2
|
||||
|
||||
runOn(master) {
|
||||
system.actorOf(Props(new Actor {
|
||||
def receive = {
|
||||
case x ⇒ testActor ! x; sender ! x
|
||||
}
|
||||
}), "echo")
|
||||
}
|
||||
|
||||
val echo = system.actorFor(node(master) / "user" / "echo")
|
||||
|
||||
"A TestConductor" must {
|
||||
|
||||
"enter a barrier" in {
|
||||
testConductor.enter("name")
|
||||
}
|
||||
|
||||
"support throttling of network connections" in {
|
||||
|
||||
runOn(slave) {
|
||||
// start remote network connection so that it can be throttled
|
||||
echo ! "start"
|
||||
}
|
||||
|
||||
expectMsg("start")
|
||||
|
||||
runOn(master) {
|
||||
testConductor.throttle(slave, master, Direction.Send, rateMBit = 0.01).await
|
||||
}
|
||||
|
||||
testConductor.enter("throttled_send")
|
||||
|
||||
runOn(slave) {
|
||||
for (i ← 0 to 9) echo ! i
|
||||
}
|
||||
|
||||
within(0.6 seconds, 2 seconds) {
|
||||
expectMsg(500 millis, 0)
|
||||
receiveN(9) must be(1 to 9)
|
||||
}
|
||||
|
||||
testConductor.enter("throttled_send2")
|
||||
|
||||
runOn(master) {
|
||||
testConductor.throttle(slave, master, Direction.Send, -1).await
|
||||
testConductor.throttle(slave, master, Direction.Receive, rateMBit = 0.01).await
|
||||
}
|
||||
|
||||
testConductor.enter("throttled_recv")
|
||||
|
||||
runOn(slave) {
|
||||
for (i ← 10 to 19) echo ! i
|
||||
}
|
||||
|
||||
val (min, max) =
|
||||
ifNode(master) {
|
||||
(0 seconds, 500 millis)
|
||||
} {
|
||||
(0.6 seconds, 2 seconds)
|
||||
}
|
||||
|
||||
within(min, max) {
|
||||
expectMsg(500 millis, 10)
|
||||
receiveN(9) must be(11 to 19)
|
||||
}
|
||||
|
||||
testConductor.enter("throttled_recv2")
|
||||
|
||||
runOn(master) {
|
||||
testConductor.throttle(slave, master, Direction.Receive, -1).await
|
||||
}
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
}
|
||||
|
|
@ -0,0 +1,471 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.remote.testconductor
|
||||
|
||||
import akka.testkit.AkkaSpec
|
||||
import akka.actor.Props
|
||||
import akka.actor.AddressFromURIString
|
||||
import akka.actor.ActorRef
|
||||
import akka.testkit.ImplicitSender
|
||||
import akka.actor.Actor
|
||||
import akka.actor.OneForOneStrategy
|
||||
import akka.actor.SupervisorStrategy
|
||||
import akka.testkit.EventFilter
|
||||
import akka.testkit.TestProbe
|
||||
import akka.util.duration._
|
||||
import akka.event.Logging
|
||||
import org.scalatest.BeforeAndAfterEach
|
||||
import java.net.InetSocketAddress
|
||||
import java.net.InetAddress
|
||||
|
||||
object BarrierSpec {
|
||||
case class Failed(ref: ActorRef, thr: Throwable)
|
||||
val config = """
|
||||
akka.testconductor.barrier-timeout = 5s
|
||||
akka.actor.provider = akka.remote.RemoteActorRefProvider
|
||||
akka.remote.netty.port = 0
|
||||
akka.actor.debug.fsm = on
|
||||
akka.actor.debug.lifecycle = on
|
||||
"""
|
||||
}
|
||||
|
||||
class BarrierSpec extends AkkaSpec(BarrierSpec.config) with ImplicitSender with BeforeAndAfterEach {
|
||||
|
||||
import BarrierSpec._
|
||||
import Controller._
|
||||
import BarrierCoordinator._
|
||||
|
||||
val A = RoleName("a")
|
||||
val B = RoleName("b")
|
||||
val C = RoleName("c")
|
||||
|
||||
override def afterEach {
|
||||
system.eventStream.setLogLevel(Logging.WarningLevel)
|
||||
}
|
||||
|
||||
"A BarrierCoordinator" must {
|
||||
|
||||
"register clients and remove them" in {
|
||||
val b = getBarrier()
|
||||
b ! NodeInfo(A, AddressFromURIString("akka://sys"), system.deadLetters)
|
||||
b ! RemoveClient(B)
|
||||
b ! RemoveClient(A)
|
||||
EventFilter[BarrierEmpty](occurrences = 1) intercept {
|
||||
b ! RemoveClient(A)
|
||||
}
|
||||
expectMsg(Failed(b, BarrierEmpty(Data(Set(), "", Nil), "no client to remove")))
|
||||
}
|
||||
|
||||
"register clients and disconnect them" in {
|
||||
val b = getBarrier()
|
||||
b ! NodeInfo(A, AddressFromURIString("akka://sys"), system.deadLetters)
|
||||
b ! ClientDisconnected(B)
|
||||
EventFilter[ClientLost](occurrences = 1) intercept {
|
||||
b ! ClientDisconnected(A)
|
||||
}
|
||||
expectMsg(Failed(b, ClientLost(Data(Set(), "", Nil), A)))
|
||||
EventFilter[BarrierEmpty](occurrences = 1) intercept {
|
||||
b ! ClientDisconnected(A)
|
||||
}
|
||||
expectMsg(Failed(b, BarrierEmpty(Data(Set(), "", Nil), "no client to disconnect")))
|
||||
}
|
||||
|
||||
"fail entering barrier when nobody registered" in {
|
||||
val b = getBarrier()
|
||||
b ! EnterBarrier("b")
|
||||
expectMsg(ToClient(BarrierResult("b", false)))
|
||||
}
|
||||
|
||||
"enter barrier" in {
|
||||
val barrier = getBarrier()
|
||||
val a, b = TestProbe()
|
||||
barrier ! NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
barrier ! NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
noMsg(a, b)
|
||||
within(1 second) {
|
||||
b.send(barrier, EnterBarrier("bar"))
|
||||
a.expectMsg(ToClient(BarrierResult("bar", true)))
|
||||
b.expectMsg(ToClient(BarrierResult("bar", true)))
|
||||
}
|
||||
}
|
||||
|
||||
"enter barrier with joining node" in {
|
||||
val barrier = getBarrier()
|
||||
val a, b, c = TestProbe()
|
||||
barrier ! NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
barrier ! NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
barrier ! NodeInfo(C, AddressFromURIString("akka://sys"), c.ref)
|
||||
b.send(barrier, EnterBarrier("bar"))
|
||||
noMsg(a, b, c)
|
||||
within(1 second) {
|
||||
c.send(barrier, EnterBarrier("bar"))
|
||||
a.expectMsg(ToClient(BarrierResult("bar", true)))
|
||||
b.expectMsg(ToClient(BarrierResult("bar", true)))
|
||||
c.expectMsg(ToClient(BarrierResult("bar", true)))
|
||||
}
|
||||
}
|
||||
|
||||
"enter barrier with leaving node" in {
|
||||
val barrier = getBarrier()
|
||||
val a, b, c = TestProbe()
|
||||
barrier ! NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
barrier ! NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
barrier ! NodeInfo(C, AddressFromURIString("akka://sys"), c.ref)
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
b.send(barrier, EnterBarrier("bar"))
|
||||
barrier ! RemoveClient(A)
|
||||
barrier ! ClientDisconnected(A)
|
||||
noMsg(a, b, c)
|
||||
b.within(1 second) {
|
||||
barrier ! RemoveClient(C)
|
||||
b.expectMsg(ToClient(BarrierResult("bar", true)))
|
||||
}
|
||||
barrier ! ClientDisconnected(C)
|
||||
expectNoMsg(1 second)
|
||||
}
|
||||
|
||||
"leave barrier when last “arrived” is removed" in {
|
||||
val barrier = getBarrier()
|
||||
val a, b = TestProbe()
|
||||
barrier ! NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
barrier ! NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
barrier ! RemoveClient(A)
|
||||
b.send(barrier, EnterBarrier("foo"))
|
||||
b.expectMsg(ToClient(BarrierResult("foo", true)))
|
||||
}
|
||||
|
||||
"fail barrier with disconnecing node" in {
|
||||
val barrier = getBarrier()
|
||||
val a, b = TestProbe()
|
||||
val nodeA = NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
barrier ! nodeA
|
||||
barrier ! NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
EventFilter[ClientLost](occurrences = 1) intercept {
|
||||
barrier ! ClientDisconnected(B)
|
||||
}
|
||||
expectMsg(Failed(barrier, ClientLost(Data(Set(nodeA), "bar", a.ref :: Nil), B)))
|
||||
}
|
||||
|
||||
"fail barrier with disconnecing node who already arrived" in {
|
||||
val barrier = getBarrier()
|
||||
val a, b, c = TestProbe()
|
||||
val nodeA = NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
val nodeC = NodeInfo(C, AddressFromURIString("akka://sys"), c.ref)
|
||||
barrier ! nodeA
|
||||
barrier ! NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
barrier ! nodeC
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
b.send(barrier, EnterBarrier("bar"))
|
||||
EventFilter[ClientLost](occurrences = 1) intercept {
|
||||
barrier ! ClientDisconnected(B)
|
||||
}
|
||||
expectMsg(Failed(barrier, ClientLost(Data(Set(nodeA, nodeC), "bar", a.ref :: Nil), B)))
|
||||
}
|
||||
|
||||
"fail when entering wrong barrier" in {
|
||||
val barrier = getBarrier()
|
||||
val a, b = TestProbe()
|
||||
val nodeA = NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
barrier ! nodeA
|
||||
val nodeB = NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
barrier ! nodeB
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
EventFilter[WrongBarrier](occurrences = 1) intercept {
|
||||
b.send(barrier, EnterBarrier("foo"))
|
||||
}
|
||||
expectMsg(Failed(barrier, WrongBarrier("foo", b.ref, Data(Set(nodeA, nodeB), "bar", a.ref :: Nil))))
|
||||
}
|
||||
|
||||
"fail barrier after first failure" in {
|
||||
val barrier = getBarrier()
|
||||
val a = TestProbe()
|
||||
EventFilter[BarrierEmpty](occurrences = 1) intercept {
|
||||
barrier ! RemoveClient(A)
|
||||
}
|
||||
expectMsg(Failed(barrier, BarrierEmpty(Data(Set(), "", Nil), "no client to remove")))
|
||||
barrier ! NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
a.send(barrier, EnterBarrier("right"))
|
||||
a.expectMsg(ToClient(BarrierResult("right", false)))
|
||||
}
|
||||
|
||||
"fail after barrier timeout" in {
|
||||
val barrier = getBarrier()
|
||||
val a, b = TestProbe()
|
||||
val nodeA = NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
val nodeB = NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
barrier ! nodeA
|
||||
barrier ! nodeB
|
||||
a.send(barrier, EnterBarrier("right"))
|
||||
EventFilter[BarrierTimeout](occurrences = 1) intercept {
|
||||
expectMsg(7 seconds, Failed(barrier, BarrierTimeout(Data(Set(nodeA, nodeB), "right", a.ref :: Nil))))
|
||||
}
|
||||
}
|
||||
|
||||
"fail if a node registers twice" in {
|
||||
val barrier = getBarrier()
|
||||
val a, b = TestProbe()
|
||||
val nodeA = NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
val nodeB = NodeInfo(A, AddressFromURIString("akka://sys"), b.ref)
|
||||
barrier ! nodeA
|
||||
EventFilter[DuplicateNode](occurrences = 1) intercept {
|
||||
barrier ! nodeB
|
||||
}
|
||||
expectMsg(Failed(barrier, DuplicateNode(Data(Set(nodeA), "", Nil), nodeB)))
|
||||
}
|
||||
|
||||
"finally have no failure messages left" in {
|
||||
expectNoMsg(1 second)
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
"A Controller with BarrierCoordinator" must {
|
||||
|
||||
"register clients and remove them" in {
|
||||
val b = getController(1)
|
||||
b ! NodeInfo(A, AddressFromURIString("akka://sys"), testActor)
|
||||
expectMsg(ToClient(Done))
|
||||
b ! Remove(B)
|
||||
b ! Remove(A)
|
||||
EventFilter[BarrierEmpty](occurrences = 1) intercept {
|
||||
b ! Remove(A)
|
||||
}
|
||||
}
|
||||
|
||||
"register clients and disconnect them" in {
|
||||
val b = getController(1)
|
||||
b ! NodeInfo(A, AddressFromURIString("akka://sys"), testActor)
|
||||
expectMsg(ToClient(Done))
|
||||
b ! ClientDisconnected(B)
|
||||
EventFilter[ClientLost](occurrences = 1) intercept {
|
||||
b ! ClientDisconnected(A)
|
||||
}
|
||||
EventFilter[BarrierEmpty](occurrences = 1) intercept {
|
||||
b ! ClientDisconnected(A)
|
||||
}
|
||||
}
|
||||
|
||||
"fail entering barrier when nobody registered" in {
|
||||
val b = getController(0)
|
||||
b ! EnterBarrier("b")
|
||||
expectMsg(ToClient(BarrierResult("b", false)))
|
||||
}
|
||||
|
||||
"enter barrier" in {
|
||||
val barrier = getController(2)
|
||||
val a, b = TestProbe()
|
||||
barrier ! NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
barrier ! NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
a.expectMsg(ToClient(Done))
|
||||
b.expectMsg(ToClient(Done))
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
noMsg(a, b)
|
||||
within(1 second) {
|
||||
b.send(barrier, EnterBarrier("bar"))
|
||||
a.expectMsg(ToClient(BarrierResult("bar", true)))
|
||||
b.expectMsg(ToClient(BarrierResult("bar", true)))
|
||||
}
|
||||
}
|
||||
|
||||
"enter barrier with joining node" in {
|
||||
val barrier = getController(2)
|
||||
val a, b, c = TestProbe()
|
||||
barrier ! NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
barrier ! NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
a.expectMsg(ToClient(Done))
|
||||
b.expectMsg(ToClient(Done))
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
barrier ! NodeInfo(C, AddressFromURIString("akka://sys"), c.ref)
|
||||
c.expectMsg(ToClient(Done))
|
||||
b.send(barrier, EnterBarrier("bar"))
|
||||
noMsg(a, b, c)
|
||||
within(1 second) {
|
||||
c.send(barrier, EnterBarrier("bar"))
|
||||
a.expectMsg(ToClient(BarrierResult("bar", true)))
|
||||
b.expectMsg(ToClient(BarrierResult("bar", true)))
|
||||
c.expectMsg(ToClient(BarrierResult("bar", true)))
|
||||
}
|
||||
}
|
||||
|
||||
"enter barrier with leaving node" in {
|
||||
val barrier = getController(3)
|
||||
val a, b, c = TestProbe()
|
||||
barrier ! NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
barrier ! NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
barrier ! NodeInfo(C, AddressFromURIString("akka://sys"), c.ref)
|
||||
a.expectMsg(ToClient(Done))
|
||||
b.expectMsg(ToClient(Done))
|
||||
c.expectMsg(ToClient(Done))
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
b.send(barrier, EnterBarrier("bar"))
|
||||
barrier ! Remove(A)
|
||||
barrier ! ClientDisconnected(A)
|
||||
noMsg(a, b, c)
|
||||
b.within(1 second) {
|
||||
barrier ! Remove(C)
|
||||
b.expectMsg(ToClient(BarrierResult("bar", true)))
|
||||
}
|
||||
barrier ! ClientDisconnected(C)
|
||||
expectNoMsg(1 second)
|
||||
}
|
||||
|
||||
"leave barrier when last “arrived” is removed" in {
|
||||
val barrier = getController(2)
|
||||
val a, b = TestProbe()
|
||||
barrier ! NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
barrier ! NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
a.expectMsg(ToClient(Done))
|
||||
b.expectMsg(ToClient(Done))
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
barrier ! Remove(A)
|
||||
b.send(barrier, EnterBarrier("foo"))
|
||||
b.expectMsg(ToClient(BarrierResult("foo", true)))
|
||||
}
|
||||
|
||||
"fail barrier with disconnecing node" in {
|
||||
val barrier = getController(2)
|
||||
val a, b = TestProbe()
|
||||
val nodeA = NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
barrier ! nodeA
|
||||
barrier ! NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
a.expectMsg(ToClient(Done))
|
||||
b.expectMsg(ToClient(Done))
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
barrier ! ClientDisconnected(RoleName("unknown"))
|
||||
noMsg(a)
|
||||
EventFilter[ClientLost](occurrences = 1) intercept {
|
||||
barrier ! ClientDisconnected(B)
|
||||
}
|
||||
a.expectMsg(ToClient(BarrierResult("bar", false)))
|
||||
}
|
||||
|
||||
"fail barrier with disconnecing node who already arrived" in {
|
||||
val barrier = getController(3)
|
||||
val a, b, c = TestProbe()
|
||||
val nodeA = NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
val nodeC = NodeInfo(C, AddressFromURIString("akka://sys"), c.ref)
|
||||
barrier ! nodeA
|
||||
barrier ! NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
barrier ! nodeC
|
||||
a.expectMsg(ToClient(Done))
|
||||
b.expectMsg(ToClient(Done))
|
||||
c.expectMsg(ToClient(Done))
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
b.send(barrier, EnterBarrier("bar"))
|
||||
EventFilter[ClientLost](occurrences = 1) intercept {
|
||||
barrier ! ClientDisconnected(B)
|
||||
}
|
||||
a.expectMsg(ToClient(BarrierResult("bar", false)))
|
||||
}
|
||||
|
||||
"fail when entering wrong barrier" in {
|
||||
val barrier = getController(2)
|
||||
val a, b = TestProbe()
|
||||
val nodeA = NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
barrier ! nodeA
|
||||
val nodeB = NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
barrier ! nodeB
|
||||
a.expectMsg(ToClient(Done))
|
||||
b.expectMsg(ToClient(Done))
|
||||
a.send(barrier, EnterBarrier("bar"))
|
||||
EventFilter[WrongBarrier](occurrences = 1) intercept {
|
||||
b.send(barrier, EnterBarrier("foo"))
|
||||
}
|
||||
a.expectMsg(ToClient(BarrierResult("bar", false)))
|
||||
b.expectMsg(ToClient(BarrierResult("foo", false)))
|
||||
}
|
||||
|
||||
"not really fail after barrier timeout" in {
|
||||
val barrier = getController(2)
|
||||
val a, b = TestProbe()
|
||||
val nodeA = NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
val nodeB = NodeInfo(B, AddressFromURIString("akka://sys"), b.ref)
|
||||
barrier ! nodeA
|
||||
barrier ! nodeB
|
||||
a.expectMsg(ToClient(Done))
|
||||
b.expectMsg(ToClient(Done))
|
||||
a.send(barrier, EnterBarrier("right"))
|
||||
EventFilter[BarrierTimeout](occurrences = 1) intercept {
|
||||
Thread.sleep(5000)
|
||||
}
|
||||
b.send(barrier, EnterBarrier("right"))
|
||||
a.expectMsg(ToClient(BarrierResult("right", true)))
|
||||
b.expectMsg(ToClient(BarrierResult("right", true)))
|
||||
}
|
||||
|
||||
"fail if a node registers twice" in {
|
||||
val controller = getController(2)
|
||||
val a, b = TestProbe()
|
||||
val nodeA = NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
val nodeB = NodeInfo(A, AddressFromURIString("akka://sys"), b.ref)
|
||||
controller ! nodeA
|
||||
EventFilter[DuplicateNode](occurrences = 1) intercept {
|
||||
controller ! nodeB
|
||||
}
|
||||
a.expectMsg(ToClient(BarrierResult("initial startup", false)))
|
||||
b.expectMsg(ToClient(BarrierResult("initial startup", false)))
|
||||
}
|
||||
|
||||
"fail subsequent barriers if a node registers twice" in {
|
||||
val controller = getController(1)
|
||||
val a, b = TestProbe()
|
||||
val nodeA = NodeInfo(A, AddressFromURIString("akka://sys"), a.ref)
|
||||
val nodeB = NodeInfo(A, AddressFromURIString("akka://sys"), b.ref)
|
||||
controller ! nodeA
|
||||
a.expectMsg(ToClient(Done))
|
||||
EventFilter[DuplicateNode](occurrences = 1) intercept {
|
||||
controller ! nodeB
|
||||
b.expectMsg(ToClient(BarrierResult("initial startup", false)))
|
||||
}
|
||||
a.send(controller, EnterBarrier("x"))
|
||||
a.expectMsg(ToClient(BarrierResult("x", false)))
|
||||
}
|
||||
|
||||
"finally have no failure messages left" in {
|
||||
expectNoMsg(1 second)
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
private def getController(participants: Int): ActorRef = {
|
||||
system.actorOf(Props(new Actor {
|
||||
val controller = context.actorOf(Props(new Controller(participants, new InetSocketAddress(InetAddress.getLocalHost, 0))))
|
||||
controller ! GetSockAddr
|
||||
override def supervisorStrategy = OneForOneStrategy() {
|
||||
case x ⇒ testActor ! Failed(controller, x); SupervisorStrategy.Restart
|
||||
}
|
||||
def receive = {
|
||||
case x: InetSocketAddress ⇒ testActor ! controller
|
||||
}
|
||||
}))
|
||||
expectMsgType[ActorRef]
|
||||
}
|
||||
|
||||
/**
|
||||
* Produce a BarrierCoordinator which is supervised with a strategy which
|
||||
* forwards all failures to the testActor.
|
||||
*/
|
||||
private def getBarrier(): ActorRef = {
|
||||
system.actorOf(Props(new Actor {
|
||||
val barrier = context.actorOf(Props[BarrierCoordinator])
|
||||
override def supervisorStrategy = OneForOneStrategy() {
|
||||
case x ⇒ testActor ! Failed(barrier, x); SupervisorStrategy.Restart
|
||||
}
|
||||
def receive = {
|
||||
case _ ⇒ sender ! barrier
|
||||
}
|
||||
})) ! ""
|
||||
expectMsgType[ActorRef]
|
||||
}
|
||||
|
||||
private def noMsg(probes: TestProbe*) {
|
||||
expectNoMsg(1 second)
|
||||
probes foreach (_.msgAvailable must be(false))
|
||||
}
|
||||
|
||||
}
|
||||
|
|
@ -0,0 +1,43 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.remote.testconductor
|
||||
|
||||
import akka.testkit.AkkaSpec
|
||||
import akka.actor.Props
|
||||
import akka.testkit.ImplicitSender
|
||||
import akka.remote.testconductor.Controller.NodeInfo
|
||||
import akka.actor.AddressFromURIString
|
||||
import java.net.InetSocketAddress
|
||||
import java.net.InetAddress
|
||||
|
||||
object ControllerSpec {
|
||||
val config = """
|
||||
akka.testconductor.barrier-timeout = 5s
|
||||
akka.actor.provider = akka.remote.RemoteActorRefProvider
|
||||
akka.remote.netty.port = 0
|
||||
akka.actor.debug.fsm = on
|
||||
akka.actor.debug.lifecycle = on
|
||||
"""
|
||||
}
|
||||
|
||||
class ControllerSpec extends AkkaSpec(ControllerSpec.config) with ImplicitSender {
|
||||
|
||||
val A = RoleName("a")
|
||||
val B = RoleName("b")
|
||||
|
||||
"A Controller" must {
|
||||
|
||||
"publish its nodes" in {
|
||||
val c = system.actorOf(Props(new Controller(1, new InetSocketAddress(InetAddress.getLocalHost, 0))))
|
||||
c ! NodeInfo(A, AddressFromURIString("akka://sys"), testActor)
|
||||
expectMsg(ToClient(Done))
|
||||
c ! NodeInfo(B, AddressFromURIString("akka://sys"), testActor)
|
||||
expectMsg(ToClient(Done))
|
||||
c ! Controller.GetNodes
|
||||
expectMsgType[Iterable[RoleName]].toSet must be(Set(A, B))
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
}
|
||||
|
|
@ -0,0 +1,191 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2012 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
package akka.remote.testkit
|
||||
|
||||
import akka.testkit.AkkaSpec
|
||||
import akka.actor.ActorSystem
|
||||
import akka.remote.testconductor.TestConductor
|
||||
import java.net.InetAddress
|
||||
import java.net.InetSocketAddress
|
||||
import akka.remote.testconductor.TestConductorExt
|
||||
import com.typesafe.config.Config
|
||||
import com.typesafe.config.ConfigFactory
|
||||
import akka.dispatch.Await.Awaitable
|
||||
import akka.dispatch.Await
|
||||
import akka.util.Duration
|
||||
import akka.actor.ActorPath
|
||||
import akka.actor.RootActorPath
|
||||
import akka.remote.testconductor.RoleName
|
||||
|
||||
/**
|
||||
* Configure the role names and participants of the test, including configuration settings.
|
||||
*/
|
||||
abstract class MultiNodeConfig {
|
||||
|
||||
private var _commonConf: Option[Config] = None
|
||||
private var _nodeConf = Map[RoleName, Config]()
|
||||
private var _roles = Seq[RoleName]()
|
||||
|
||||
/**
|
||||
* Register a common base config for all test participants, if so desired.
|
||||
*/
|
||||
def commonConfig(config: Config): Unit = _commonConf = Some(config)
|
||||
|
||||
/**
|
||||
* Register a config override for a specific participant.
|
||||
*/
|
||||
def nodeConfig(role: RoleName, config: Config): Unit = _nodeConf += role -> config
|
||||
|
||||
/**
|
||||
* Include for verbose debug logging
|
||||
* @param on when `true` debug Config is returned, otherwise empty Config
|
||||
*/
|
||||
def debugConfig(on: Boolean): Config =
|
||||
if (on)
|
||||
ConfigFactory.parseString("""
|
||||
akka.loglevel = DEBUG
|
||||
akka.remote {
|
||||
log-received-messages = on
|
||||
log-sent-messages = on
|
||||
}
|
||||
akka.actor.debug {
|
||||
receive = on
|
||||
fsm = on
|
||||
}
|
||||
""")
|
||||
else ConfigFactory.empty
|
||||
|
||||
/**
|
||||
* Construct a RoleName and return it, to be used as an identifier in the
|
||||
* test. Registration of a role name creates a role which then needs to be
|
||||
* filled.
|
||||
*/
|
||||
def role(name: String): RoleName = {
|
||||
if (_roles exists (_.name == name)) throw new IllegalArgumentException("non-unique role name " + name)
|
||||
val r = RoleName(name)
|
||||
_roles :+= r
|
||||
r
|
||||
}
|
||||
|
||||
private[testkit] lazy val mySelf: RoleName = {
|
||||
require(_roles.size > MultiNodeSpec.selfIndex, "not enough roles declared for this test")
|
||||
_roles(MultiNodeSpec.selfIndex)
|
||||
}
|
||||
|
||||
private[testkit] def config: Config = {
|
||||
val configs = (_nodeConf get mySelf).toList ::: _commonConf.toList ::: MultiNodeSpec.nodeConfig :: AkkaSpec.testConf :: Nil
|
||||
configs reduce (_ withFallback _)
|
||||
}
|
||||
|
||||
}
|
||||
|
||||
object MultiNodeSpec {
|
||||
|
||||
/**
|
||||
* Names (or IP addresses; must be resolvable using InetAddress.getByName)
|
||||
* of all nodes taking part in this test, including symbolic name and host
|
||||
* definition:
|
||||
*
|
||||
* {{{
|
||||
* -D"multinode.hosts=host1@workerA.example.com,host2@workerB.example.com"
|
||||
* }}}
|
||||
*/
|
||||
val nodeNames: Seq[String] = Vector.empty ++ (
|
||||
Option(System.getProperty("multinode.hosts")) getOrElse
|
||||
(throw new IllegalStateException("need system property multinode.hosts to be set")) split ",")
|
||||
|
||||
require(nodeNames != List(""), "multinode.hosts must not be empty")
|
||||
|
||||
/**
|
||||
* Index of this node in the nodeNames / nodeAddresses lists. The TestConductor
|
||||
* is started in “controller” mode on selfIndex 0, i.e. there you can inject
|
||||
* failures and shutdown other nodes etc.
|
||||
*/
|
||||
val selfIndex = Option(Integer.getInteger("multinode.index")) getOrElse
|
||||
(throw new IllegalStateException("need system property multinode.index to be set"))
|
||||
|
||||
require(selfIndex >= 0 && selfIndex < nodeNames.size, "selfIndex out of bounds: " + selfIndex)
|
||||
|
||||
val nodeConfig = AkkaSpec.mapToConfig(Map(
|
||||
"akka.actor.provider" -> "akka.remote.RemoteActorRefProvider",
|
||||
"akka.remote.transport" -> "akka.remote.testconductor.TestConductorTransport",
|
||||
"akka.remote.netty.hostname" -> nodeNames(selfIndex),
|
||||
"akka.remote.netty.port" -> 0))
|
||||
|
||||
}
|
||||
|
||||
abstract class MultiNodeSpec(val mySelf: RoleName, _system: ActorSystem) extends AkkaSpec(_system) {
|
||||
|
||||
import MultiNodeSpec._
|
||||
|
||||
def this(config: MultiNodeConfig) = this(config.mySelf, ActorSystem(AkkaSpec.getCallerName, config.config))
|
||||
|
||||
/*
|
||||
* Test Class Interface
|
||||
*/
|
||||
|
||||
/**
|
||||
* TO BE DEFINED BY USER: Defines the number of participants required for starting the test. This
|
||||
* might not be equals to the number of nodes available to the test.
|
||||
*
|
||||
* Must be a `def`:
|
||||
* {{{
|
||||
* def initialParticipants = 5
|
||||
* }}}
|
||||
*/
|
||||
def initialParticipants: Int
|
||||
require(initialParticipants > 0, "initialParticipants must be a 'def' or early initializer, and it must be greater zero")
|
||||
require(initialParticipants <= nodeNames.size, "not enough nodes to run this test")
|
||||
|
||||
/**
|
||||
* Access to the barriers, failure injection, etc. The extension will have
|
||||
* been started either in Conductor or Player mode when the constructor of
|
||||
* MultiNodeSpec finishes, i.e. do not call the start*() methods yourself!
|
||||
*/
|
||||
val testConductor: TestConductorExt = TestConductor(system)
|
||||
|
||||
/**
|
||||
* Execute the given block of code only on the given nodes (names according
|
||||
* to the `roleMap`).
|
||||
*/
|
||||
def runOn(nodes: RoleName*)(thunk: ⇒ Unit): Unit = {
|
||||
if (nodes exists (_ == mySelf)) {
|
||||
thunk
|
||||
}
|
||||
}
|
||||
|
||||
def ifNode[T](nodes: RoleName*)(yes: ⇒ T)(no: ⇒ T): T = {
|
||||
if (nodes exists (_ == mySelf)) yes else no
|
||||
}
|
||||
|
||||
/**
|
||||
* Query the controller for the transport address of the given node (by role name) and
|
||||
* return that as an ActorPath for easy composition:
|
||||
*
|
||||
* {{{
|
||||
* val serviceA = system.actorFor(node("master") / "user" / "serviceA")
|
||||
* }}}
|
||||
*/
|
||||
def node(role: RoleName): ActorPath = RootActorPath(testConductor.getAddressFor(role).await)
|
||||
|
||||
/**
|
||||
* Enrich `.await()` onto all Awaitables, using BarrierTimeout.
|
||||
*/
|
||||
implicit def awaitHelper[T](w: Awaitable[T]) = new AwaitHelper(w)
|
||||
class AwaitHelper[T](w: Awaitable[T]) {
|
||||
def await: T = Await.result(w, testConductor.Settings.BarrierTimeout.duration)
|
||||
}
|
||||
|
||||
/*
|
||||
* Implementation (i.e. wait for start etc.)
|
||||
*/
|
||||
|
||||
private val controllerAddr = new InetSocketAddress(nodeNames(0), 4711)
|
||||
if (selfIndex == 0) {
|
||||
testConductor.startController(initialParticipants, mySelf, controllerAddr).await
|
||||
} else {
|
||||
testConductor.startClient(mySelf, controllerAddr).await
|
||||
}
|
||||
|
||||
}
|
||||
|
|
@ -100,8 +100,6 @@ private[akka] class ActiveRemoteClient private[akka] (
|
|||
private var connection: ChannelFuture = _
|
||||
@volatile
|
||||
private[remote] var openChannels: DefaultChannelGroup = _
|
||||
@volatile
|
||||
private var executionHandler: ExecutionHandler = _
|
||||
|
||||
@volatile
|
||||
private var reconnectionTimeWindowStart = 0L
|
||||
|
|
@ -144,9 +142,8 @@ private[akka] class ActiveRemoteClient private[akka] (
|
|||
runSwitch switchOn {
|
||||
openChannels = new DefaultDisposableChannelGroup(classOf[RemoteClient].getName)
|
||||
|
||||
executionHandler = new ExecutionHandler(netty.executor)
|
||||
val b = new ClientBootstrap(netty.clientChannelFactory)
|
||||
b.setPipelineFactory(new ActiveRemoteClientPipelineFactory(name, b, executionHandler, remoteAddress, localAddress, this))
|
||||
b.setPipelineFactory(netty.createPipeline(new ActiveRemoteClientHandler(name, b, remoteAddress, localAddress, netty.timer, this), true))
|
||||
b.setOption("tcpNoDelay", true)
|
||||
b.setOption("keepAlive", true)
|
||||
b.setOption("connectTimeoutMillis", settings.ConnectionTimeout.toMillis)
|
||||
|
|
@ -164,6 +161,7 @@ private[akka] class ActiveRemoteClient private[akka] (
|
|||
notifyListeners(RemoteClientError(connection.getCause, netty, remoteAddress))
|
||||
false
|
||||
} else {
|
||||
ChannelAddress.set(connection.getChannel, Some(remoteAddress))
|
||||
sendSecureCookie(connection)
|
||||
notifyListeners(RemoteClientStarted(netty, remoteAddress))
|
||||
true
|
||||
|
|
@ -187,14 +185,15 @@ private[akka] class ActiveRemoteClient private[akka] (
|
|||
|
||||
notifyListeners(RemoteClientShutdown(netty, remoteAddress))
|
||||
try {
|
||||
if ((connection ne null) && (connection.getChannel ne null))
|
||||
if ((connection ne null) && (connection.getChannel ne null)) {
|
||||
ChannelAddress.remove(connection.getChannel)
|
||||
connection.getChannel.close()
|
||||
}
|
||||
} finally {
|
||||
try {
|
||||
if (openChannels ne null) openChannels.close.awaitUninterruptibly()
|
||||
} finally {
|
||||
connection = null
|
||||
executionHandler = null
|
||||
}
|
||||
}
|
||||
|
||||
|
|
@ -307,35 +306,9 @@ private[akka] class ActiveRemoteClientHandler(
|
|||
}
|
||||
}
|
||||
|
||||
private[akka] class ActiveRemoteClientPipelineFactory(
|
||||
name: String,
|
||||
bootstrap: ClientBootstrap,
|
||||
executionHandler: ExecutionHandler,
|
||||
remoteAddress: Address,
|
||||
localAddress: Address,
|
||||
client: ActiveRemoteClient) extends ChannelPipelineFactory {
|
||||
|
||||
import client.netty.settings
|
||||
|
||||
def getPipeline: ChannelPipeline = {
|
||||
val timeout = new IdleStateHandler(client.netty.timer,
|
||||
settings.ReadTimeout.toSeconds.toInt,
|
||||
settings.WriteTimeout.toSeconds.toInt,
|
||||
settings.AllTimeout.toSeconds.toInt)
|
||||
val lenDec = new LengthFieldBasedFrameDecoder(settings.MessageFrameSize, 0, 4, 0, 4)
|
||||
val lenPrep = new LengthFieldPrepender(4)
|
||||
val messageDec = new RemoteMessageDecoder
|
||||
val messageEnc = new RemoteMessageEncoder(client.netty)
|
||||
val remoteClient = new ActiveRemoteClientHandler(name, bootstrap, remoteAddress, localAddress, client.netty.timer, client)
|
||||
|
||||
new StaticChannelPipeline(timeout, lenDec, messageDec, lenPrep, messageEnc, executionHandler, remoteClient)
|
||||
}
|
||||
}
|
||||
|
||||
private[akka] class PassiveRemoteClient(val currentChannel: Channel,
|
||||
netty: NettyRemoteTransport,
|
||||
remoteAddress: Address)
|
||||
extends RemoteClient(netty, remoteAddress) {
|
||||
remoteAddress: Address) extends RemoteClient(netty, remoteAddress) {
|
||||
|
||||
def connect(reconnectIfAlreadyConnected: Boolean = false): Boolean = runSwitch switchOn {
|
||||
netty.notifyListeners(RemoteClientStarted(netty, remoteAddress))
|
||||
|
|
|
|||
|
|
@ -12,9 +12,11 @@ import java.util.concurrent.Executors
|
|||
import scala.collection.mutable.HashMap
|
||||
import org.jboss.netty.channel.group.{ DefaultChannelGroup, ChannelGroupFuture }
|
||||
import org.jboss.netty.channel.socket.nio.NioClientSocketChannelFactory
|
||||
import org.jboss.netty.channel.{ ChannelHandlerContext, Channel }
|
||||
import org.jboss.netty.channel.{ ChannelHandlerContext, Channel, StaticChannelPipeline, ChannelHandler, ChannelPipelineFactory, ChannelLocal }
|
||||
import org.jboss.netty.handler.codec.frame.{ LengthFieldPrepender, LengthFieldBasedFrameDecoder }
|
||||
import org.jboss.netty.handler.codec.protobuf.{ ProtobufEncoder, ProtobufDecoder }
|
||||
import org.jboss.netty.handler.execution.OrderedMemoryAwareThreadPoolExecutor
|
||||
import org.jboss.netty.handler.execution.{ ExecutionHandler, OrderedMemoryAwareThreadPoolExecutor }
|
||||
import org.jboss.netty.handler.timeout.IdleStateHandler
|
||||
import org.jboss.netty.util.HashedWheelTimer
|
||||
import akka.event.Logging
|
||||
import akka.remote.RemoteProtocol.AkkaRemoteProtocol
|
||||
|
|
@ -22,6 +24,10 @@ import akka.remote.{ RemoteTransportException, RemoteTransport, RemoteActorRefPr
|
|||
import akka.util.NonFatal
|
||||
import akka.actor.{ ExtendedActorSystem, Address, ActorRef }
|
||||
|
||||
object ChannelAddress extends ChannelLocal[Option[Address]] {
|
||||
override def initialValue(ch: Channel): Option[Address] = None
|
||||
}
|
||||
|
||||
/**
|
||||
* Provides the implementation of the Netty remote support
|
||||
*/
|
||||
|
|
@ -31,28 +37,111 @@ private[akka] class NettyRemoteTransport(_system: ExtendedActorSystem, _provider
|
|||
|
||||
val settings = new NettySettings(remoteSettings.config.getConfig("akka.remote.netty"), remoteSettings.systemName)
|
||||
|
||||
// TODO replace by system.scheduler
|
||||
val timer: HashedWheelTimer = new HashedWheelTimer(system.threadFactory)
|
||||
|
||||
val executor = new OrderedMemoryAwareThreadPoolExecutor(
|
||||
// TODO make configurable/shareable with server socket factory
|
||||
val clientChannelFactory = new NioClientSocketChannelFactory(
|
||||
Executors.newCachedThreadPool(system.threadFactory),
|
||||
Executors.newCachedThreadPool(system.threadFactory))
|
||||
|
||||
/**
|
||||
* Backing scaffolding for the default implementation of NettyRemoteSupport.createPipeline.
|
||||
*/
|
||||
object PipelineFactory {
|
||||
/**
|
||||
* Construct a StaticChannelPipeline from a sequence of handlers; to be used
|
||||
* in implementations of ChannelPipelineFactory.
|
||||
*/
|
||||
def apply(handlers: Seq[ChannelHandler]): StaticChannelPipeline = new StaticChannelPipeline(handlers: _*)
|
||||
|
||||
/**
|
||||
* Constructs the NettyRemoteTransport default pipeline with the give “head” handler, which
|
||||
* is taken by-name to allow it not to be shared across pipelines.
|
||||
*
|
||||
* @param withTimeout determines whether an IdleStateHandler shall be included
|
||||
*/
|
||||
def apply(endpoint: ⇒ Seq[ChannelHandler], withTimeout: Boolean): ChannelPipelineFactory =
|
||||
new ChannelPipelineFactory {
|
||||
def getPipeline = apply(defaultStack(withTimeout) ++ endpoint)
|
||||
}
|
||||
|
||||
/**
|
||||
* Construct a default protocol stack, excluding the “head” handler (i.e. the one which
|
||||
* actually dispatches the received messages to the local target actors).
|
||||
*/
|
||||
def defaultStack(withTimeout: Boolean): Seq[ChannelHandler] =
|
||||
(if (withTimeout) timeout :: Nil else Nil) :::
|
||||
msgFormat :::
|
||||
authenticator :::
|
||||
executionHandler ::
|
||||
Nil
|
||||
|
||||
/**
|
||||
* Construct an IdleStateHandler which uses [[akka.remote.netty.NettyRemoteTransport]].timer.
|
||||
*/
|
||||
def timeout = new IdleStateHandler(timer,
|
||||
settings.ReadTimeout.toSeconds.toInt,
|
||||
settings.WriteTimeout.toSeconds.toInt,
|
||||
settings.AllTimeout.toSeconds.toInt)
|
||||
|
||||
/**
|
||||
* Construct frame&protobuf encoder/decoder.
|
||||
*/
|
||||
def msgFormat = new LengthFieldBasedFrameDecoder(settings.MessageFrameSize, 0, 4, 0, 4) ::
|
||||
new LengthFieldPrepender(4) ::
|
||||
new RemoteMessageDecoder ::
|
||||
new RemoteMessageEncoder(NettyRemoteTransport.this) ::
|
||||
Nil
|
||||
|
||||
/**
|
||||
* Construct an ExecutionHandler which is used to ensure that message dispatch does not
|
||||
* happen on a netty thread (that could be bad if re-sending over the network for
|
||||
* remote-deployed actors).
|
||||
*/
|
||||
val executionHandler = new ExecutionHandler(new OrderedMemoryAwareThreadPoolExecutor(
|
||||
settings.ExecutionPoolSize,
|
||||
settings.MaxChannelMemorySize,
|
||||
settings.MaxTotalMemorySize,
|
||||
settings.ExecutionPoolKeepalive.length,
|
||||
settings.ExecutionPoolKeepalive.unit,
|
||||
system.threadFactory)
|
||||
system.threadFactory))
|
||||
|
||||
val clientChannelFactory = new NioClientSocketChannelFactory(
|
||||
Executors.newCachedThreadPool(system.threadFactory),
|
||||
Executors.newCachedThreadPool(system.threadFactory))
|
||||
/**
|
||||
* Construct and authentication handler which uses the SecureCookie to somewhat
|
||||
* protect the TCP port from unauthorized use (don’t rely on it too much, though,
|
||||
* as this is NOT a cryptographic feature).
|
||||
*/
|
||||
def authenticator = if (settings.RequireCookie) new RemoteServerAuthenticationHandler(settings.SecureCookie) :: Nil else Nil
|
||||
}
|
||||
|
||||
/**
|
||||
* This method is factored out to provide an extension point in case the
|
||||
* pipeline shall be changed. It is recommended to use
|
||||
*/
|
||||
def createPipeline(endpoint: ⇒ ChannelHandler, withTimeout: Boolean): ChannelPipelineFactory =
|
||||
PipelineFactory(Seq(endpoint), withTimeout)
|
||||
|
||||
private val remoteClients = new HashMap[Address, RemoteClient]
|
||||
private val clientsLock = new ReentrantReadWriteLock
|
||||
|
||||
override protected def useUntrustedMode = remoteSettings.UntrustedMode
|
||||
|
||||
val server = try new NettyRemoteServer(this) catch {
|
||||
case ex ⇒ shutdown(); throw ex
|
||||
}
|
||||
val server: NettyRemoteServer = try createServer() catch { case NonFatal(ex) ⇒ shutdown(); throw ex }
|
||||
|
||||
/**
|
||||
* Override this method to inject a subclass of NettyRemoteServer instead of
|
||||
* the normal one, e.g. for inserting security hooks. If this method throws
|
||||
* an exception, the transport will shut itself down and re-throw.
|
||||
*/
|
||||
protected def createServer(): NettyRemoteServer = new NettyRemoteServer(this)
|
||||
|
||||
/**
|
||||
* Override this method to inject a subclass of RemoteClient instead of
|
||||
* the normal one, e.g. for inserting security hooks. Get this transport’s
|
||||
* address from `this.address`.
|
||||
*/
|
||||
protected def createClient(recipient: Address): RemoteClient = new ActiveRemoteClient(this, recipient, address)
|
||||
|
||||
// the address is set in start() or from the RemoteServerHandler, whichever comes first
|
||||
private val _address = new AtomicReference[Address]
|
||||
|
|
@ -91,11 +180,7 @@ private[akka] class NettyRemoteTransport(_system: ExtendedActorSystem, _provider
|
|||
try {
|
||||
timer.stop()
|
||||
} finally {
|
||||
try {
|
||||
clientChannelFactory.releaseExternalResources()
|
||||
} finally {
|
||||
executor.shutdown()
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
@ -121,7 +206,7 @@ private[akka] class NettyRemoteTransport(_system: ExtendedActorSystem, _provider
|
|||
//Recheck for addition, race between upgrades
|
||||
case Some(client) ⇒ client //If already populated by other writer
|
||||
case None ⇒ //Populate map
|
||||
val client = new ActiveRemoteClient(this, recipientAddress, address)
|
||||
val client = createClient(recipientAddress)
|
||||
remoteClients += recipientAddress -> client
|
||||
client
|
||||
}
|
||||
|
|
|
|||
|
|
@ -35,14 +35,12 @@ private[akka] class NettyRemoteServer(val netty: NettyRemoteTransport) {
|
|||
new NioServerSocketChannelFactory(Executors.newCachedThreadPool(), Executors.newCachedThreadPool())
|
||||
}
|
||||
|
||||
private val executionHandler = new ExecutionHandler(netty.executor)
|
||||
|
||||
// group of open channels, used for clean-up
|
||||
private val openChannels: ChannelGroup = new DefaultDisposableChannelGroup("akka-remote-server")
|
||||
|
||||
private val bootstrap = {
|
||||
val b = new ServerBootstrap(factory)
|
||||
b.setPipelineFactory(new RemoteServerPipelineFactory(openChannels, executionHandler, netty))
|
||||
b.setPipelineFactory(netty.createPipeline(new RemoteServerHandler(openChannels, netty), false))
|
||||
b.setOption("backlog", settings.Backlog)
|
||||
b.setOption("tcpNoDelay", true)
|
||||
b.setOption("child.keepAlive", true)
|
||||
|
|
@ -82,26 +80,6 @@ private[akka] class NettyRemoteServer(val netty: NettyRemoteTransport) {
|
|||
}
|
||||
}
|
||||
|
||||
private[akka] class RemoteServerPipelineFactory(
|
||||
val openChannels: ChannelGroup,
|
||||
val executionHandler: ExecutionHandler,
|
||||
val netty: NettyRemoteTransport) extends ChannelPipelineFactory {
|
||||
|
||||
import netty.settings
|
||||
|
||||
def getPipeline: ChannelPipeline = {
|
||||
val lenDec = new LengthFieldBasedFrameDecoder(settings.MessageFrameSize, 0, 4, 0, 4)
|
||||
val lenPrep = new LengthFieldPrepender(4)
|
||||
val messageDec = new RemoteMessageDecoder
|
||||
val messageEnc = new RemoteMessageEncoder(netty)
|
||||
|
||||
val authenticator = if (settings.RequireCookie) new RemoteServerAuthenticationHandler(settings.SecureCookie) :: Nil else Nil
|
||||
val remoteServer = new RemoteServerHandler(openChannels, netty)
|
||||
val stages: List[ChannelHandler] = lenDec :: messageDec :: lenPrep :: messageEnc :: executionHandler :: authenticator ::: remoteServer :: Nil
|
||||
new StaticChannelPipeline(stages: _*)
|
||||
}
|
||||
}
|
||||
|
||||
@ChannelHandler.Sharable
|
||||
private[akka] class RemoteServerAuthenticationHandler(secureCookie: Option[String]) extends SimpleChannelUpstreamHandler {
|
||||
val authenticated = new AnyRef
|
||||
|
|
@ -134,10 +112,6 @@ private[akka] class RemoteServerHandler(
|
|||
val openChannels: ChannelGroup,
|
||||
val netty: NettyRemoteTransport) extends SimpleChannelUpstreamHandler {
|
||||
|
||||
val channelAddress = new ChannelLocal[Option[Address]](false) {
|
||||
override def initialValue(channel: Channel) = None
|
||||
}
|
||||
|
||||
import netty.settings
|
||||
|
||||
private var addressToSet = true
|
||||
|
|
@ -161,16 +135,16 @@ private[akka] class RemoteServerHandler(
|
|||
override def channelConnected(ctx: ChannelHandlerContext, event: ChannelStateEvent) = ()
|
||||
|
||||
override def channelDisconnected(ctx: ChannelHandlerContext, event: ChannelStateEvent) = {
|
||||
netty.notifyListeners(RemoteServerClientDisconnected(netty, channelAddress.get(ctx.getChannel)))
|
||||
netty.notifyListeners(RemoteServerClientDisconnected(netty, ChannelAddress.get(ctx.getChannel)))
|
||||
}
|
||||
|
||||
override def channelClosed(ctx: ChannelHandlerContext, event: ChannelStateEvent) = {
|
||||
val address = channelAddress.get(ctx.getChannel)
|
||||
val address = ChannelAddress.get(ctx.getChannel)
|
||||
if (address.isDefined && settings.UsePassiveConnections)
|
||||
netty.unbindClient(address.get)
|
||||
|
||||
netty.notifyListeners(RemoteServerClientClosed(netty, address))
|
||||
channelAddress.remove(ctx.getChannel)
|
||||
ChannelAddress.remove(ctx.getChannel)
|
||||
}
|
||||
|
||||
override def messageReceived(ctx: ChannelHandlerContext, event: MessageEvent) = try {
|
||||
|
|
@ -184,7 +158,7 @@ private[akka] class RemoteServerHandler(
|
|||
case CommandType.CONNECT ⇒
|
||||
val origin = instruction.getOrigin
|
||||
val inbound = Address("akka", origin.getSystem, origin.getHostname, origin.getPort)
|
||||
channelAddress.set(event.getChannel, Option(inbound))
|
||||
ChannelAddress.set(event.getChannel, Option(inbound))
|
||||
|
||||
//If we want to reuse the inbound connections as outbound we need to get busy
|
||||
if (settings.UsePassiveConnections)
|
||||
|
|
|
|||
|
|
@ -1,6 +1,7 @@
|
|||
package akka.remote
|
||||
|
||||
import com.typesafe.config.{Config, ConfigFactory}
|
||||
import akka.actor.Address
|
||||
|
||||
trait AbstractRemoteActorMultiJvmSpec {
|
||||
def NrOfNodes: Int
|
||||
|
|
@ -8,7 +9,6 @@ trait AbstractRemoteActorMultiJvmSpec {
|
|||
|
||||
def PortRangeStart = 1990
|
||||
def NodeRange = 1 to NrOfNodes
|
||||
def PortRange = PortRangeStart to NrOfNodes
|
||||
|
||||
private[this] val remotes: IndexedSeq[String] = {
|
||||
val nodesOpt = Option(AkkaRemoteSpec.testNodes).map(_.split(",").toIndexedSeq)
|
||||
|
|
|
|||
0
file-based/mailbox_user__a
Normal file
0
file-based/mailbox_user__a
Normal file
0
file-based/mailbox_user__b
Normal file
0
file-based/mailbox_user__b
Normal file
BIN
file-based/mailbox_user__c
Normal file
BIN
file-based/mailbox_user__c
Normal file
Binary file not shown.
|
|
@ -38,7 +38,7 @@ object AkkaBuild extends Build {
|
|||
sphinxLatex <<= sphinxLatex in LocalProject(docs.id),
|
||||
sphinxPdf <<= sphinxPdf in LocalProject(docs.id)
|
||||
),
|
||||
aggregate = Seq(actor, testkit, actorTests, remote, camel, cluster, slf4j, agent, transactor, mailboxes, zeroMQ, kernel, akkaSbtPlugin, samples, tutorials, docs)
|
||||
aggregate = Seq(actor, testkit, actorTests, remote, remoteTests, camel, cluster, slf4j, agent, transactor, mailboxes, zeroMQ, kernel, akkaSbtPlugin, samples, tutorials, docs)
|
||||
)
|
||||
|
||||
lazy val actor = Project(
|
||||
|
|
@ -86,17 +86,31 @@ object AkkaBuild extends Build {
|
|||
(name: String) => (src ** (name + ".conf")).get.headOption.map("-Dakka.config=" + _.absolutePath).toSeq
|
||||
},
|
||||
scalatestOptions in MultiJvm := Seq("-r", "org.scalatest.akka.QuietReporter"),
|
||||
jvmOptions in MultiJvm := {
|
||||
if (getBoolean("sbt.log.noformat")) Seq("-Dakka.test.nocolor=true") else Nil
|
||||
jvmOptions in MultiJvm := defaultMultiJvmOptions,
|
||||
test in Test <<= ((test in Test), (test in MultiJvm)) map { case x => x }
|
||||
)
|
||||
) configs (MultiJvm)
|
||||
|
||||
lazy val remoteTests = Project(
|
||||
id = "akka-remote-tests",
|
||||
base = file("akka-remote-tests"),
|
||||
dependencies = Seq(remote % "compile;test->test;multi-jvm->multi-jvm", actorTests % "test->test", testkit % "test->test"),
|
||||
settings = defaultSettings ++ multiJvmSettings ++ Seq(
|
||||
// disable parallel tests
|
||||
parallelExecution in Test := false,
|
||||
extraOptions in MultiJvm <<= (sourceDirectory in MultiJvm) { src =>
|
||||
(name: String) => (src ** (name + ".conf")).get.headOption.map("-Dakka.config=" + _.absolutePath).toSeq
|
||||
},
|
||||
test in Test <<= (test in Test) dependsOn (test in MultiJvm)
|
||||
scalatestOptions in MultiJvm := Seq("-r", "org.scalatest.akka.QuietReporter"),
|
||||
jvmOptions in MultiJvm := defaultMultiJvmOptions,
|
||||
test in Test <<= ((test in Test), (test in MultiJvm)) map { case x => x }
|
||||
)
|
||||
) configs (MultiJvm)
|
||||
|
||||
lazy val cluster = Project(
|
||||
id = "akka-cluster",
|
||||
base = file("akka-cluster"),
|
||||
dependencies = Seq(remote, remote % "test->test", testkit % "test->test"),
|
||||
dependencies = Seq(remote, remoteTests % "compile;test->test;multi-jvm->multi-jvm", testkit % "test->test"),
|
||||
settings = defaultSettings ++ multiJvmSettings ++ Seq(
|
||||
libraryDependencies ++= Dependencies.cluster,
|
||||
// disable parallel tests
|
||||
|
|
@ -105,10 +119,8 @@ object AkkaBuild extends Build {
|
|||
(name: String) => (src ** (name + ".conf")).get.headOption.map("-Dakka.config=" + _.absolutePath).toSeq
|
||||
},
|
||||
scalatestOptions in MultiJvm := Seq("-r", "org.scalatest.akka.QuietReporter"),
|
||||
jvmOptions in MultiJvm := {
|
||||
if (getBoolean("sbt.log.noformat")) Seq("-Dakka.test.nocolor=true") else Nil
|
||||
},
|
||||
test in Test <<= (test in Test) dependsOn (test in MultiJvm)
|
||||
jvmOptions in MultiJvm := defaultMultiJvmOptions,
|
||||
test in Test <<= ((test in Test), (test in MultiJvm)) map { case x => x }
|
||||
)
|
||||
) configs (MultiJvm)
|
||||
|
||||
|
|
@ -286,6 +298,14 @@ object AkkaBuild extends Build {
|
|||
|
||||
val defaultExcludedTags = Seq("timing", "long-running")
|
||||
|
||||
val defaultMultiJvmOptions: Seq[String] = {
|
||||
(System.getProperty("akka.test.timefactor") match {
|
||||
case null => Nil
|
||||
case x => List("-Dakka.test.timefactor=" + x)
|
||||
}) :::
|
||||
(if (getBoolean("sbt.log.noformat")) List("-Dakka.test.nocolor=true") else Nil)
|
||||
}
|
||||
|
||||
lazy val defaultSettings = baseSettings ++ formatSettings ++ Seq(
|
||||
resolvers += "Typesafe Repo" at "http://repo.typesafe.com/typesafe/releases/",
|
||||
|
||||
|
|
|
|||
|
|
@ -1,11 +1,13 @@
|
|||
|
||||
resolvers += Classpaths.typesafeResolver
|
||||
|
||||
addSbtPlugin("com.typesafe.sbtmultijvm" % "sbt-multi-jvm" % "0.1.9")
|
||||
addSbtPlugin("com.typesafe.sbtmultijvm" % "sbt-multi-jvm" % "0.2.0-M1")
|
||||
|
||||
addSbtPlugin("com.typesafe.sbtscalariform" % "sbtscalariform" % "0.4.0")
|
||||
|
||||
resolvers ++= Seq(
|
||||
// needed for sbt-assembly, which comes with sbt-multi-jvm
|
||||
Resolver.url("sbtonline", url("http://scalasbt.artifactoryonline.com/scalasbt/sbt-plugin-releases"))(Resolver.ivyStylePatterns),
|
||||
"less is" at "http://repo.lessis.me",
|
||||
"coda" at "http://repo.codahale.com")
|
||||
|
||||
|
|
|
|||
3
scripts/fix-protobuf.sh
Executable file
3
scripts/fix-protobuf.sh
Executable file
|
|
@ -0,0 +1,3 @@
|
|||
#!/bin/bash
|
||||
|
||||
find . -name \*.java -print0 | xargs -0 perl -pi -e 's/\Qprivate Builder(BuilderParent parent)/private Builder(com.google.protobuf.GeneratedMessage.BuilderParent parent)/'
|
||||
Loading…
Add table
Add a link
Reference in a new issue