newActor(() => refactored
This commit is contained in:
parent
57e46e2ecc
commit
ae6eb54ee9
20 changed files with 53 additions and 53 deletions
|
|
@ -94,7 +94,7 @@ object AMQP {
|
||||||
returnListener: Option[ReturnListener],
|
returnListener: Option[ReturnListener],
|
||||||
shutdownListener: Option[ShutdownListener],
|
shutdownListener: Option[ShutdownListener],
|
||||||
initReconnectDelay: Long): ActorRef = {
|
initReconnectDelay: Long): ActorRef = {
|
||||||
val producer = newActor(() => new Producer(
|
val producer = actorOf( new Producer(
|
||||||
new ConnectionFactory(config),
|
new ConnectionFactory(config),
|
||||||
hostname, port,
|
hostname, port,
|
||||||
exchangeName,
|
exchangeName,
|
||||||
|
|
@ -117,7 +117,7 @@ object AMQP {
|
||||||
durable: Boolean,
|
durable: Boolean,
|
||||||
autoDelete: Boolean,
|
autoDelete: Boolean,
|
||||||
configurationArguments: Map[String, AnyRef]): ActorRef = {
|
configurationArguments: Map[String, AnyRef]): ActorRef = {
|
||||||
val consumer = newActor(() => new Consumer(
|
val consumer = actorOf( new Consumer(
|
||||||
new ConnectionFactory(config),
|
new ConnectionFactory(config),
|
||||||
hostname, port,
|
hostname, port,
|
||||||
exchangeName,
|
exchangeName,
|
||||||
|
|
|
||||||
|
|
@ -21,7 +21,7 @@ trait CamelService extends Bootable with Logging {
|
||||||
import CamelContextManager._
|
import CamelContextManager._
|
||||||
|
|
||||||
private[camel] val consumerPublisher = newActor[ConsumerPublisher]
|
private[camel] val consumerPublisher = newActor[ConsumerPublisher]
|
||||||
private[camel] val publishRequestor = newActor(() => new PublishRequestor(consumerPublisher))
|
private[camel] val publishRequestor = actorOf(new PublishRequestor(consumerPublisher))
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Starts the CamelService. Any started actor that is a consumer actor will be (asynchronously)
|
* Starts the CamelService. Any started actor that is a consumer actor will be (asynchronously)
|
||||||
|
|
|
||||||
|
|
@ -36,7 +36,7 @@ class ProducerFeatureTest extends FeatureSpec with BeforeAndAfterAll with Before
|
||||||
|
|
||||||
scenario("produce message sync and receive response") {
|
scenario("produce message sync and receive response") {
|
||||||
given("a registered synchronous two-way producer for endpoint direct:producer-test-2")
|
given("a registered synchronous two-way producer for endpoint direct:producer-test-2")
|
||||||
val producer = newActor(() => new TestProducer("direct:producer-test-2") with Sync)
|
val producer = actorOf(new TestProducer("direct:producer-test-2") with Sync)
|
||||||
producer.start
|
producer.start
|
||||||
|
|
||||||
when("a test message is sent to the producer")
|
when("a test message is sent to the producer")
|
||||||
|
|
@ -50,7 +50,7 @@ class ProducerFeatureTest extends FeatureSpec with BeforeAndAfterAll with Before
|
||||||
|
|
||||||
scenario("produce message async and receive response") {
|
scenario("produce message async and receive response") {
|
||||||
given("a registered asynchronous two-way producer for endpoint direct:producer-test-2")
|
given("a registered asynchronous two-way producer for endpoint direct:producer-test-2")
|
||||||
val producer = newActor(() => new TestProducer("direct:producer-test-2"))
|
val producer = actorOf(new TestProducer("direct:producer-test-2"))
|
||||||
producer.start
|
producer.start
|
||||||
|
|
||||||
when("a test message is sent to the producer")
|
when("a test message is sent to the producer")
|
||||||
|
|
@ -64,7 +64,7 @@ class ProducerFeatureTest extends FeatureSpec with BeforeAndAfterAll with Before
|
||||||
|
|
||||||
scenario("produce message sync and receive failure") {
|
scenario("produce message sync and receive failure") {
|
||||||
given("a registered synchronous two-way producer for endpoint direct:producer-test-2")
|
given("a registered synchronous two-way producer for endpoint direct:producer-test-2")
|
||||||
val producer = newActor(() => new TestProducer("direct:producer-test-2") with Sync)
|
val producer = actorOf(new TestProducer("direct:producer-test-2") with Sync)
|
||||||
producer.start
|
producer.start
|
||||||
|
|
||||||
when("a fail message is sent to the producer")
|
when("a fail message is sent to the producer")
|
||||||
|
|
@ -80,7 +80,7 @@ class ProducerFeatureTest extends FeatureSpec with BeforeAndAfterAll with Before
|
||||||
|
|
||||||
scenario("produce message async and receive failure") {
|
scenario("produce message async and receive failure") {
|
||||||
given("a registered asynchronous two-way producer for endpoint direct:producer-test-2")
|
given("a registered asynchronous two-way producer for endpoint direct:producer-test-2")
|
||||||
val producer = newActor(() => new TestProducer("direct:producer-test-2"))
|
val producer = actorOf(new TestProducer("direct:producer-test-2"))
|
||||||
producer.start
|
producer.start
|
||||||
|
|
||||||
when("a fail message is sent to the producer")
|
when("a fail message is sent to the producer")
|
||||||
|
|
@ -96,7 +96,7 @@ class ProducerFeatureTest extends FeatureSpec with BeforeAndAfterAll with Before
|
||||||
|
|
||||||
scenario("produce message sync oneway") {
|
scenario("produce message sync oneway") {
|
||||||
given("a registered synchronous one-way producer for endpoint direct:producer-test-1")
|
given("a registered synchronous one-way producer for endpoint direct:producer-test-1")
|
||||||
val producer = newActor(() => new TestProducer("direct:producer-test-1") with Sync with Oneway)
|
val producer = actorOf(new TestProducer("direct:producer-test-1") with Sync with Oneway)
|
||||||
producer.start
|
producer.start
|
||||||
|
|
||||||
when("a test message is sent to the producer")
|
when("a test message is sent to the producer")
|
||||||
|
|
@ -109,7 +109,7 @@ class ProducerFeatureTest extends FeatureSpec with BeforeAndAfterAll with Before
|
||||||
|
|
||||||
scenario("produce message async oneway") {
|
scenario("produce message async oneway") {
|
||||||
given("a registered asynchronous one-way producer for endpoint direct:producer-test-1")
|
given("a registered asynchronous one-way producer for endpoint direct:producer-test-1")
|
||||||
val producer = newActor(() => new TestProducer("direct:producer-test-1") with Oneway)
|
val producer = actorOf(new TestProducer("direct:producer-test-1") with Oneway)
|
||||||
producer.start
|
producer.start
|
||||||
|
|
||||||
when("a test message is sent to the producer")
|
when("a test message is sent to the producer")
|
||||||
|
|
|
||||||
|
|
@ -18,7 +18,7 @@ class ActorProducerTest extends JUnitSuite with BeforeAndAfterAll {
|
||||||
@After def tearDown = ActorRegistry.shutdownAll
|
@After def tearDown = ActorRegistry.shutdownAll
|
||||||
|
|
||||||
@Test def shouldSendMessageToActor = {
|
@Test def shouldSendMessageToActor = {
|
||||||
val actor = newActor(() => new Tester with Retain with Countdown[Message])
|
val actor = actorOf(new Tester with Retain with Countdown[Message])
|
||||||
val endpoint = mockEndpoint("actor:uuid:%s" format actor.uuid)
|
val endpoint = mockEndpoint("actor:uuid:%s" format actor.uuid)
|
||||||
val exchange = endpoint.createExchange(ExchangePattern.InOnly)
|
val exchange = endpoint.createExchange(ExchangePattern.InOnly)
|
||||||
actor.start
|
actor.start
|
||||||
|
|
@ -31,7 +31,7 @@ class ActorProducerTest extends JUnitSuite with BeforeAndAfterAll {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Test def shouldSendMessageToActorAndReceiveResponse = {
|
@Test def shouldSendMessageToActorAndReceiveResponse = {
|
||||||
val actor = newActor(() => new Tester with Respond {
|
val actor = actorOf(new Tester with Respond {
|
||||||
override def response(msg: Message) = Message(super.response(msg), Map("k2" -> "v2"))
|
override def response(msg: Message) = Message(super.response(msg), Map("k2" -> "v2"))
|
||||||
})
|
})
|
||||||
val endpoint = mockEndpoint("actor:uuid:%s" format actor.uuid)
|
val endpoint = mockEndpoint("actor:uuid:%s" format actor.uuid)
|
||||||
|
|
@ -45,7 +45,7 @@ class ActorProducerTest extends JUnitSuite with BeforeAndAfterAll {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Test def shouldSendMessageToActorAndReceiveFailure = {
|
@Test def shouldSendMessageToActorAndReceiveFailure = {
|
||||||
val actor = newActor(() => new Tester with Respond {
|
val actor = actorOf(new Tester with Respond {
|
||||||
override def response(msg: Message) = Failure(new Exception("testmsg"), Map("k3" -> "v3"))
|
override def response(msg: Message) = Failure(new Exception("testmsg"), Map("k3" -> "v3"))
|
||||||
})
|
})
|
||||||
val endpoint = mockEndpoint("actor:uuid:%s" format actor.uuid)
|
val endpoint = mockEndpoint("actor:uuid:%s" format actor.uuid)
|
||||||
|
|
@ -60,7 +60,7 @@ class ActorProducerTest extends JUnitSuite with BeforeAndAfterAll {
|
||||||
}
|
}
|
||||||
|
|
||||||
@Test def shouldSendMessageToActorAndTimeout: Unit = {
|
@Test def shouldSendMessageToActorAndTimeout: Unit = {
|
||||||
val actor = newActor(() => new Tester {
|
val actor = actorOf(new Tester {
|
||||||
timeout = 1
|
timeout = 1
|
||||||
})
|
})
|
||||||
val endpoint = mockEndpoint("actor:uuid:%s" format actor.uuid)
|
val endpoint = mockEndpoint("actor:uuid:%s" format actor.uuid)
|
||||||
|
|
|
||||||
|
|
@ -38,7 +38,7 @@ class CamelServiceFeatureTest extends FeatureSpec with BeforeAndAfterAll with Gi
|
||||||
override protected def beforeAll = {
|
override protected def beforeAll = {
|
||||||
ActorRegistry.shutdownAll
|
ActorRegistry.shutdownAll
|
||||||
// register test consumer before starting the CamelService
|
// register test consumer before starting the CamelService
|
||||||
newActor(() => new TestConsumer("direct:publish-test-1")).start
|
actorOf(new TestConsumer("direct:publish-test-1")).start
|
||||||
// Consigure a custom camel route
|
// Consigure a custom camel route
|
||||||
CamelContextManager.init
|
CamelContextManager.init
|
||||||
CamelContextManager.context.addRoutes(new TestRoute)
|
CamelContextManager.context.addRoutes(new TestRoute)
|
||||||
|
|
@ -61,7 +61,7 @@ class CamelServiceFeatureTest extends FeatureSpec with BeforeAndAfterAll with Gi
|
||||||
|
|
||||||
given("two consumer actors registered before and after CamelService startup")
|
given("two consumer actors registered before and after CamelService startup")
|
||||||
service.consumerPublisher.actor.asInstanceOf[ConsumerPublisher].expectPublishCount(1)
|
service.consumerPublisher.actor.asInstanceOf[ConsumerPublisher].expectPublishCount(1)
|
||||||
newActor(() => new TestConsumer("direct:publish-test-2")).start
|
actorOf(new TestConsumer("direct:publish-test-2")).start
|
||||||
|
|
||||||
when("requests are sent to these actors")
|
when("requests are sent to these actors")
|
||||||
service.consumerPublisher.actor.asInstanceOf[ConsumerPublisher].awaitPublish
|
service.consumerPublisher.actor.asInstanceOf[ConsumerPublisher].awaitPublish
|
||||||
|
|
|
||||||
|
|
@ -24,12 +24,12 @@ class PublishRequestorTest extends JUnitSuite {
|
||||||
@After def tearDown = ActorRegistry.shutdownAll
|
@After def tearDown = ActorRegistry.shutdownAll
|
||||||
|
|
||||||
@Test def shouldReceivePublishRequestOnActorRegisteredEvent = {
|
@Test def shouldReceivePublishRequestOnActorRegisteredEvent = {
|
||||||
val consumer = newActor(() => new Actor with Consumer {
|
val consumer = actorOf(new Actor with Consumer {
|
||||||
def endpointUri = "mock:test"
|
def endpointUri = "mock:test"
|
||||||
protected def receive = null
|
protected def receive = null
|
||||||
}).start
|
}).start
|
||||||
val publisher = newActor(() => new PublisherMock with Countdown[Publish])
|
val publisher = actorOf(new PublisherMock with Countdown[Publish])
|
||||||
val requestor = newActor(() => new PublishRequestor(publisher))
|
val requestor = actorOf(new PublishRequestor(publisher))
|
||||||
publisher.start
|
publisher.start
|
||||||
requestor.start
|
requestor.start
|
||||||
requestor.!(ActorRegistered(consumer))(None)
|
requestor.!(ActorRegistered(consumer))(None)
|
||||||
|
|
|
||||||
|
|
@ -97,13 +97,13 @@ object Actor extends Logging {
|
||||||
* This function should <b>NOT</b> be used for remote actors.
|
* This function should <b>NOT</b> be used for remote actors.
|
||||||
* <pre>
|
* <pre>
|
||||||
* import Actor._
|
* import Actor._
|
||||||
* val actor = newActor(() => new MyActor)
|
* val actor = actorOf(new MyActor)
|
||||||
* actor.start
|
* actor.start
|
||||||
* actor ! message
|
* actor ! message
|
||||||
* actor.stop
|
* actor.stop
|
||||||
* </pre>
|
* </pre>
|
||||||
*/
|
*/
|
||||||
def newActor(factory: () => Actor): ActorRef = new ActorRef(factory)
|
def actorOf(factory: => Actor): ActorRef = new ActorRef(() => factory)
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Use to create an anonymous event-driven actor.
|
* Use to create an anonymous event-driven actor.
|
||||||
|
|
@ -288,7 +288,7 @@ object ActorRef {
|
||||||
* <pre>
|
* <pre>
|
||||||
* import Actor._
|
* import Actor._
|
||||||
*
|
*
|
||||||
* val actor = newActor(() => new MyActor(...))
|
* val actor = actorOf(new MyActor(...))
|
||||||
* actor.start
|
* actor.start
|
||||||
* actor ! message
|
* actor ! message
|
||||||
* actor.stop
|
* actor.stop
|
||||||
|
|
|
||||||
|
|
@ -27,7 +27,7 @@ object Patterns {
|
||||||
/** Creates a LoadBalancer from the thunk-supplied InfiniteIterator
|
/** Creates a LoadBalancer from the thunk-supplied InfiniteIterator
|
||||||
*/
|
*/
|
||||||
def loadBalancerActor(actors: => InfiniteIterator[ActorRef]): ActorRef =
|
def loadBalancerActor(actors: => InfiniteIterator[ActorRef]): ActorRef =
|
||||||
newActor(() => new Actor with LoadBalancer {
|
actorOf(new Actor with LoadBalancer {
|
||||||
start
|
start
|
||||||
val seq = actors
|
val seq = actors
|
||||||
})
|
})
|
||||||
|
|
@ -35,7 +35,7 @@ object Patterns {
|
||||||
/** Creates a Dispatcher given a routing and a message-transforming function
|
/** Creates a Dispatcher given a routing and a message-transforming function
|
||||||
*/
|
*/
|
||||||
def dispatcherActor(routing: PF[Any, ActorRef], msgTransformer: (Any) => Any): ActorRef =
|
def dispatcherActor(routing: PF[Any, ActorRef], msgTransformer: (Any) => Any): ActorRef =
|
||||||
newActor(() => new Actor with Dispatcher {
|
actorOf(new Actor with Dispatcher {
|
||||||
start
|
start
|
||||||
override def transform(msg: Any) = msgTransformer(msg)
|
override def transform(msg: Any) = msgTransformer(msg)
|
||||||
def routes = routing
|
def routes = routing
|
||||||
|
|
@ -43,7 +43,7 @@ object Patterns {
|
||||||
|
|
||||||
/** Creates a Dispatcher given a routing
|
/** Creates a Dispatcher given a routing
|
||||||
*/
|
*/
|
||||||
def dispatcherActor(routing: PF[Any, ActorRef]): ActorRef = newActor(() => new Actor with Dispatcher {
|
def dispatcherActor(routing: PF[Any, ActorRef]): ActorRef = actorOf(new Actor with Dispatcher {
|
||||||
start
|
start
|
||||||
def routes = routing
|
def routes = routing
|
||||||
})
|
})
|
||||||
|
|
|
||||||
|
|
@ -27,7 +27,7 @@ import se.scalablesolutions.akka.actor.Actor
|
||||||
import se.scalablesolutions.akka.dispatch.CompletableFuture
|
import se.scalablesolutions.akka.dispatch.CompletableFuture
|
||||||
|
|
||||||
def thread(body: => Unit) = {
|
def thread(body: => Unit) = {
|
||||||
val thread = newActor(() => new IsolatedEventBasedThread(body)).start
|
val thread = actorOf(new IsolatedEventBasedThread(body)).start
|
||||||
thread ! Start
|
thread ! Start
|
||||||
thread
|
thread
|
||||||
}
|
}
|
||||||
|
|
@ -98,7 +98,7 @@ import se.scalablesolutions.akka.dispatch.CompletableFuture
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
private[this] val in = newActor(() => new In(this))
|
private[this] val in = actorOf(new In(this))
|
||||||
|
|
||||||
def <<(ref: DataFlowVariable[T]) = in ! Set(ref())
|
def <<(ref: DataFlowVariable[T]) = in ! Set(ref())
|
||||||
|
|
||||||
|
|
@ -108,7 +108,7 @@ import se.scalablesolutions.akka.dispatch.CompletableFuture
|
||||||
val ref = value.get
|
val ref = value.get
|
||||||
if (ref.isDefined) ref.get
|
if (ref.isDefined) ref.get
|
||||||
else {
|
else {
|
||||||
val out = newActor(() => new Out(this))
|
val out = actorOf(new Out(this))
|
||||||
blockedReaders.offer(out)
|
blockedReaders.offer(out)
|
||||||
val result = out !! Get
|
val result = out !! Get
|
||||||
out ! Exit
|
out ! Exit
|
||||||
|
|
|
||||||
|
|
@ -48,7 +48,7 @@ class ActorFireForgetRequestReplySpec extends JUnitSuite {
|
||||||
state.finished.reset
|
state.finished.reset
|
||||||
val replyActor = newActor[ReplyActor]
|
val replyActor = newActor[ReplyActor]
|
||||||
replyActor.start
|
replyActor.start
|
||||||
val senderActor = newActor(() => new SenderActor(replyActor))
|
val senderActor = actorOf(new SenderActor(replyActor))
|
||||||
senderActor.start
|
senderActor.start
|
||||||
senderActor ! "Init"
|
senderActor ! "Init"
|
||||||
try { state.finished.await(1L, TimeUnit.SECONDS) }
|
try { state.finished.await(1L, TimeUnit.SECONDS) }
|
||||||
|
|
@ -61,7 +61,7 @@ class ActorFireForgetRequestReplySpec extends JUnitSuite {
|
||||||
state.finished.reset
|
state.finished.reset
|
||||||
val replyActor = newActor[ReplyActor]
|
val replyActor = newActor[ReplyActor]
|
||||||
replyActor.start
|
replyActor.start
|
||||||
val senderActor = newActor(() => new SenderActor(replyActor))
|
val senderActor = actorOf(new SenderActor(replyActor))
|
||||||
senderActor.start
|
senderActor.start
|
||||||
senderActor ! "InitImplicit"
|
senderActor ! "InitImplicit"
|
||||||
try { state.finished.await(1L, TimeUnit.SECONDS) }
|
try { state.finished.await(1L, TimeUnit.SECONDS) }
|
||||||
|
|
|
||||||
|
|
@ -87,7 +87,7 @@ class ActorPatternsTest extends junit.framework.TestCase with Suite with MustMat
|
||||||
@Test def testListener = {
|
@Test def testListener = {
|
||||||
val latch = new CountDownLatch(2)
|
val latch = new CountDownLatch(2)
|
||||||
val num = new AtomicInteger(0)
|
val num = new AtomicInteger(0)
|
||||||
val i = newActor(() => new Actor with Listeners {
|
val i = actorOf(new Actor with Listeners {
|
||||||
def receive = listenerManagement orElse {
|
def receive = listenerManagement orElse {
|
||||||
case "foo" => gossip("bar")
|
case "foo" => gossip("bar")
|
||||||
}
|
}
|
||||||
|
|
|
||||||
|
|
@ -39,8 +39,8 @@ class ExecutorBasedEventDrivenDispatcherActorsSpec extends JUnitSuite with MustM
|
||||||
@Test def slowActorShouldntBlockFastActor = {
|
@Test def slowActorShouldntBlockFastActor = {
|
||||||
val sFinished = new CountDownLatch(50)
|
val sFinished = new CountDownLatch(50)
|
||||||
val fFinished = new CountDownLatch(10)
|
val fFinished = new CountDownLatch(10)
|
||||||
val s = newActor(() => new SlowActor(sFinished)).start
|
val s = actorOf(new SlowActor(sFinished)).start
|
||||||
val f = newActor(() => new FastActor(fFinished)).start
|
val f = actorOf(new FastActor(fFinished)).start
|
||||||
|
|
||||||
// send a lot of stuff to s
|
// send a lot of stuff to s
|
||||||
for (i <- 1 to 50) {
|
for (i <- 1 to 50) {
|
||||||
|
|
|
||||||
|
|
@ -57,8 +57,8 @@ class ExecutorBasedEventDrivenWorkStealingDispatcherSpec extends JUnitSuite with
|
||||||
@Test def fastActorShouldStealWorkFromSlowActor = {
|
@Test def fastActorShouldStealWorkFromSlowActor = {
|
||||||
val finishedCounter = new CountDownLatch(110)
|
val finishedCounter = new CountDownLatch(110)
|
||||||
|
|
||||||
val slow = newActor(() => new DelayableActor("slow", 50, finishedCounter)).start
|
val slow = actorOf(new DelayableActor("slow", 50, finishedCounter)).start
|
||||||
val fast = newActor(() => new DelayableActor("fast", 10, finishedCounter)).start
|
val fast = actorOf(new DelayableActor("fast", 10, finishedCounter)).start
|
||||||
|
|
||||||
for (i <- 1 to 100) {
|
for (i <- 1 to 100) {
|
||||||
// send most work to slow actor
|
// send most work to slow actor
|
||||||
|
|
|
||||||
|
|
@ -110,7 +110,7 @@ class InMemFailerActor extends Actor {
|
||||||
class InMemoryActorSpec extends JUnitSuite {
|
class InMemoryActorSpec extends JUnitSuite {
|
||||||
@Test
|
@Test
|
||||||
def shouldOneWayMapShouldNotRollbackStateForStatefulServerInCaseOfSuccess = {
|
def shouldOneWayMapShouldNotRollbackStateForStatefulServerInCaseOfSuccess = {
|
||||||
val stateful = newActor(() => new InMemStatefulActor(2))
|
val stateful = actorOf(new InMemStatefulActor(2))
|
||||||
stateful.start
|
stateful.start
|
||||||
stateful ! SetMapStateOneWay("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "init") // set init state
|
stateful ! SetMapStateOneWay("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "init") // set init state
|
||||||
stateful ! SuccessOneWay("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired
|
stateful ! SuccessOneWay("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired
|
||||||
|
|
@ -130,7 +130,7 @@ class InMemoryActorSpec extends JUnitSuite {
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
def shouldOneWayMapShouldRollbackStateForStatefulServerInCaseOfFailure = {
|
def shouldOneWayMapShouldRollbackStateForStatefulServerInCaseOfFailure = {
|
||||||
val stateful = newActor(() => new InMemStatefulActor(2))
|
val stateful = actorOf(new InMemStatefulActor(2))
|
||||||
stateful.start
|
stateful.start
|
||||||
val failer = newActor[InMemFailerActor]
|
val failer = newActor[InMemFailerActor]
|
||||||
failer.start
|
failer.start
|
||||||
|
|
@ -157,7 +157,7 @@ class InMemoryActorSpec extends JUnitSuite {
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
def shouldOneWayVectorShouldNotRollbackStateForStatefulServerInCaseOfSuccess = {
|
def shouldOneWayVectorShouldNotRollbackStateForStatefulServerInCaseOfSuccess = {
|
||||||
val stateful = newActor(() => new InMemStatefulActor(2))
|
val stateful = actorOf(new InMemStatefulActor(2))
|
||||||
stateful.start
|
stateful.start
|
||||||
stateful ! SetVectorStateOneWay("init") // set init state
|
stateful ! SetVectorStateOneWay("init") // set init state
|
||||||
stateful ! SuccessOneWay("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired
|
stateful ! SuccessOneWay("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired
|
||||||
|
|
@ -177,7 +177,7 @@ class InMemoryActorSpec extends JUnitSuite {
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
def shouldOneWayVectorShouldRollbackStateForStatefulServerInCaseOfFailure = {
|
def shouldOneWayVectorShouldRollbackStateForStatefulServerInCaseOfFailure = {
|
||||||
val stateful = newActor(() => new InMemStatefulActor(2))
|
val stateful = actorOf(new InMemStatefulActor(2))
|
||||||
stateful.start
|
stateful.start
|
||||||
stateful ! SetVectorStateOneWay("init") // set init state
|
stateful ! SetVectorStateOneWay("init") // set init state
|
||||||
Thread.sleep(1000)
|
Thread.sleep(1000)
|
||||||
|
|
@ -205,7 +205,7 @@ class InMemoryActorSpec extends JUnitSuite {
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
def shouldOneWayRefShouldNotRollbackStateForStatefulServerInCaseOfSuccess = {
|
def shouldOneWayRefShouldNotRollbackStateForStatefulServerInCaseOfSuccess = {
|
||||||
val stateful = newActor(() => new InMemStatefulActor(2))
|
val stateful = actorOf(new InMemStatefulActor(2))
|
||||||
stateful.start
|
stateful.start
|
||||||
stateful ! SetRefStateOneWay("init") // set init state
|
stateful ! SetRefStateOneWay("init") // set init state
|
||||||
stateful ! SuccessOneWay("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired
|
stateful ! SuccessOneWay("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired
|
||||||
|
|
@ -225,7 +225,7 @@ class InMemoryActorSpec extends JUnitSuite {
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
def shouldOneWayRefShouldRollbackStateForStatefulServerInCaseOfFailure = {
|
def shouldOneWayRefShouldRollbackStateForStatefulServerInCaseOfFailure = {
|
||||||
val stateful = newActor(() => new InMemStatefulActor(2))
|
val stateful = actorOf(new InMemStatefulActor(2))
|
||||||
stateful.start
|
stateful.start
|
||||||
stateful ! SetRefStateOneWay("init") // set init state
|
stateful ! SetRefStateOneWay("init") // set init state
|
||||||
Thread.sleep(1000)
|
Thread.sleep(1000)
|
||||||
|
|
|
||||||
|
|
@ -26,7 +26,7 @@ class ReactorBasedThreadPoolEventDrivenDispatcherActorSpec extends JUnitSuite {
|
||||||
|
|
||||||
@Test def shouldSendOneWay = {
|
@Test def shouldSendOneWay = {
|
||||||
val oneWay = new CountDownLatch(1)
|
val oneWay = new CountDownLatch(1)
|
||||||
val actor = newActor(() => new Actor {
|
val actor = actorOf(new Actor {
|
||||||
dispatcher = Dispatchers.newReactorBasedThreadPoolEventDrivenDispatcher(uuid)
|
dispatcher = Dispatchers.newReactorBasedThreadPoolEventDrivenDispatcher(uuid)
|
||||||
def receive = {
|
def receive = {
|
||||||
case "OneWay" => oneWay.countDown
|
case "OneWay" => oneWay.countDown
|
||||||
|
|
|
||||||
|
|
@ -27,7 +27,7 @@ class ThreadBasedActorSpec extends JUnitSuite {
|
||||||
|
|
||||||
@Test def shouldSendOneWay = {
|
@Test def shouldSendOneWay = {
|
||||||
var oneWay = new CountDownLatch(1)
|
var oneWay = new CountDownLatch(1)
|
||||||
val actor = newActor(() => new Actor {
|
val actor = actorOf(new Actor {
|
||||||
dispatcher = Dispatchers.newThreadBasedDispatcher(this)
|
dispatcher = Dispatchers.newThreadBasedDispatcher(this)
|
||||||
def receive = {
|
def receive = {
|
||||||
case "OneWay" => oneWay.countDown
|
case "OneWay" => oneWay.countDown
|
||||||
|
|
|
||||||
|
|
@ -14,9 +14,9 @@ import Actor._
|
||||||
|
|
||||||
class ThreadBasedDispatcherSpec extends JUnitSuite {
|
class ThreadBasedDispatcherSpec extends JUnitSuite {
|
||||||
private var threadingIssueDetected: AtomicBoolean = null
|
private var threadingIssueDetected: AtomicBoolean = null
|
||||||
val key1 = newActor(() => new Actor { def receive = { case _ => {}} })
|
val key1 = actorOf(new Actor { def receive = { case _ => {}} })
|
||||||
val key2 = newActor(() => new Actor { def receive = { case _ => {}} })
|
val key2 = actorOf(new Actor { def receive = { case _ => {}} })
|
||||||
val key3 = newActor(() => new Actor { def receive = { case _ => {}} })
|
val key3 = actorOf(new Actor { def receive = { case _ => {}} })
|
||||||
|
|
||||||
class TestMessageHandle(handleLatch: CountDownLatch) extends MessageInvoker {
|
class TestMessageHandle(handleLatch: CountDownLatch) extends MessageInvoker {
|
||||||
val guardLock: Lock = new ReentrantLock
|
val guardLock: Lock = new ReentrantLock
|
||||||
|
|
|
||||||
|
|
@ -35,8 +35,8 @@ class Boot {
|
||||||
// Routing example
|
// Routing example
|
||||||
|
|
||||||
val producer = newActor[Producer1]
|
val producer = newActor[Producer1]
|
||||||
val mediator = newActor(() => new Transformer(producer))
|
val mediator = actorOf(new Transformer(producer))
|
||||||
val consumer = newActor(() => new Consumer3(mediator))
|
val consumer = actorOf(new Consumer3(mediator))
|
||||||
|
|
||||||
producer.start
|
producer.start
|
||||||
mediator.start
|
mediator.start
|
||||||
|
|
@ -55,9 +55,9 @@ class Boot {
|
||||||
//val cometdPublisher = new Publisher("cometd-publisher", cometdUri).start
|
//val cometdPublisher = new Publisher("cometd-publisher", cometdUri).start
|
||||||
|
|
||||||
val jmsUri = "jms:topic:test"
|
val jmsUri = "jms:topic:test"
|
||||||
val jmsSubscriber1 = newActor(() => new Subscriber("jms-subscriber-1", jmsUri)).start
|
val jmsSubscriber1 = actorOf(new Subscriber("jms-subscriber-1", jmsUri)).start
|
||||||
val jmsSubscriber2 = newActor(() => new Subscriber("jms-subscriber-2", jmsUri)).start
|
val jmsSubscriber2 = actorOf(new Subscriber("jms-subscriber-2", jmsUri)).start
|
||||||
val jmsPublisher = newActor(() => new Publisher("jms-publisher", jmsUri)).start
|
val jmsPublisher = actorOf(new Publisher("jms-publisher", jmsUri)).start
|
||||||
|
|
||||||
//val cometdPublisherBridge = new PublisherBridge("jetty:http://0.0.0.0:8877/camel/pub/cometd", cometdPublisher).start
|
//val cometdPublisherBridge = new PublisherBridge("jetty:http://0.0.0.0:8877/camel/pub/cometd", cometdPublisher).start
|
||||||
val jmsPublisherBridge = new PublisherBridge("jetty:http://0.0.0.0:8877/camel/pub/jms", jmsPublisher).start
|
val jmsPublisherBridge = new PublisherBridge("jetty:http://0.0.0.0:8877/camel/pub/jms", jmsPublisher).start
|
||||||
|
|
|
||||||
|
|
@ -130,7 +130,7 @@ trait SessionManagement { this: Actor =>
|
||||||
protected def sessionManagement: PartialFunction[Any, Unit] = {
|
protected def sessionManagement: PartialFunction[Any, Unit] = {
|
||||||
case Login(username) =>
|
case Login(username) =>
|
||||||
log.info("User [%s] has logged in", username)
|
log.info("User [%s] has logged in", username)
|
||||||
val session = newActor(() => new Session(username, storage))
|
val session = actorOf(new Session(username, storage))
|
||||||
session.start
|
session.start
|
||||||
sessions += (username -> session)
|
sessions += (username -> session)
|
||||||
|
|
||||||
|
|
|
||||||
|
|
@ -52,7 +52,7 @@ import se.scalablesolutions.akka.actor.Actor._
|
||||||
object Pub {
|
object Pub {
|
||||||
println("starting publishing service ..")
|
println("starting publishing service ..")
|
||||||
val r = new RedisClient("localhost", 6379)
|
val r = new RedisClient("localhost", 6379)
|
||||||
val p = newActor(() => new Publisher(r))
|
val p = actorOf(new Publisher(r))
|
||||||
p.start
|
p.start
|
||||||
|
|
||||||
def publish(channel: String, message: String) = {
|
def publish(channel: String, message: String) = {
|
||||||
|
|
@ -63,7 +63,7 @@ object Pub {
|
||||||
object Sub {
|
object Sub {
|
||||||
println("starting subscription service ..")
|
println("starting subscription service ..")
|
||||||
val r = new RedisClient("localhost", 6379)
|
val r = new RedisClient("localhost", 6379)
|
||||||
val s = newActor(() => new Subscriber(r))
|
val s = actorOf(new Subscriber(r))
|
||||||
s.start
|
s.start
|
||||||
s ! Register(callback)
|
s ! Register(callback)
|
||||||
|
|
||||||
|
|
|
||||||
Loading…
Add table
Add a link
Reference in a new issue