Added flushing shutdown to Remoting
This commit is contained in:
parent
ddcb2192eb
commit
eb3a0de01b
1 changed files with 57 additions and 27 deletions
|
|
@ -4,22 +4,21 @@ import scala.language.postfixOps
|
||||||
import akka.actor.SupervisorStrategy._
|
import akka.actor.SupervisorStrategy._
|
||||||
import akka.actor._
|
import akka.actor._
|
||||||
import akka.event.{ Logging, LoggingAdapter }
|
import akka.event.{ Logging, LoggingAdapter }
|
||||||
import akka.pattern.{ gracefulStop, pipe }
|
import akka.japi.Util.immutableSeq
|
||||||
import akka.remote.EndpointManager.{ StartupFinished, ManagementCommand, Listen, Send }
|
import akka.pattern.{ gracefulStop, pipe, ask }
|
||||||
import akka.remote.transport.Transport.{ AssociationEventListener, InboundAssociation }
|
import akka.remote.EndpointManager._
|
||||||
|
import akka.remote.Remoting.TransportSupervisor
|
||||||
|
import akka.remote.transport.Transport.AssociationEventListener
|
||||||
|
import akka.remote.transport.Transport.InboundAssociation
|
||||||
import akka.remote.transport._
|
import akka.remote.transport._
|
||||||
import akka.util.Timeout
|
import akka.util.Timeout
|
||||||
import com.typesafe.config.{ ConfigFactory, Config }
|
import com.typesafe.config.Config
|
||||||
|
import java.net.URLEncoder
|
||||||
|
import java.util.concurrent.TimeoutException
|
||||||
import scala.collection.immutable.{ Seq, HashMap }
|
import scala.collection.immutable.{ Seq, HashMap }
|
||||||
import scala.concurrent.duration._
|
import scala.concurrent.duration._
|
||||||
import scala.concurrent.{ Promise, Await, Future }
|
import scala.concurrent.{ Promise, Await, Future }
|
||||||
import scala.util.control.NonFatal
|
import scala.util.control.NonFatal
|
||||||
import java.net.URLEncoder
|
|
||||||
import java.util.concurrent.TimeoutException
|
|
||||||
import scala.util.{ Failure, Success }
|
|
||||||
import scala.collection.immutable
|
|
||||||
import akka.japi.Util.immutableSeq
|
|
||||||
import akka.remote.Remoting.{ TransportSupervisor, RegisterTransportActor }
|
|
||||||
|
|
||||||
class RemotingSettings(val config: Config) {
|
class RemotingSettings(val config: Config) {
|
||||||
|
|
||||||
|
|
@ -30,6 +29,8 @@ class RemotingSettings(val config: Config) {
|
||||||
|
|
||||||
val ShutdownTimeout: FiniteDuration = Duration(getMilliseconds("akka.remoting.shutdown-timeout"), MILLISECONDS)
|
val ShutdownTimeout: FiniteDuration = Duration(getMilliseconds("akka.remoting.shutdown-timeout"), MILLISECONDS)
|
||||||
|
|
||||||
|
val FlushWait: FiniteDuration = Duration(getMilliseconds("akka.remoting.flush-wait-on-shutdown"), MILLISECONDS)
|
||||||
|
|
||||||
val StartupTimeout: FiniteDuration = Duration(getMilliseconds("akka.remoting.startup-timeout"), MILLISECONDS)
|
val StartupTimeout: FiniteDuration = Duration(getMilliseconds("akka.remoting.startup-timeout"), MILLISECONDS)
|
||||||
|
|
||||||
val RetryGateClosedFor: Long = getNanoseconds("akka.remoting.retry-gate-closed-for")
|
val RetryGateClosedFor: Long = getNanoseconds("akka.remoting.retry-gate-closed-for")
|
||||||
|
|
@ -145,11 +146,13 @@ private[remote] class Remoting(_system: ExtendedActorSystem, _provider: RemoteAc
|
||||||
endpointManager match {
|
endpointManager match {
|
||||||
case Some(manager) ⇒
|
case Some(manager) ⇒
|
||||||
try {
|
try {
|
||||||
val stopped: Future[Boolean] = gracefulStop(manager, settings.ShutdownTimeout)(system)
|
implicit val timeout = new Timeout(settings.ShutdownTimeout)
|
||||||
|
val stopped: Future[Boolean] = (manager ? ShutdownAndFlush).mapTo[Boolean]
|
||||||
|
|
||||||
if (Await.result(stopped, settings.ShutdownTimeout)) {
|
if (!Await.result(stopped, settings.ShutdownTimeout))
|
||||||
eventPublisher.notifyListeners(RemotingShutdownEvent)
|
log.warning("Shutdown finished, but flushing timed out. Some messages might not have been sent. " +
|
||||||
}
|
"Increase akka.remoting.flush-wait-on-shutdown to a larger value to avoid this message.")
|
||||||
|
eventPublisher.notifyListeners(RemotingShutdownEvent)
|
||||||
|
|
||||||
} catch {
|
} catch {
|
||||||
case e: TimeoutException ⇒ notifyError("Shutdown timed out.", e)
|
case e: TimeoutException ⇒ notifyError("Shutdown timed out.", e)
|
||||||
|
|
@ -231,16 +234,18 @@ private[remote] class Remoting(_system: ExtendedActorSystem, _provider: RemoteAc
|
||||||
|
|
||||||
private[remote] object EndpointManager {
|
private[remote] object EndpointManager {
|
||||||
|
|
||||||
|
// Messages between Remoting and EndpointManager
|
||||||
sealed trait RemotingCommand
|
sealed trait RemotingCommand
|
||||||
case class Listen(addressesPromise: Promise[Seq[(Transport, Address)]]) extends RemotingCommand
|
case class Listen(addressesPromise: Promise[Seq[(Transport, Address)]]) extends RemotingCommand
|
||||||
case object StartupFinished extends RemotingCommand
|
case object StartupFinished extends RemotingCommand
|
||||||
|
case object ShutdownAndFlush extends RemotingCommand
|
||||||
case class Send(message: Any, senderOption: Option[ActorRef], recipient: RemoteActorRef) extends RemotingCommand {
|
case class Send(message: Any, senderOption: Option[ActorRef], recipient: RemoteActorRef) extends RemotingCommand {
|
||||||
override def toString = s"Remote message $senderOption -> $recipient"
|
override def toString = s"Remote message $senderOption -> $recipient"
|
||||||
}
|
}
|
||||||
|
|
||||||
case class ManagementCommand(cmd: Any, statusPromise: Promise[Boolean]) extends RemotingCommand
|
case class ManagementCommand(cmd: Any, statusPromise: Promise[Boolean]) extends RemotingCommand
|
||||||
|
|
||||||
|
// Messages internal to EndpointManager
|
||||||
|
case object Prune
|
||||||
case class ListensResult(addressesPromise: Promise[Seq[(Transport, Address)]],
|
case class ListensResult(addressesPromise: Promise[Seq[(Transport, Address)]],
|
||||||
results: Seq[(Transport, Address, Promise[AssociationEventListener])])
|
results: Seq[(Transport, Address, Promise[AssociationEventListener])])
|
||||||
|
|
||||||
|
|
@ -249,8 +254,6 @@ private[remote] object EndpointManager {
|
||||||
case class Gated(timeOfFailure: Long) extends EndpointPolicy
|
case class Gated(timeOfFailure: Long) extends EndpointPolicy
|
||||||
case class Quarantined(reason: Throwable) extends EndpointPolicy
|
case class Quarantined(reason: Throwable) extends EndpointPolicy
|
||||||
|
|
||||||
case object Prune
|
|
||||||
|
|
||||||
// Not threadsafe -- only to be used in HeadActor
|
// Not threadsafe -- only to be used in HeadActor
|
||||||
private[EndpointManager] class EndpointRegistry {
|
private[EndpointManager] class EndpointRegistry {
|
||||||
private var addressToEndpointAndPolicy = HashMap[Address, EndpointPolicy]()
|
private var addressToEndpointAndPolicy = HashMap[Address, EndpointPolicy]()
|
||||||
|
|
@ -311,6 +314,8 @@ private[remote] object EndpointManager {
|
||||||
addressToPassive = addressToPassive - address
|
addressToPassive = addressToPassive - address
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
def allEndpoints: Iterable[ActorRef] = endpointToAddress.keys
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
@ -358,7 +363,6 @@ private[remote] class EndpointManager(conf: Config, log: LoggingAdapter) extends
|
||||||
def receive = {
|
def receive = {
|
||||||
case Listen(addressesPromise) ⇒
|
case Listen(addressesPromise) ⇒
|
||||||
listens map { ListensResult(addressesPromise, _) } pipeTo self
|
listens map { ListensResult(addressesPromise, _) } pipeTo self
|
||||||
|
|
||||||
case ListensResult(addressesPromise, results) ⇒
|
case ListensResult(addressesPromise, results) ⇒
|
||||||
transportMapping = results.groupBy {
|
transportMapping = results.groupBy {
|
||||||
case (_, transportAddress, _) ⇒ transportAddress
|
case (_, transportAddress, _) ⇒ transportAddress
|
||||||
|
|
@ -374,10 +378,13 @@ private[remote] class EndpointManager(conf: Config, log: LoggingAdapter) extends
|
||||||
transport -> address
|
transport -> address
|
||||||
}
|
}
|
||||||
addressesPromise.success(transportsAndAddresses)
|
addressesPromise.success(transportsAndAddresses)
|
||||||
|
case ManagementCommand(_, statusPromise) ⇒
|
||||||
case ManagementCommand(_, statusPromise) ⇒ statusPromise.success(false)
|
statusPromise.success(false)
|
||||||
|
case StartupFinished ⇒
|
||||||
case StartupFinished ⇒ context.become(accepting)
|
context.become(accepting)
|
||||||
|
case ShutdownAndFlush ⇒
|
||||||
|
sender ! true
|
||||||
|
context.stop(self) // Nothing to flush at this point
|
||||||
}
|
}
|
||||||
|
|
||||||
val accepting: Receive = {
|
val accepting: Receive = {
|
||||||
|
|
@ -398,8 +405,8 @@ private[remote] class EndpointManager(conf: Config, log: LoggingAdapter) extends
|
||||||
None)
|
None)
|
||||||
endpoints.registerActiveEndpoint(recipientAddress, endpoint)
|
endpoints.registerActiveEndpoint(recipientAddress, endpoint)
|
||||||
endpoint ! s
|
endpoint ! s
|
||||||
} else extendedSystem.deadLetters forward message
|
} else forwardToDeadLetters(s)
|
||||||
case Some(Quarantined(_)) ⇒ extendedSystem.deadLetters forward message
|
case Some(Quarantined(_)) ⇒ forwardToDeadLetters(s)
|
||||||
case None ⇒
|
case None ⇒
|
||||||
val endpoint = createEndpoint(
|
val endpoint = createEndpoint(
|
||||||
recipientAddress,
|
recipientAddress,
|
||||||
|
|
@ -428,8 +435,31 @@ private[remote] class EndpointManager(conf: Config, log: LoggingAdapter) extends
|
||||||
endpoints.registerPassiveEndpoint(handle.remoteAddress, endpoint)
|
endpoints.registerPassiveEndpoint(handle.remoteAddress, endpoint)
|
||||||
else handle.disassociate()
|
else handle.disassociate()
|
||||||
}
|
}
|
||||||
case Terminated(endpoint) ⇒ endpoints.removeIfNotGated(endpoint)
|
case Terminated(endpoint) ⇒
|
||||||
case Prune ⇒ endpoints.prune(settings.RetryGateClosedFor)
|
endpoints.removeIfNotGated(endpoint)
|
||||||
|
case Prune ⇒
|
||||||
|
endpoints.prune(settings.RetryGateClosedFor)
|
||||||
|
case ShutdownAndFlush ⇒
|
||||||
|
// Shutdown all endpoints and signal to sender when ready (and whether all endpoints were shut down gracefully)
|
||||||
|
val sys = context.system // Avoid closing over context
|
||||||
|
Future sequence endpoints.allEndpoints.map {
|
||||||
|
gracefulStop(_, settings.FlushWait)(sys)
|
||||||
|
} map { _.foldLeft(true) { _ && _ } } pipeTo sender
|
||||||
|
// Ignore all other writes
|
||||||
|
context.become(flushing)
|
||||||
|
}
|
||||||
|
|
||||||
|
def flushing: Receive = {
|
||||||
|
case s: Send ⇒ forwardToDeadLetters(s)
|
||||||
|
case InboundAssociation(h) ⇒ h.disassociate()
|
||||||
|
}
|
||||||
|
|
||||||
|
private def forwardToDeadLetters(s: Send): Unit = {
|
||||||
|
val sender = s.senderOption match {
|
||||||
|
case Some(sender) ⇒ sender
|
||||||
|
case None ⇒ Actor.noSender
|
||||||
|
}
|
||||||
|
extendedSystem.deadLetters.tell(s.message, sender)
|
||||||
}
|
}
|
||||||
|
|
||||||
private def listens: Future[Seq[(Transport, Address, Promise[AssociationEventListener])]] = {
|
private def listens: Future[Seq[(Transport, Address, Promise[AssociationEventListener])]] = {
|
||||||
|
|
|
||||||
Loading…
Add table
Add a link
Reference in a new issue