Fixed bug in fault handling of TEMPORARY Actors + ported all Active Object Java tests to Scala (using Java POJOs)

This commit is contained in:
Jonas Bonér 2010-06-29 15:19:09 +02:00
parent c6b7ba6a01
commit e8581123da
17 changed files with 225 additions and 166 deletions

View file

@ -13,6 +13,7 @@ public class InMemStateful {
private Ref<String> refState; private Ref<String> refState;
private boolean isInitialized = false; private boolean isInitialized = false;
@inittransactionalstate
public void init() { public void init() {
if (!isInitialized) { if (!isInitialized) {
mapState = new TransactionalMap(); mapState = new TransactionalMap();

View file

@ -11,6 +11,7 @@ public class InMemStatefulNested {
private Ref<String> refState; private Ref<String> refState;
private boolean isInitialized = false; private boolean isInitialized = false;
@inittransactionalstate
public void init() { public void init() {
if (!isInitialized) { if (!isInitialized) {
mapState = new TransactionalMap(); mapState = new TransactionalMap();

View file

@ -1311,53 +1311,62 @@ sealed class LocalActorRef private[akka](
} }
protected[akka] def restart(reason: Throwable): Unit = { protected[akka] def restart(reason: Throwable): Unit = {
//_isBeingRestarted = true
Actor.log.info("Restarting actor [%s] configured as PERMANENT.", id)
restartLinkedActors(reason)
val failedActor = actorInstance.get val failedActor = actorInstance.get
failedActor.synchronized { failedActor.synchronized {
Actor.log.debug("Restarting linked actors for actor [%s].", id) lifeCycle.get match {
Actor.log.debug("Invoking 'preRestart' for failed actor instance [%s].", id) case LifeCycle(scope, _) => {
failedActor.preRestart(reason) scope match {
nullOutActorRefReferencesFor(failedActor) case Permanent =>
val freshActor = newActor Actor.log.info("Restarting actor [%s] configured as PERMANENT.", id)
freshActor.synchronized { restartLinkedActors(reason)
freshActor.init Actor.log.debug("Restarting linked actors for actor [%s].", id)
freshActor.initTransactionalState Actor.log.debug("Invoking 'preRestart' for failed actor instance [%s].", id)
actorInstance.set(freshActor) failedActor.preRestart(reason)
Actor.log.debug("Invoking 'postRestart' for new actor instance [%s].", id) nullOutActorRefReferencesFor(failedActor)
freshActor.postRestart(reason) val freshActor = newActor
freshActor.synchronized {
freshActor.init
freshActor.initTransactionalState
actorInstance.set(freshActor)
Actor.log.debug("Invoking 'postRestart' for new actor instance [%s].", id)
freshActor.postRestart(reason)
}
_isBeingRestarted = false
case Temporary => shutDownTemporaryActor(this)
}
}
} }
_isBeingRestarted = false
} }
} }
protected[akka] def restartLinkedActors(reason: Throwable) = guard.withGuard { protected[akka] def restartLinkedActors(reason: Throwable) = guard.withGuard {
linkedActorsAsList.foreach { actorRef => linkedActorsAsList.foreach { actorRef =>
if (actorRef.lifeCycle.isEmpty) actorRef.lifeCycle = Some(LifeCycle(Permanent)) if (actorRef.lifeCycle.isEmpty) actorRef.lifeCycle = Some(LifeCycle(Permanent))
actorRef.lifeCycle.get match { actorRef.lifeCycle.get match {
case LifeCycle(scope, _) => { case LifeCycle(scope, _) => {
scope match { scope match {
case Permanent => case Permanent => actorRef.restart(reason)
actorRef.restart(reason) case Temporary => shutDownTemporaryActor(actorRef)
case Temporary =>
Actor.log.info("Actor [%s] configured as TEMPORARY and will not be restarted.", actorRef.id)
actorRef.stop
linkedActors.remove(actorRef.uuid) // remove the temporary actor
// if last temporary actor is gone, then unlink me from supervisor
if (linkedActors.isEmpty) {
Actor.log.info(
"All linked actors have died permanently (they were all configured as TEMPORARY)" +
"\n\tshutting down and unlinking supervisor actor as well [%s].",
actorRef.id)
_supervisor.foreach(_ ! UnlinkAndStop(this))
}
} }
} }
} }
} }
} }
private def shutDownTemporaryActor(temporaryActor: ActorRef) = {
Actor.log.info("Actor [%s] configured as TEMPORARY and will not be restarted.", temporaryActor.id)
temporaryActor.stop
linkedActors.remove(temporaryActor.uuid) // remove the temporary actor
// if last temporary actor is gone, then unlink me from supervisor
if (linkedActors.isEmpty) {
Actor.log.info(
"All linked actors have died permanently (they were all configured as TEMPORARY)" +
"\n\tshutting down and unlinking supervisor actor as well [%s].",
temporaryActor.id)
_supervisor.foreach(_ ! UnlinkAndStop(this))
}
}
protected[akka] def registerSupervisorAsRemoteActor: Option[String] = guard.withGuard { protected[akka] def registerSupervisorAsRemoteActor: Option[String] = guard.withGuard {
if (_supervisor.isDefined) { if (_supervisor.isDefined) {
RemoteClient.clientFor(remoteAddress.get).registerSupervisorForActor(this) RemoteClient.clientFor(remoteAddress.get).registerSupervisorForActor(this)

View file

@ -425,14 +425,14 @@ class RemoteServerHandler(
log.error(e.getCause, "Could not invoke remote active object [%s :: %s]", request.getMethod, request.getTarget) log.error(e.getCause, "Could not invoke remote active object [%s :: %s]", request.getMethod, request.getTarget)
val replyBuilder = RemoteReplyProtocol.newBuilder val replyBuilder = RemoteReplyProtocol.newBuilder
.setId(request.getId) .setId(request.getId)
.setException(ExceptionProtocol.newBuilder.setClassname(e.getClass.getName).setMessage(e.getMessage).build) .setException(ExceptionProtocol.newBuilder.setClassname(e.getCause.getClass.getName).setMessage(e.getCause.getMessage).build)
.setIsSuccessful(false) .setIsSuccessful(false)
.setIsActor(false) .setIsActor(false)
if (request.hasSupervisorUuid) replyBuilder.setSupervisorUuid(request.getSupervisorUuid) if (request.hasSupervisorUuid) replyBuilder.setSupervisorUuid(request.getSupervisorUuid)
val replyMessage = replyBuilder.build val replyMessage = replyBuilder.build
channel.write(replyMessage) channel.write(replyMessage)
case e: Throwable => case e: Throwable =>
log.error(e.getCause, "Could not invoke remote active object [%s :: %s]", request.getMethod, request.getTarget) log.error(e, "Could not invoke remote active object [%s :: %s]", request.getMethod, request.getTarget)
val replyBuilder = RemoteReplyProtocol.newBuilder val replyBuilder = RemoteReplyProtocol.newBuilder
.setId(request.getId) .setId(request.getId)
.setException(ExceptionProtocol.newBuilder.setClassname(e.getClass.getName).setMessage(e.getMessage).build) .setException(ExceptionProtocol.newBuilder.setClassname(e.getClass.getName).setMessage(e.getMessage).build)

View file

@ -87,35 +87,7 @@ object Transaction {
// --- public methods --------- // --- public methods ---------
def begin = synchronized {
jta.foreach { txContainer =>
txContainer.begin
txContainer.registerSynchronization(new StmSynchronization(txContainer, this))
}
}
def commit = synchronized {
log.trace("Committing transaction %s", toString)
persistentStateMap.valuesIterator.foreach(_.commit)
status = TransactionStatus.Completed
jta.foreach(_.commit)
}
def abort = synchronized {
log.trace("Aborting transaction %s", toString)
jta.foreach(_.rollback)
persistentStateMap.valuesIterator.foreach(_.abort)
persistentStateMap.clear
}
def isNew = synchronized { status == TransactionStatus.New }
def isActive = synchronized { status == TransactionStatus.Active }
def isCompleted = synchronized { status == TransactionStatus.Completed }
def isAborted = synchronized { status == TransactionStatus.Aborted }
// --- internal methods --------- // --- internal methods ---------
private def isJtaTxActive(status: Int) = status == Status.STATUS_ACTIVE private def isJtaTxActive(status: Int) = status == Status.STATUS_ACTIVE

View file

@ -1,6 +1,6 @@
package se.scalablesolutions.akka.actor; package se.scalablesolutions.akka.actor;
public class InMemFailer implements java.io.Serializable { public class ActiveObjectFailer implements java.io.Serializable {
public int fail() { public int fail() {
throw new RuntimeException("expected"); throw new RuntimeException("expected");
} }

View file

@ -17,7 +17,7 @@ public class Foo extends se.scalablesolutions.akka.serialization.Serializable.Ja
} }
public String longRunning() { public String longRunning() {
try { try {
Thread.sleep(10000); Thread.sleep(1000);
} catch (InterruptedException e) { } catch (InterruptedException e) {
} }
return "test"; return "test";

View file

@ -5,12 +5,13 @@ import se.scalablesolutions.akka.actor.annotation.inittransactionalstate;
import se.scalablesolutions.akka.stm.*; import se.scalablesolutions.akka.stm.*;
@transactionrequired @transactionrequired
public class InMemStatefulNested { public class NestedTransactionalActiveObject {
private TransactionalMap<String, String> mapState; private TransactionalMap<String, String> mapState;
private TransactionalVector<String> vectorState; private TransactionalVector<String> vectorState;
private Ref<String> refState; private Ref<String> refState;
private boolean isInitialized = false; private boolean isInitialized = false;
@inittransactionalstate
public void init() { public void init() {
if (!isInitialized) { if (!isInitialized) {
mapState = new TransactionalMap(); mapState = new TransactionalMap();
@ -57,7 +58,7 @@ public class InMemStatefulNested {
} }
public String failure(String key, String msg, InMemFailer failer) { public String failure(String key, String msg, ActiveObjectFailer failer) {
mapState.put(key, msg); mapState.put(key, msg);
vectorState.add(msg); vectorState.add(msg);
refState.swap(msg); refState.swap(msg);
@ -66,7 +67,7 @@ public class InMemStatefulNested {
} }
public void thisMethodHangs(String key, String msg, InMemFailer failer) { public void thisMethodHangs(String key, String msg, ActiveObjectFailer failer) {
setMapState(key, msg); setMapState(key, msg);
} }

View file

@ -7,12 +7,13 @@ import se.scalablesolutions.akka.actor.annotation.inittransactionalstate;
import se.scalablesolutions.akka.stm.*; import se.scalablesolutions.akka.stm.*;
@transactionrequired @transactionrequired
public class InMemStateful { public class TransactionalActiveObject {
private TransactionalMap<String, String> mapState; private TransactionalMap<String, String> mapState;
private TransactionalVector<String> vectorState; private TransactionalVector<String> vectorState;
private Ref<String> refState; private Ref<String> refState;
private boolean isInitialized = false; private boolean isInitialized = false;
@inittransactionalstate
public void init() { public void init() {
if (!isInitialized) { if (!isInitialized) {
mapState = new TransactionalMap(); mapState = new TransactionalMap();
@ -52,14 +53,14 @@ public class InMemStateful {
refState.swap(msg); refState.swap(msg);
} }
public void success(String key, String msg, InMemStatefulNested nested) { public void success(String key, String msg, NestedTransactionalActiveObject nested) {
mapState.put(key, msg); mapState.put(key, msg);
vectorState.add(msg); vectorState.add(msg);
refState.swap(msg); refState.swap(msg);
nested.success(key, msg); nested.success(key, msg);
} }
public String failure(String key, String msg, InMemFailer failer) { public String failure(String key, String msg, ActiveObjectFailer failer) {
mapState.put(key, msg); mapState.put(key, msg);
vectorState.add(msg); vectorState.add(msg);
refState.swap(msg); refState.swap(msg);
@ -67,7 +68,7 @@ public class InMemStateful {
return msg; return msg;
} }
public String failure(String key, String msg, InMemStatefulNested nested, InMemFailer failer) { public String failure(String key, String msg, NestedTransactionalActiveObject nested, ActiveObjectFailer failer) {
mapState.put(key, msg); mapState.put(key, msg);
vectorState.add(msg); vectorState.add(msg);
refState.swap(msg); refState.swap(msg);
@ -75,7 +76,7 @@ public class InMemStateful {
return msg; return msg;
} }
public void thisMethodHangs(String key, String msg, InMemFailer failer) { public void thisMethodHangs(String key, String msg, ActiveObjectFailer failer) {
setMapState(key, msg); setMapState(key, msg);
} }

View file

@ -18,7 +18,7 @@ import se.scalablesolutions.akka.config.JavaConfig._
import se.scalablesolutions.akka.actor._ import se.scalablesolutions.akka.actor._
@RunWith(classOf[JUnitRunner]) @RunWith(classOf[JUnitRunner])
class InMemoryNestedStateSpec extends class NestedTransactionalActiveObjectSpec extends
Spec with Spec with
ShouldMatchers with ShouldMatchers with
BeforeAndAfterAll { BeforeAndAfterAll {
@ -31,13 +31,13 @@ class InMemoryNestedStateSpec extends
conf.configure( conf.configure(
new RestartStrategy(new AllForOne, 3, 5000, List(classOf[Exception]).toArray), new RestartStrategy(new AllForOne, 3, 5000, List(classOf[Exception]).toArray),
List( List(
new Component(classOf[InMemStateful], new Component(classOf[TransactionalActiveObject],
new LifeCycle(new Permanent), new LifeCycle(new Permanent),
10000), 10000),
new Component(classOf[InMemStatefulNested], new Component(classOf[NestedTransactionalActiveObject],
new LifeCycle(new Permanent), new LifeCycle(new Permanent),
10000), 10000),
new Component(classOf[InMemFailer], new Component(classOf[ActiveObjectFailer],
new LifeCycle(new Permanent), new LifeCycle(new Permanent),
10000) 10000)
).toArray).supervise ).toArray).supervise
@ -50,11 +50,11 @@ class InMemoryNestedStateSpec extends
describe("Transactional nested in-memory Active Object") { describe("Transactional nested in-memory Active Object") {
it("map should not rollback state for stateful server in case of success") { it("map should not rollback state for stateful server in case of success") {
val stateful = conf.getInstance(classOf[InMemStateful]) val stateful = conf.getInstance(classOf[TransactionalActiveObject])
stateful.init stateful.init
stateful.setMapState("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "init") // set init state stateful.setMapState("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "init") // set init state
Thread.sleep(100) Thread.sleep(100)
val nested = conf.getInstance(classOf[InMemStatefulNested]) val nested = conf.getInstance(classOf[NestedTransactionalActiveObject])
nested.init nested.init
nested.setMapState("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "init") // set init state nested.setMapState("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "init") // set init state
Thread.sleep(100) Thread.sleep(100)
@ -65,15 +65,15 @@ class InMemoryNestedStateSpec extends
} }
it("map should rollback state for stateful server in case of failure") { it("map should rollback state for stateful server in case of failure") {
val stateful = conf.getInstance(classOf[InMemStateful]) val stateful = conf.getInstance(classOf[TransactionalActiveObject])
stateful.init stateful.init
stateful.setMapState("testShouldRollbackStateForStatefulServerInCaseOfFailure", "init") // set init state stateful.setMapState("testShouldRollbackStateForStatefulServerInCaseOfFailure", "init") // set init state
Thread.sleep(100) Thread.sleep(100)
val nested = conf.getInstance(classOf[InMemStatefulNested]) val nested = conf.getInstance(classOf[NestedTransactionalActiveObject])
nested.init nested.init
nested.setMapState("testShouldRollbackStateForStatefulServerInCaseOfFailure", "init") // set init state nested.setMapState("testShouldRollbackStateForStatefulServerInCaseOfFailure", "init") // set init state
Thread.sleep(100) Thread.sleep(100)
val failer = conf.getInstance(classOf[InMemFailer]) val failer = conf.getInstance(classOf[ActiveObjectFailer])
try { try {
stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", nested, failer) stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", nested, failer)
Thread.sleep(100) Thread.sleep(100)
@ -85,11 +85,11 @@ class InMemoryNestedStateSpec extends
} }
it("vector should not rollback state for stateful server in case of success") { it("vector should not rollback state for stateful server in case of success") {
val stateful = conf.getInstance(classOf[InMemStateful]) val stateful = conf.getInstance(classOf[TransactionalActiveObject])
stateful.init stateful.init
stateful.setVectorState("init") // set init state stateful.setVectorState("init") // set init state
Thread.sleep(100) Thread.sleep(100)
val nested = conf.getInstance(classOf[InMemStatefulNested]) val nested = conf.getInstance(classOf[NestedTransactionalActiveObject])
nested.init nested.init
Thread.sleep(100) Thread.sleep(100)
nested.setVectorState("init") // set init state nested.setVectorState("init") // set init state
@ -102,15 +102,15 @@ class InMemoryNestedStateSpec extends
} }
it("vector should rollback state for stateful server in case of failure") { it("vector should rollback state for stateful server in case of failure") {
val stateful = conf.getInstance(classOf[InMemStateful]) val stateful = conf.getInstance(classOf[TransactionalActiveObject])
stateful.init stateful.init
stateful.setVectorState("init") // set init state stateful.setVectorState("init") // set init state
Thread.sleep(100) Thread.sleep(100)
val nested = conf.getInstance(classOf[InMemStatefulNested]) val nested = conf.getInstance(classOf[NestedTransactionalActiveObject])
nested.init nested.init
nested.setVectorState("init") // set init state nested.setVectorState("init") // set init state
Thread.sleep(100) Thread.sleep(100)
val failer = conf.getInstance(classOf[InMemFailer]) val failer = conf.getInstance(classOf[ActiveObjectFailer])
try { try {
stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", nested, failer) stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", nested, failer)
Thread.sleep(100) Thread.sleep(100)
@ -122,9 +122,9 @@ class InMemoryNestedStateSpec extends
} }
it("ref should not rollback state for stateful server in case of success") { it("ref should not rollback state for stateful server in case of success") {
val stateful = conf.getInstance(classOf[InMemStateful]) val stateful = conf.getInstance(classOf[TransactionalActiveObject])
stateful.init stateful.init
val nested = conf.getInstance(classOf[InMemStatefulNested]) val nested = conf.getInstance(classOf[NestedTransactionalActiveObject])
nested.init nested.init
stateful.setRefState("init") // set init state stateful.setRefState("init") // set init state
Thread.sleep(100) Thread.sleep(100)
@ -138,15 +138,15 @@ class InMemoryNestedStateSpec extends
} }
it("ref should rollback state for stateful server in case of failure") { it("ref should rollback state for stateful server in case of failure") {
val stateful = conf.getInstance(classOf[InMemStateful]) val stateful = conf.getInstance(classOf[TransactionalActiveObject])
stateful.init stateful.init
val nested = conf.getInstance(classOf[InMemStatefulNested]) val nested = conf.getInstance(classOf[NestedTransactionalActiveObject])
nested.init nested.init
stateful.setRefState("init") // set init state stateful.setRefState("init") // set init state
Thread.sleep(100) Thread.sleep(100)
nested.setRefState("init") // set init state nested.setRefState("init") // set init state
Thread.sleep(100) Thread.sleep(100)
val failer = conf.getInstance(classOf[InMemFailer]) val failer = conf.getInstance(classOf[ActiveObjectFailer])
try { try {
stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", nested, failer) stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", nested, failer)
Thread.sleep(100) Thread.sleep(100)

View file

@ -7,6 +7,7 @@ package se.scalablesolutions.akka.actor
import java.util.concurrent.{LinkedBlockingQueue, TimeUnit, BlockingQueue} import java.util.concurrent.{LinkedBlockingQueue, TimeUnit, BlockingQueue}
import se.scalablesolutions.akka.serialization.BinaryString import se.scalablesolutions.akka.serialization.BinaryString
import se.scalablesolutions.akka.config.ScalaConfig._ import se.scalablesolutions.akka.config.ScalaConfig._
import se.scalablesolutions.akka.config.Config
import se.scalablesolutions.akka.remote.{RemoteNode, RemoteServer} import se.scalablesolutions.akka.remote.{RemoteNode, RemoteServer}
import se.scalablesolutions.akka.OneWay import se.scalablesolutions.akka.OneWay
import se.scalablesolutions.akka.dispatch.Dispatchers import se.scalablesolutions.akka.dispatch.Dispatchers
@ -71,19 +72,21 @@ object Log {
} }
} }
object RemoteSupervisorSpec {
val HOSTNAME = "localhost"
val PORT = 9988
var server: RemoteServer = null
}
/** /**
* @author <a href="http://jonasboner.com">Jonas Bon&#233;r</a> * @author <a href="http://jonasboner.com">Jonas Bon&#233;r</a>
*/ */
class RemoteSupervisorSpec extends JUnitSuite { class RemoteSupervisorSpec extends JUnitSuite {
import RemoteSupervisorSpec._
Config.config
se.scalablesolutions.akka.config.Config.config server = new RemoteServer()
server.start(HOSTNAME, PORT)
new Thread(new Runnable() {
def run = {
RemoteNode.start(RemoteServer.HOSTNAME, 9988)
}
}).start
Thread.sleep(1000)
var pingpong1: ActorRef = _ var pingpong1: ActorRef = _
var pingpong2: ActorRef = _ var pingpong2: ActorRef = _
@ -460,7 +463,7 @@ class RemoteSupervisorSpec extends JUnitSuite {
// implementation of the Actors we want to use. // implementation of the Actors we want to use.
pingpong1 = actorOf[RemotePingPong1Actor] pingpong1 = actorOf[RemotePingPong1Actor]
pingpong1.makeRemote(RemoteServer.HOSTNAME, 9988) pingpong1.makeRemote(HOSTNAME, PORT)
pingpong1.start pingpong1.start
val factory = SupervisorFactory( val factory = SupervisorFactory(
@ -476,7 +479,7 @@ class RemoteSupervisorSpec extends JUnitSuite {
def getSingleActorOneForOneSupervisor: Supervisor = { def getSingleActorOneForOneSupervisor: Supervisor = {
pingpong1 = actorOf[RemotePingPong1Actor] pingpong1 = actorOf[RemotePingPong1Actor]
pingpong1.makeRemote(RemoteServer.HOSTNAME, 9988) pingpong1.makeRemote(HOSTNAME, PORT)
pingpong1.start pingpong1.start
val factory = SupervisorFactory( val factory = SupervisorFactory(
@ -491,13 +494,13 @@ class RemoteSupervisorSpec extends JUnitSuite {
def getMultipleActorsAllForOneConf: Supervisor = { def getMultipleActorsAllForOneConf: Supervisor = {
pingpong1 = actorOf[RemotePingPong1Actor] pingpong1 = actorOf[RemotePingPong1Actor]
pingpong1.makeRemote(RemoteServer.HOSTNAME, 9988) pingpong1.makeRemote(HOSTNAME, PORT)
pingpong1.start pingpong1.start
pingpong2 = actorOf[RemotePingPong2Actor] pingpong2 = actorOf[RemotePingPong2Actor]
pingpong2.makeRemote(RemoteServer.HOSTNAME, 9988) pingpong2.makeRemote(HOSTNAME, PORT)
pingpong2.start pingpong2.start
pingpong3 = actorOf[RemotePingPong3Actor] pingpong3 = actorOf[RemotePingPong3Actor]
pingpong3.makeRemote(RemoteServer.HOSTNAME, 9988) pingpong3.makeRemote(HOSTNAME, PORT)
pingpong3.start pingpong3.start
val factory = SupervisorFactory( val factory = SupervisorFactory(
@ -520,15 +523,15 @@ class RemoteSupervisorSpec extends JUnitSuite {
def getMultipleActorsOneForOneConf: Supervisor = { def getMultipleActorsOneForOneConf: Supervisor = {
pingpong1 = actorOf[RemotePingPong1Actor] pingpong1 = actorOf[RemotePingPong1Actor]
pingpong1.makeRemote(RemoteServer.HOSTNAME, 9988) pingpong1.makeRemote(HOSTNAME, PORT)
pingpong1 = actorOf[RemotePingPong1Actor] pingpong1 = actorOf[RemotePingPong1Actor]
pingpong1.makeRemote(RemoteServer.HOSTNAME, 9988) pingpong1.makeRemote(HOSTNAME, PORT)
pingpong1.start pingpong1.start
pingpong2 = actorOf[RemotePingPong2Actor] pingpong2 = actorOf[RemotePingPong2Actor]
pingpong2.makeRemote(RemoteServer.HOSTNAME, 9988) pingpong2.makeRemote(HOSTNAME, PORT)
pingpong2.start pingpong2.start
pingpong3 = actorOf[RemotePingPong3Actor] pingpong3 = actorOf[RemotePingPong3Actor]
pingpong3.makeRemote(RemoteServer.HOSTNAME, 9988) pingpong3.makeRemote(HOSTNAME, PORT)
pingpong3.start pingpong3.start
val factory = SupervisorFactory( val factory = SupervisorFactory(
@ -551,13 +554,13 @@ class RemoteSupervisorSpec extends JUnitSuite {
def getNestedSupervisorsAllForOneConf: Supervisor = { def getNestedSupervisorsAllForOneConf: Supervisor = {
pingpong1 = actorOf[RemotePingPong1Actor] pingpong1 = actorOf[RemotePingPong1Actor]
pingpong1.makeRemote(RemoteServer.HOSTNAME, 9988) pingpong1.makeRemote(HOSTNAME, PORT)
pingpong1.start pingpong1.start
pingpong2 = actorOf[RemotePingPong2Actor] pingpong2 = actorOf[RemotePingPong2Actor]
pingpong2.makeRemote(RemoteServer.HOSTNAME, 9988) pingpong2.makeRemote(HOSTNAME, PORT)
pingpong2.start pingpong2.start
pingpong3 = actorOf[RemotePingPong3Actor] pingpong3 = actorOf[RemotePingPong3Actor]
pingpong3.makeRemote(RemoteServer.HOSTNAME, 9988) pingpong3.makeRemote(HOSTNAME, PORT)
pingpong3.start pingpong3.start
val factory = SupervisorFactory( val factory = SupervisorFactory(

View file

@ -13,40 +13,44 @@ import org.junit.runner.RunWith
import se.scalablesolutions.akka.config.Config import se.scalablesolutions.akka.config.Config
import se.scalablesolutions.akka.config.ActiveObjectConfigurator import se.scalablesolutions.akka.config.ActiveObjectConfigurator
import se.scalablesolutions.akka.remote.RemoteNode import se.scalablesolutions.akka.remote.{RemoteNode, RemoteServer}
object RemoteTransactionalActiveObjectSpec {
val HOSTNAME = "localhost"
val PORT = 9988
var server: RemoteServer = null
}
@RunWith(classOf[JUnitRunner]) @RunWith(classOf[JUnitRunner])
class RemoteInMemoryStateSpec extends class RemoteTransactionalActiveObjectSpec extends
Spec with Spec with
ShouldMatchers with ShouldMatchers with
BeforeAndAfterAll { BeforeAndAfterAll {
import RemoteTransactionalActiveObjectSpec._
Config.config
server = new RemoteServer()
server.start(HOSTNAME, PORT)
private val conf = new ActiveObjectConfigurator private val conf = new ActiveObjectConfigurator
private var messageLog = "" private var messageLog = ""
override def beforeAll = {
Config.config
new Thread(new Runnable {
def run = RemoteNode.start
}).start
Thread.sleep(1000)
}
override def afterAll = conf.stop override def afterAll = conf.stop
describe("Remote transactional in-memory Active Object ") { describe("Remote transactional in-memory Active Object ") {
it("map should not rollback state for stateful server in case of success") { it("map should not rollback state for stateful server in case of success") {
val stateful = ActiveObject.newRemoteInstance(classOf[InMemStateful], 1000, "localhost", 9999) val stateful = ActiveObject.newRemoteInstance(classOf[TransactionalActiveObject], 1000, HOSTNAME, PORT)
stateful.setMapState("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "init") // set init state stateful.setMapState("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "init") // set init state
stateful.success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired stateful.success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired
stateful.getMapState("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess") should equal("new state") stateful.getMapState("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess") should equal("new state")
} }
it("map should rollback state for stateful server in case of failure") { it("map should rollback state for stateful server in case of failure") {
val stateful = ActiveObject.newRemoteInstance(classOf[InMemStateful], 1000, "localhost", 9999) val stateful = ActiveObject.newRemoteInstance(classOf[TransactionalActiveObject], 1000, HOSTNAME, PORT)
stateful.setMapState("testShouldRollbackStateForStatefulServerInCaseOfFailure", "init") // set init state stateful.setMapState("testShouldRollbackStateForStatefulServerInCaseOfFailure", "init") // set init state
val failer =ActiveObject.newRemoteInstance(classOf[InMemFailer], 1000, "localhost", 9999) //conf.getInstance(classOf[InMemFailer]) val failer =ActiveObject.newRemoteInstance(classOf[ActiveObjectFailer], 1000, HOSTNAME, PORT) //conf.getInstance(classOf[ActiveObjectFailer])
try { try {
stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method
fail("should have thrown an exception") fail("should have thrown an exception")
@ -55,16 +59,16 @@ class RemoteInMemoryStateSpec extends
} }
it("vector should not rollback state for stateful server in case of success") { it("vector should not rollback state for stateful server in case of success") {
val stateful = ActiveObject.newRemoteInstance(classOf[InMemStateful], 1000, "localhost", 9999) val stateful = ActiveObject.newRemoteInstance(classOf[TransactionalActiveObject], 1000, HOSTNAME, PORT)
stateful.setVectorState("init") // set init state stateful.setVectorState("init") // set init state
stateful.success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired stateful.success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired
stateful.getVectorState should equal("new state") stateful.getVectorState should equal("new state")
} }
it("vector should rollback state for stateful server in case of failure") { it("vector should rollback state for stateful server in case of failure") {
val stateful = ActiveObject.newRemoteInstance(classOf[InMemStateful], 1000, "localhost", 9999) val stateful = ActiveObject.newRemoteInstance(classOf[TransactionalActiveObject], 1000, HOSTNAME, PORT)
stateful.setVectorState("init") // set init state stateful.setVectorState("init") // set init state
val failer =ActiveObject.newRemoteInstance(classOf[InMemFailer], 1000, "localhost", 9999) //conf.getInstance(classOf[InMemFailer]) val failer =ActiveObject.newRemoteInstance(classOf[ActiveObjectFailer], 1000, HOSTNAME, PORT) //conf.getInstance(classOf[ActiveObjectFailer])
try { try {
stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method
fail("should have thrown an exception") fail("should have thrown an exception")
@ -73,16 +77,16 @@ class RemoteInMemoryStateSpec extends
} }
it("ref should not rollback state for stateful server in case of success") { it("ref should not rollback state for stateful server in case of success") {
val stateful = ActiveObject.newRemoteInstance(classOf[InMemStateful], 1000, "localhost", 9999) val stateful = ActiveObject.newRemoteInstance(classOf[TransactionalActiveObject], 1000, HOSTNAME, PORT)
stateful.setRefState("init") // set init state stateful.setRefState("init") // set init state
stateful.success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired stateful.success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired
stateful.getRefState should equal("new state") stateful.getRefState should equal("new state")
} }
it("ref should rollback state for stateful server in case of failure") { it("ref should rollback state for stateful server in case of failure") {
val stateful = ActiveObject.newRemoteInstance(classOf[InMemStateful], 1000, "localhost", 9999) val stateful = ActiveObject.newRemoteInstance(classOf[TransactionalActiveObject], 1000, HOSTNAME, PORT)
stateful.setRefState("init") // set init state stateful.setRefState("init") // set init state
val failer =ActiveObject.newRemoteInstance(classOf[InMemFailer], 1000, "localhost", 9999) //conf.getInstance(classOf[InMemFailer]) val failer =ActiveObject.newRemoteInstance(classOf[ActiveObjectFailer], 1000, HOSTNAME, PORT) //conf.getInstance(classOf[ActiveObjectFailer])
try { try {
stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method
fail("should have thrown an exception") fail("should have thrown an exception")

View file

@ -100,7 +100,7 @@ class StmSpec extends
} }
} }
} }
/*
describe("Transactor") { describe("Transactor") {
it("should be able receive message sent with !! and pass it along to nested transactor with !! and receive reply; multiple times in a row") { it("should be able receive message sent with !! and pass it along to nested transactor with !! and receive reply; multiple times in a row") {
import GlobalTransactionVectorTestActor._ import GlobalTransactionVectorTestActor._
@ -121,6 +121,7 @@ class StmSpec extends
size4 should equal(3) size4 should equal(3)
} }
} }
*/
} }
object GlobalTransactionVectorTestActor { object GlobalTransactionVectorTestActor {

View file

@ -5,6 +5,7 @@
package se.scalablesolutions.akka.actor package se.scalablesolutions.akka.actor
import se.scalablesolutions.akka.config.ScalaConfig._ import se.scalablesolutions.akka.config.ScalaConfig._
import se.scalablesolutions.akka.config.OneForOneStrategy
import se.scalablesolutions.akka.dispatch.Dispatchers import se.scalablesolutions.akka.dispatch.Dispatchers
import se.scalablesolutions.akka.{OneWay, Die, Ping} import se.scalablesolutions.akka.{OneWay, Die, Ping}
import Actor._ import Actor._
@ -75,6 +76,33 @@ object SupervisorSpec {
messageLog.put(reason.getMessage) messageLog.put(reason.getMessage)
} }
} }
class TemporaryActor extends Actor {
import self._
lifeCycle = Some(LifeCycle(Temporary))
def receive = {
case Ping =>
messageLog.put("ping")
reply("pong")
case Die =>
println("******************** GOT DIE 3")
throw new RuntimeException("DIE")
}
override def postRestart(reason: Throwable) {
println("******************** restart temporary")
messageLog.put(reason.getMessage)
}
}
class Master extends Actor {
self.trapExit = classOf[Exception] :: Nil
self.faultHandler = Some(OneForOneStrategy(5, 1000))
val temp = self.spawnLink[TemporaryActor]
override def receive = {
case Die => temp !! (Die, 5000)
}
}
} }
/** /**
@ -86,7 +114,9 @@ class SupervisorSpec extends JUnitSuite {
var pingpong1: ActorRef = _ var pingpong1: ActorRef = _
var pingpong2: ActorRef = _ var pingpong2: ActorRef = _
var pingpong3: ActorRef = _ var pingpong3: ActorRef = _
var temporaryActor: ActorRef = _
/*
@Test def shouldStartServer = { @Test def shouldStartServer = {
clearMessageLogs clearMessageLogs
val sup = getSingleActorAllForOneSupervisor val sup = getSingleActorAllForOneSupervisor
@ -96,6 +126,30 @@ class SupervisorSpec extends JUnitSuite {
(pingpong1 !! (Ping, 5000)).getOrElse("nil") (pingpong1 !! (Ping, 5000)).getOrElse("nil")
} }
} }
*/
@Test def shoulNotRestartProgrammaticallyLinkedTemporaryActor = {
clearMessageLogs
val master = actorOf[Master].start
intercept[RuntimeException] {
master !! (Die, 5000)
}
Thread.sleep(1000)
assert(messageLog.size === 0)
}
@Test def shoulNotRestartTemporaryActor = {
clearMessageLogs
val sup = getTemporaryActorAllForOneSupervisor
intercept[RuntimeException] {
temporaryActor !! (Die, 5000)
}
Thread.sleep(1000)
assert(messageLog.size === 0)
}
@Test def shouldStartServerForNestedSupervisorHierarchy = { @Test def shouldStartServerForNestedSupervisorHierarchy = {
clearMessageLogs clearMessageLogs
@ -445,6 +499,18 @@ class SupervisorSpec extends JUnitSuite {
// ============================================= // =============================================
// Create some supervisors with different configurations // Create some supervisors with different configurations
def getTemporaryActorAllForOneSupervisor: Supervisor = {
temporaryActor = actorOf[TemporaryActor].start
Supervisor(
SupervisorConfig(
RestartStrategy(AllForOne, 3, 5000, List(classOf[Exception])),
Supervise(
temporaryActor,
LifeCycle(Temporary))
:: Nil))
}
def getSingleActorAllForOneSupervisor: Supervisor = { def getSingleActorAllForOneSupervisor: Supervisor = {
pingpong1 = actorOf[PingPong1Actor].start pingpong1 = actorOf[PingPong1Actor].start

View file

@ -18,7 +18,7 @@ import se.scalablesolutions.akka.config.JavaConfig._
import se.scalablesolutions.akka.actor._ import se.scalablesolutions.akka.actor._
@RunWith(classOf[JUnitRunner]) @RunWith(classOf[JUnitRunner])
class InMemoryStateSpec extends class TransactionalActiveObjectSpec extends
Spec with Spec with
ShouldMatchers with ShouldMatchers with
BeforeAndAfterAll { BeforeAndAfterAll {
@ -31,11 +31,11 @@ class InMemoryStateSpec extends
conf.configure( conf.configure(
new RestartStrategy(new AllForOne, 3, 5000, List(classOf[Exception]).toArray), new RestartStrategy(new AllForOne, 3, 5000, List(classOf[Exception]).toArray),
List( List(
new Component(classOf[InMemStateful], new Component(classOf[TransactionalActiveObject],
new LifeCycle(new Permanent), new LifeCycle(new Permanent),
//new RestartCallbacks("preRestart", "postRestart")), //new RestartCallbacks("preRestart", "postRestart")),
10000), 10000),
new Component(classOf[InMemFailer], new Component(classOf[ActiveObjectFailer],
new LifeCycle(new Permanent), new LifeCycle(new Permanent),
10000)).toArray 10000)).toArray
).supervise ).supervise
@ -48,7 +48,7 @@ class InMemoryStateSpec extends
describe("Transactional in-memory Active Object ") { describe("Transactional in-memory Active Object ") {
it("map should not rollback state for stateful server in case of success") { it("map should not rollback state for stateful server in case of success") {
val stateful = conf.getInstance(classOf[InMemStateful]) val stateful = conf.getInstance(classOf[TransactionalActiveObject])
stateful.init stateful.init
stateful.setMapState("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "init") stateful.setMapState("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "init")
stateful.success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") stateful.success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state")
@ -56,10 +56,10 @@ class InMemoryStateSpec extends
} }
it("map should rollback state for stateful server in case of failure") { it("map should rollback state for stateful server in case of failure") {
val stateful = conf.getInstance(classOf[InMemStateful]) val stateful = conf.getInstance(classOf[TransactionalActiveObject])
stateful.init stateful.init
stateful.setMapState("testShouldRollbackStateForStatefulServerInCaseOfFailure", "init") stateful.setMapState("testShouldRollbackStateForStatefulServerInCaseOfFailure", "init")
val failer = conf.getInstance(classOf[InMemFailer]) val failer = conf.getInstance(classOf[ActiveObjectFailer])
try { try {
stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer)
fail("should have thrown an exception") fail("should have thrown an exception")
@ -68,10 +68,10 @@ class InMemoryStateSpec extends
} }
it("vector should rollback state for stateful server in case of failure") { it("vector should rollback state for stateful server in case of failure") {
val stateful = conf.getInstance(classOf[InMemStateful]) val stateful = conf.getInstance(classOf[TransactionalActiveObject])
stateful.init stateful.init
stateful.setVectorState("init") // set init state stateful.setVectorState("init") // set init state
val failer = conf.getInstance(classOf[InMemFailer]) val failer = conf.getInstance(classOf[ActiveObjectFailer])
try { try {
stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer)
fail("should have thrown an exception") fail("should have thrown an exception")
@ -80,7 +80,7 @@ class InMemoryStateSpec extends
} }
it("vector should not rollback state for stateful server in case of success") { it("vector should not rollback state for stateful server in case of success") {
val stateful = conf.getInstance(classOf[InMemStateful]) val stateful = conf.getInstance(classOf[TransactionalActiveObject])
stateful.init stateful.init
stateful.setVectorState("init") // set init state stateful.setVectorState("init") // set init state
stateful.success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") stateful.success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state")
@ -88,10 +88,10 @@ class InMemoryStateSpec extends
} }
it("ref should rollback state for stateful server in case of failure") { it("ref should rollback state for stateful server in case of failure") {
val stateful = conf.getInstance(classOf[InMemStateful]) val stateful = conf.getInstance(classOf[TransactionalActiveObject])
stateful.init stateful.init
stateful.setRefState("init") // set init state stateful.setRefState("init") // set init state
val failer = conf.getInstance(classOf[InMemFailer]) val failer = conf.getInstance(classOf[ActiveObjectFailer])
try { try {
stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) stateful.failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer)
fail("should have thrown an exception") fail("should have thrown an exception")
@ -100,7 +100,7 @@ class InMemoryStateSpec extends
} }
it("ref should not rollback state for stateful server in case of success") { it("ref should not rollback state for stateful server in case of success") {
val stateful = conf.getInstance(classOf[InMemStateful]) val stateful = conf.getInstance(classOf[TransactionalActiveObject])
stateful.init stateful.init
stateful.setRefState("init") // set init state stateful.setRefState("init") // set init state
stateful.success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") stateful.success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state")

View file

@ -7,7 +7,7 @@ import org.junit.Test
import se.scalablesolutions.akka.stm.{Ref, TransactionalMap, TransactionalVector} import se.scalablesolutions.akka.stm.{Ref, TransactionalMap, TransactionalVector}
import Actor._ import Actor._
object InMemoryActorSpec { object TransactorSpec {
case class GetMapState(key: String) case class GetMapState(key: String)
case object GetVectorState case object GetVectorState
case object GetVectorSize case object GetVectorSize
@ -27,9 +27,9 @@ object InMemoryActorSpec {
case object GetNotifier case object GetNotifier
} }
import InMemoryActorSpec._ import TransactorSpec._
class InMemStatefulActor(expectedInvocationCount: Int) extends Transactor { class StatefulTransactor(expectedInvocationCount: Int) extends Transactor {
def this() = this(0) def this() = this(0)
self.timeout = 5000 self.timeout = 5000
@ -101,7 +101,7 @@ class InMemStatefulActor(expectedInvocationCount: Int) extends Transactor {
} }
@serializable @serializable
class InMemFailerActor extends Transactor { class FailerTransactor extends Transactor {
def receive = { def receive = {
case "Failure" => case "Failure" =>
@ -109,10 +109,10 @@ class InMemFailerActor extends Transactor {
} }
} }
class InMemoryActorSpec extends JUnitSuite { class TransactorSpec extends JUnitSuite {
@Test @Test
def shouldOneWayMapShouldNotRollbackStateForStatefulServerInCaseOfSuccess = { def shouldOneWayMapShouldNotRollbackStateForStatefulServerInCaseOfSuccess = {
val stateful = actorOf(new InMemStatefulActor(2)) val stateful = actorOf(new StatefulTransactor(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
@ -123,7 +123,7 @@ class InMemoryActorSpec extends JUnitSuite {
@Test @Test
def shouldMapShouldNotRollbackStateForStatefulServerInCaseOfSuccess = { def shouldMapShouldNotRollbackStateForStatefulServerInCaseOfSuccess = {
val stateful = actorOf[InMemStatefulActor] val stateful = actorOf[StatefulTransactor]
stateful.start stateful.start
stateful !! SetMapState("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "init") // set init state stateful !! SetMapState("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "init") // set init state
stateful !! Success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired stateful !! Success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired
@ -132,9 +132,9 @@ class InMemoryActorSpec extends JUnitSuite {
@Test @Test
def shouldOneWayMapShouldRollbackStateForStatefulServerInCaseOfFailure = { def shouldOneWayMapShouldRollbackStateForStatefulServerInCaseOfFailure = {
val stateful = actorOf(new InMemStatefulActor(2)) val stateful = actorOf(new StatefulTransactor(2))
stateful.start stateful.start
val failer = actorOf[InMemFailerActor] val failer = actorOf[FailerTransactor]
failer.start failer.start
stateful ! SetMapStateOneWay("testShouldRollbackStateForStatefulServerInCaseOfFailure", "init") // set init state stateful ! SetMapStateOneWay("testShouldRollbackStateForStatefulServerInCaseOfFailure", "init") // set init state
stateful ! FailureOneWay("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method stateful ! FailureOneWay("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method
@ -145,10 +145,10 @@ class InMemoryActorSpec extends JUnitSuite {
@Test @Test
def shouldMapShouldRollbackStateForStatefulServerInCaseOfFailure = { def shouldMapShouldRollbackStateForStatefulServerInCaseOfFailure = {
val stateful = actorOf[InMemStatefulActor] val stateful = actorOf[StatefulTransactor]
stateful.start stateful.start
stateful !! SetMapState("testShouldRollbackStateForStatefulServerInCaseOfFailure", "init") // set init state stateful !! SetMapState("testShouldRollbackStateForStatefulServerInCaseOfFailure", "init") // set init state
val failer = actorOf[InMemFailerActor] val failer = actorOf[FailerTransactor]
failer.start failer.start
try { try {
stateful !! Failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method stateful !! Failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method
@ -159,7 +159,7 @@ class InMemoryActorSpec extends JUnitSuite {
@Test @Test
def shouldOneWayVectorShouldNotRollbackStateForStatefulServerInCaseOfSuccess = { def shouldOneWayVectorShouldNotRollbackStateForStatefulServerInCaseOfSuccess = {
val stateful = actorOf(new InMemStatefulActor(2)) val stateful = actorOf(new StatefulTransactor(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
@ -170,7 +170,7 @@ class InMemoryActorSpec extends JUnitSuite {
@Test @Test
def shouldVectorShouldNotRollbackStateForStatefulServerInCaseOfSuccess = { def shouldVectorShouldNotRollbackStateForStatefulServerInCaseOfSuccess = {
val stateful = actorOf[InMemStatefulActor] val stateful = actorOf[StatefulTransactor]
stateful.start stateful.start
stateful !! SetVectorState("init") // set init state stateful !! SetVectorState("init") // set init state
stateful !! Success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired stateful !! Success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired
@ -179,11 +179,11 @@ class InMemoryActorSpec extends JUnitSuite {
@Test @Test
def shouldOneWayVectorShouldRollbackStateForStatefulServerInCaseOfFailure = { def shouldOneWayVectorShouldRollbackStateForStatefulServerInCaseOfFailure = {
val stateful = actorOf(new InMemStatefulActor(2)) val stateful = actorOf(new StatefulTransactor(2))
stateful.start stateful.start
stateful ! SetVectorStateOneWay("init") // set init state stateful ! SetVectorStateOneWay("init") // set init state
Thread.sleep(1000) Thread.sleep(1000)
val failer = actorOf[InMemFailerActor] val failer = actorOf[FailerTransactor]
failer.start failer.start
stateful ! FailureOneWay("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method stateful ! FailureOneWay("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method
val notifier = (stateful !! GetNotifier).as[CountDownLatch] val notifier = (stateful !! GetNotifier).as[CountDownLatch]
@ -193,10 +193,10 @@ class InMemoryActorSpec extends JUnitSuite {
@Test @Test
def shouldVectorShouldRollbackStateForStatefulServerInCaseOfFailure = { def shouldVectorShouldRollbackStateForStatefulServerInCaseOfFailure = {
val stateful = actorOf[InMemStatefulActor] val stateful = actorOf[StatefulTransactor]
stateful.start stateful.start
stateful !! SetVectorState("init") // set init state stateful !! SetVectorState("init") // set init state
val failer = actorOf[InMemFailerActor] val failer = actorOf[FailerTransactor]
failer.start failer.start
try { try {
stateful !! Failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method stateful !! Failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method
@ -207,7 +207,7 @@ class InMemoryActorSpec extends JUnitSuite {
@Test @Test
def shouldOneWayRefShouldNotRollbackStateForStatefulServerInCaseOfSuccess = { def shouldOneWayRefShouldNotRollbackStateForStatefulServerInCaseOfSuccess = {
val stateful = actorOf(new InMemStatefulActor(2)) val stateful = actorOf(new StatefulTransactor(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
@ -218,7 +218,7 @@ class InMemoryActorSpec extends JUnitSuite {
@Test @Test
def shouldRefShouldNotRollbackStateForStatefulServerInCaseOfSuccess = { def shouldRefShouldNotRollbackStateForStatefulServerInCaseOfSuccess = {
val stateful = actorOf[InMemStatefulActor] val stateful = actorOf[StatefulTransactor]
stateful.start stateful.start
stateful !! SetRefState("init") // set init state stateful !! SetRefState("init") // set init state
stateful !! Success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired stateful !! Success("testShouldNotRollbackStateForStatefulServerInCaseOfSuccess", "new state") // transactionrequired
@ -227,11 +227,11 @@ class InMemoryActorSpec extends JUnitSuite {
@Test @Test
def shouldOneWayRefShouldRollbackStateForStatefulServerInCaseOfFailure = { def shouldOneWayRefShouldRollbackStateForStatefulServerInCaseOfFailure = {
val stateful = actorOf(new InMemStatefulActor(2)) val stateful = actorOf(new StatefulTransactor(2))
stateful.start stateful.start
stateful ! SetRefStateOneWay("init") // set init state stateful ! SetRefStateOneWay("init") // set init state
Thread.sleep(1000) Thread.sleep(1000)
val failer = actorOf[InMemFailerActor] val failer = actorOf[FailerTransactor]
failer.start failer.start
stateful ! FailureOneWay("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method stateful ! FailureOneWay("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method
val notifier = (stateful !! GetNotifier).as[CountDownLatch] val notifier = (stateful !! GetNotifier).as[CountDownLatch]
@ -241,10 +241,10 @@ class InMemoryActorSpec extends JUnitSuite {
@Test @Test
def shouldRefShouldRollbackStateForStatefulServerInCaseOfFailure = { def shouldRefShouldRollbackStateForStatefulServerInCaseOfFailure = {
val stateful = actorOf[InMemStatefulActor] val stateful = actorOf[StatefulTransactor]
stateful.start stateful.start
stateful !! SetRefState("init") // set init state stateful !! SetRefState("init") // set init state
val failer = actorOf[InMemFailerActor] val failer = actorOf[FailerTransactor]
failer.start failer.start
try { try {
stateful !! Failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method stateful !! Failure("testShouldRollbackStateForStatefulServerInCaseOfFailure", "new state", failer) // call failing transactionrequired method