implemented waiting for pending transactions to complete before aborting + config

This commit is contained in:
Jonas Boner 2009-07-04 12:06:07 +02:00
parent d75d769351
commit 3830aed805
6 changed files with 602 additions and 211 deletions

View file

@ -10,7 +10,7 @@ import java.util.concurrent.CopyOnWriteArraySet
import kernel.nio.{RemoteServer, RemoteClient, RemoteRequest}
import kernel.reactor._
import kernel.config.ScalaConfig._
import kernel.stm.{TransactionAwareWrapperException, TransactionManagement}
import kernel.stm.{TransactionRollbackException, TransactionAwareWrapperException, TransactionManagement}
import kernel.util.Helpers.ReadWriteLock
import kernel.util.Logging
import org.codehaus.aspectwerkz.proxy.Uuid
@ -35,17 +35,16 @@ class ActorMessageHandler(val actor: Actor) extends MessageHandler {
}
object Actor {
val timeout = kernel.Kernel.config.getInt("akka.actor.timeout", 5000)
val TIMEOUT = kernel.Kernel.config.getInt("akka.actor.timeout", 5000)
}
trait Actor extends Logging with TransactionManagement {
trait Actor extends Logging with TransactionManagement {
@volatile private[this] var isRunning: Boolean = false
private[this] val remoteFlagLock = new ReadWriteLock
private[this] val transactionalFlagLock = new ReadWriteLock
private var hotswap: Option[PartialFunction[Any, Unit]] = None
private var config: Option[AnyRef] = None
@volatile protected[this] var isTransactional = false
@volatile protected[this] var remoteAddress: Option[InetSocketAddress] = None
@volatile protected[kernel] var supervisor: Option[Actor] = None
@ -54,6 +53,9 @@ trait Actor extends Logging with TransactionManagement {
protected[this] val linkedActors = new CopyOnWriteArraySet[Actor]
protected[actor] var lifeCycleConfig: Option[LifeCycle] = None
protected[this] var latestMessage: Option[MessageHandle] = None
protected[this] var messageToReschedule: Option[MessageHandle] = None
// ====================================
// ==== USER CALLBACKS TO OVERRIDE ====
// ====================================
@ -63,7 +65,7 @@ trait Actor extends Logging with TransactionManagement {
*
* Defines the default timeout for '!!' invocations, e.g. the timeout for the future returned by the call to '!!'.
*/
@volatile var timeout: Long = Actor.timeout
@volatile var timeout: Long = Actor.TIMEOUT
/**
* User overridable callback/setting.
@ -80,7 +82,7 @@ trait Actor extends Logging with TransactionManagement {
* <pre/>
* // default - executorService can be build up using the ThreadPoolBuilder
* new EventBasedThreadPoolDispatcher(executor: ExecutorService)
*
*
* new EventBasedSingleThreadDispatcher
* </pre>
*/
@ -112,7 +114,7 @@ trait Actor extends Logging with TransactionManagement {
* Can be one of:
* <pre/>
* AllForOneStrategy(maxNrOfRetries: Int, withinTimeRange: Int)
*
*
* OneForOneStrategy(maxNrOfRetries: Int, withinTimeRange: Int)
* </pre>
*/
@ -196,15 +198,15 @@ trait Actor extends Logging with TransactionManagement {
isRunning = false
} else throw new IllegalStateException("Actor has not been started, you need to invoke 'actor.start' before using it")
}
/**
* Sends a one-way asynchronous message. E.g. fire-and-forget semantics.
* Sends a one-way asynchronous message. E.g. fire-and-forget semantics.
*/
def !(message: AnyRef): Unit = if (isRunning) {
if (TransactionManagement.isTransactionalityEnabled) transactionalDispatch(message, timeout, false, true)
else postMessageToMailbox(message)
} else throw new IllegalStateException("Actor has not been started, you need to invoke 'actor.start' before using it")
/**
* Sends a message asynchronously and waits on a future for a reply message.
* It waits on the reply either until it receives it (returns Some(replyMessage) or until the timeout expires (returns None).
@ -236,7 +238,7 @@ trait Actor extends Logging with TransactionManagement {
def !![T](message: AnyRef): Option[T] = !![T](message, timeout)
/**
* Sends a message asynchronously, but waits on a future indefinitely. E.g. emulates a synchronous call.
* Sends a message asynchronously, but waits on a future indefinitely. E.g. emulates a synchronous call.
* E.g. send-and-receive-eventually semantics.
*/
def !?[T](message: AnyRef): T = if (isRunning) {
@ -288,7 +290,7 @@ trait Actor extends Logging with TransactionManagement {
* Links an other actor to this actor. Links are unidirectional and means that a the linking actor will receive a notification nif the linked actor has crashed.
* If the 'trapExit' flag has been set then it will 'trap' the failure and automatically restart the linked actors according to the restart strategy defined by the 'faultHandler'.
* <p/>
* To be invoked from within the actor itself.
* To be invoked from within the actor itself.
*/
protected[this] def link(actor: Actor) = {
if (isRunning) {
@ -300,9 +302,9 @@ trait Actor extends Logging with TransactionManagement {
}
/**
* Unlink the actor.
* Unlink the actor.
* <p/>
* To be invoked from within the actor itself.
* To be invoked from within the actor itself.
*/
protected[this] def unlink(actor: Actor) = {
if (isRunning) {
@ -312,11 +314,11 @@ trait Actor extends Logging with TransactionManagement {
log.debug("Unlinking actor [%s] from actor [%s]", actor, this)
} else throw new IllegalStateException("Actor has not been started, you need to invoke 'actor.start' before using it")
}
/**
* Atomically start and link an actor.
* <p/>
* To be invoked from within the actor itself.
* To be invoked from within the actor itself.
*/
protected[this] def startLink(actor: Actor) = {
actor.start
@ -326,7 +328,7 @@ trait Actor extends Logging with TransactionManagement {
/**
* Atomically start, link and make an actor remote.
* <p/>
* To be invoked from within the actor itself.
* To be invoked from within the actor itself.
*/
protected[this] def startLinkRemote(actor: Actor) = {
actor.makeRemote(RemoteServer.HOSTNAME, RemoteServer.PORT)
@ -337,10 +339,10 @@ trait Actor extends Logging with TransactionManagement {
/**
* Atomically create (from actor class) and start an actor.
* <p/>
* To be invoked from within the actor itself.
* To be invoked from within the actor itself.
*/
protected[this] def spawn(actorClass: Class[_]): Actor = {
val actor = actorClass.newInstance.asInstanceOf[Actor]
val actor = actorClass.newInstance.asInstanceOf[Actor]
actor.dispatcher = dispatcher
actor.mailbox = mailbox
actor.start
@ -350,10 +352,10 @@ trait Actor extends Logging with TransactionManagement {
/**
* Atomically create (from actor class), start and make an actor remote.
* <p/>
* To be invoked from within the actor itself.
* To be invoked from within the actor itself.
*/
protected[this] def spawnRemote(actorClass: Class[_]): Actor = {
val actor = actorClass.newInstance.asInstanceOf[Actor]
val actor = actorClass.newInstance.asInstanceOf[Actor]
actor.makeRemote(RemoteServer.HOSTNAME, RemoteServer.PORT)
actor.dispatcher = dispatcher
actor.mailbox = mailbox
@ -364,7 +366,7 @@ trait Actor extends Logging with TransactionManagement {
/**
* Atomically create (from actor class), start and link an actor.
* <p/>
* To be invoked from within the actor itself.
* To be invoked from within the actor itself.
*/
protected[this] def spawnLink(actorClass: Class[_]): Actor = {
val actor = spawn(actorClass)
@ -375,7 +377,7 @@ trait Actor extends Logging with TransactionManagement {
/**
* Atomically create (from actor class), start, link and make an actor remote.
* <p/>
* To be invoked from within the actor itself.
* To be invoked from within the actor itself.
*/
protected[this] def spawnLinkRemote(actorClass: Class[_]): Actor = {
val actor = spawn(actorClass)
@ -392,7 +394,11 @@ trait Actor extends Logging with TransactionManagement {
if (remoteAddress.isDefined) {
val supervisorUuid = registerSupervisorAsRemoteActor
RemoteClient.clientFor(remoteAddress.get).send(new RemoteRequest(true, message, null, this.getClass.getName, timeout, null, true, false, supervisorUuid))
} else mailbox.append(new MessageHandle(this, message, None, activeTx))
} else {
val handle = new MessageHandle(this, message, None, activeTx)
mailbox.append(handle)
latestMessage = Some(handle)
}
}
private def postMessageToMailboxAndCreateFutureResultWithTimeout(message: AnyRef, timeout: Long): CompletableFutureResult = remoteFlagLock.withReadLock { // the price you pay for being able to make an actor remote at runtime
@ -403,13 +409,31 @@ trait Actor extends Logging with TransactionManagement {
else throw new IllegalStateException("Expected a future from remote call to actor " + toString)
} else {
val future = new DefaultCompletableFutureResult(timeout)
mailbox.append(new MessageHandle(this, message, Some(future), TransactionManagement.threadBoundTx.get))
val handle = new MessageHandle(this, message, Some(future), TransactionManagement.threadBoundTx.get)
mailbox.append(handle)
latestMessage = Some(handle)
future
}
}
private def transactionalDispatch[T](message: AnyRef, timeout: Long, blocking: Boolean, oneWay: Boolean): Option[T] = {
tryToCommitTransaction
import TransactionManagement._
if (!tryToCommitTransaction) {
var nrRetries = 0 // FIXME only if top-level
var failed = true
do {
Thread.sleep(TIME_WAITING_FOR_COMPLETION)
nrRetries += 1
log.debug("Pending transaction [%s] not completed, waiting %s milliseconds. Attempt %s", activeTx.get, TIME_WAITING_FOR_COMPLETION, nrRetries)
failed = !tryToCommitTransaction
} while(nrRetries < NR_OF_TIMES_WAITING_FOR_COMPLETION && failed)
if (failed) {
log.debug("Pending transaction [%s] still not completed, aborting and rescheduling message [%s]", activeTx.get, latestMessage)
rollback(activeTx)
if (RESTART_TRANSACTION_ON_COLLISION) messageToReschedule = Some(latestMessage.get)
else throw new TransactionRollbackException("Conflicting transactions, rolling back transaction for message [" + latestMessage + "]")
}
}
if (isInExistingTransaction) joinExistingTransaction
else if (isTransactional) startNewTransaction
incrementTransaction
@ -433,6 +457,13 @@ trait Actor extends Logging with TransactionManagement {
if (isTransactionAborted) removeTransactionIfTopLevel
else tryToPrecommitTransaction
TransactionManagement.threadBoundTx.set(None)
if (messageToReschedule.isDefined) {
val handle = messageToReschedule.get
val newTx = startNewTransaction
val clone = new MessageHandle(handle.sender, handle.message, handle.future, newTx)
log.debug("Rescheduling message %s", clone)
mailbox.append(clone) // FIXME append or prepend rescheduled messages?
}
}
}
@ -481,7 +512,7 @@ trait Actor extends Logging with TransactionManagement {
if (trapExit) {
if (faultHandler.isDefined) {
faultHandler.get match {
// FIXME: implement support for maxNrOfRetries and withinTimeRange in RestartStrategy
// FIXME: implement support for maxNrOfRetries and withinTimeRange in RestartStrategy
case AllForOneStrategy(maxNrOfRetries, withinTimeRange) => restartLinkedActors(reason)
case OneForOneStrategy(maxNrOfRetries, withinTimeRange) => dead.restart(reason)
}

View file

@ -57,6 +57,8 @@ class MessageHandle(val sender: AnyRef,
that.asInstanceOf[MessageHandle].future.get == future.get &&
that.asInstanceOf[MessageHandle].tx.isDefined == tx.isDefined &&
that.asInstanceOf[MessageHandle].tx.get.id == tx.get.id
override def toString(): String = "MessageHandle[message = " + message + ", sender = " + sender + ", future = " + future + ", tx = " + tx + "]"
}
class MessageQueue {

View file

@ -8,7 +8,7 @@ import java.util.concurrent.atomic.{AtomicInteger, AtomicLong}
import kernel.state.Transactional
import kernel.util.Logging
import scala.collection.mutable.{HashSet, HashMap}
class TransactionRollbackException(msg: String) extends RuntimeException(msg)
@serializable sealed abstract class TransactionStatus
object TransactionStatus {
@ -72,7 +72,7 @@ object TransactionIdFactory {
}
}
def commit(participant: String) = synchronized {
def commit(participant: String): Boolean = synchronized {
if (status == TransactionStatus.Active) {
log.debug("TX COMMIT - Committing transaction [%s] for server with UUID [%s]", toString, participant)
val haveAllPreCommitted =
@ -85,9 +85,13 @@ object TransactionIdFactory {
if (haveAllPreCommitted) {
transactionals.items.foreach(_.commit)
status = TransactionStatus.Completed
} else rollback(participant)
reset
true
} else false
} else {
reset
true
}
reset
}
def rollback(participant: String) = synchronized {
@ -98,6 +102,13 @@ object TransactionIdFactory {
reset
}
def rollbackForRescheduling(participant: String) = synchronized {
ensureIsActiveOrAborted
log.debug("TX ROLLBACK for recheduling - Server with UUID [%s] has initiated transaction rollback for [%s]", participant, toString)
transactionals.items.foreach(_.rollback)
reset
}
def join(participant: String) = synchronized {
ensureIsActive
log.debug("TX JOIN - Server with UUID [%s] is joining transaction [%s]" , participant, toString)

View file

@ -14,10 +14,13 @@ class TransactionAwareWrapperException(val cause: Throwable, val tx: Option[Tran
}
object TransactionManagement {
private val txEnabled = new AtomicBoolean(kernel.Kernel.config.getBool("akka.stm.service", true))
val TIME_WAITING_FOR_COMPLETION = kernel.Kernel.config.getInt("akka.stm.wait-for-completion", 100)
val NR_OF_TIMES_WAITING_FOR_COMPLETION = kernel.Kernel.config.getInt("akka.stm.wait-nr-of-times", 3)
val TRANSACTION_ENABLED = new AtomicBoolean(kernel.Kernel.config.getBool("akka.stm.service", true))
val RESTART_TRANSACTION_ON_COLLISION = kernel.Kernel.config.getBool("akka.stm.restart-transaction", true)
def isTransactionalityEnabled = txEnabled.get
def disableTransactions = txEnabled.set(false)
def isTransactionalityEnabled = TRANSACTION_ENABLED.get
def disableTransactions = TRANSACTION_ENABLED.set(false)
private[kernel] val threadBoundTx: ThreadLocal[Option[Transaction]] = new ThreadLocal[Option[Transaction]]() {
override protected def initialValue: Option[Transaction] = None
@ -31,12 +34,13 @@ trait TransactionManagement extends Logging {
import TransactionManagement.threadBoundTx
private[kernel] var activeTx: Option[Transaction] = None
protected def startNewTransaction = {
protected def startNewTransaction: Option[Transaction] = {
val newTx = new Transaction
newTx.begin(uuid)
val tx = Some(newTx)
activeTx = tx
threadBoundTx.set(tx)
tx
}
protected def joinExistingTransaction = {
@ -52,10 +56,11 @@ trait TransactionManagement extends Logging {
protected def tryToCommitTransaction: Boolean = if (activeTx.isDefined) {
val tx = activeTx.get
tx.commit(uuid)
removeTransactionIfTopLevel
true
} else false
if (tx.commit(uuid)) {
removeTransactionIfTopLevel
true
} else false
} else true
protected def rollback(tx: Option[Transaction]) = tx match {
case None => {} // no tx; nothing to do
@ -63,14 +68,13 @@ trait TransactionManagement extends Logging {
tx.rollback(uuid)
}
protected def isInExistingTransaction =
// FIXME should not need to have this runtime "fix" - investigate what is causing this to happen
// if (TransactionManagement.threadBoundTx.get == null) {
// TransactionManagement.threadBoundTx.set(None)
// false
// } else {
TransactionManagement.threadBoundTx.get.isDefined
// }
protected def rollbackForRescheduling(tx: Option[Transaction]) = tx match {
case None => {} // no tx; nothing to do
case Some(tx) =>
tx.rollbackForRescheduling(uuid)
}
protected def isInExistingTransaction = TransactionManagement.threadBoundTx.get.isDefined
protected def isTransactionAborted = activeTx.isDefined && activeTx.get.isAborted

View file

@ -0,0 +1,95 @@
package se.scalablesolutions.akka.kernel.actor
import junit.framework.TestCase
import kernel.stm.TransactionRollbackException
import org.junit.{Test, Before}
import org.junit.Assert._
import kernel.state.TransactionalState
object Log {
var log = ""
}
class TxActor(clasher: Actor) extends Actor {
timeout = 1000000
makeTransactional
def receive: PartialFunction[Any, Unit] = {
case msg: AnyRef =>
clasher !! msg
reply(msg)
}
}
class TxClasherActor extends Actor {
val vector = TransactionalState.newInMemoryVector[String]
timeout = 1000000
makeTransactional
var count = 0
def receive: PartialFunction[Any, Unit] = {
case "First" =>
if (count == 0) Thread.sleep(5000)
count += 1
println("FIRST")
vector.add("First")
println("--- VECTOR: " + vector)
reply("First")
case "Second" =>
println("SECOND")
vector.add("Second")
println("--- VECTOR: " + vector)
reply("Second")
case "Index0" =>
reply(vector(0))
case "Index1" =>
reply(vector(1))
}
}
class TransactionClasherSpec extends TestCase {
@Test
def testX = {
val clasher = new TxClasherActor
clasher.start
val txActor1 = new TxActor(clasher)
txActor1.start
val txActor2 = new TxActor(clasher)
txActor2.start
val t1 = new Thread(new Runnable() {
def run = {
txActor1 !! "First"
}
}).start
Thread.sleep(1000)
try {
txActor2 !! "Second"
fail("Expected TransactionRollbackException")
} catch { case e: TransactionRollbackException => {} }
}
/*
@Test
def testX = {
val clasher = new TxClasherActor
clasher.start
val txActor1 = new TxActor(clasher)
txActor1.start
val txActor2 = new TxActor(clasher)
txActor2.start
val t1 = new Thread(new Runnable() {
def run = {
txActor1 !! "First"
}
}).start
Thread.sleep(1000)
val res2 = txActor2 !! "Second"
Thread.sleep(10000)
assertEquals("Second", (clasher !! "Index0").get)
assertEquals("First", (clasher !! "Index1").get)
}
*/
}