2013-01-09 12:21:31 +01:00
|
|
|
/**
|
|
|
|
|
* Copyright (C) 2009-2013 Typesafe Inc. <http://www.typesafe.com>
|
|
|
|
|
*/
|
2012-09-12 11:18:42 +02:00
|
|
|
package akka.remote
|
|
|
|
|
|
|
|
|
|
import akka.actor.SupervisorStrategy._
|
|
|
|
|
import akka.actor._
|
|
|
|
|
import akka.event.{ Logging, LoggingAdapter }
|
2012-12-13 14:27:34 +01:00
|
|
|
import akka.japi.Util.immutableSeq
|
|
|
|
|
import akka.pattern.{ gracefulStop, pipe, ask }
|
|
|
|
|
import akka.remote.EndpointManager._
|
|
|
|
|
import akka.remote.Remoting.TransportSupervisor
|
2012-12-18 12:54:17 +01:00
|
|
|
import akka.remote.transport.Transport.{ ActorAssociationEventListener, AssociationEventListener, InboundAssociation }
|
2012-09-12 11:18:42 +02:00
|
|
|
import akka.remote.transport._
|
|
|
|
|
import akka.util.Timeout
|
2012-12-13 14:27:34 +01:00
|
|
|
import com.typesafe.config.Config
|
|
|
|
|
import java.net.URLEncoder
|
|
|
|
|
import java.util.concurrent.TimeoutException
|
2012-09-12 11:18:42 +02:00
|
|
|
import scala.collection.immutable.{ Seq, HashMap }
|
|
|
|
|
import scala.concurrent.duration._
|
|
|
|
|
import scala.concurrent.{ Promise, Await, Future }
|
|
|
|
|
import scala.util.control.NonFatal
|
2012-12-14 16:09:38 +01:00
|
|
|
import scala.util.{ Failure, Success }
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2013-02-08 13:13:52 +01:00
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
2012-12-12 14:49:38 +01:00
|
|
|
private[remote] object AddressUrlEncoder {
|
|
|
|
|
def apply(address: Address): String = URLEncoder.encode(address.toString, "utf-8")
|
|
|
|
|
}
|
|
|
|
|
|
2013-02-08 13:13:52 +01:00
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
2012-12-11 13:08:36 +01:00
|
|
|
private[remote] case class RARP(provider: RemoteActorRefProvider) extends Extension
|
2013-02-08 13:13:52 +01:00
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
2012-12-11 13:08:36 +01:00
|
|
|
private[remote] object RARP extends ExtensionId[RARP] with ExtensionIdProvider {
|
|
|
|
|
|
|
|
|
|
override def lookup() = RARP
|
|
|
|
|
|
|
|
|
|
override def createExtension(system: ExtendedActorSystem) = RARP(system.provider.asInstanceOf[RemoteActorRefProvider])
|
|
|
|
|
}
|
|
|
|
|
|
2013-02-08 13:13:52 +01:00
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
2012-09-12 11:18:42 +02:00
|
|
|
private[remote] object Remoting {
|
|
|
|
|
|
2012-11-22 13:33:48 +01:00
|
|
|
final val EndpointManagerName = "endpointManager"
|
2012-09-12 11:18:42 +02:00
|
|
|
|
|
|
|
|
def localAddressForRemote(transportMapping: Map[String, Set[(Transport, Address)]], remote: Address): Address = {
|
|
|
|
|
|
|
|
|
|
transportMapping.get(remote.protocol) match {
|
|
|
|
|
case Some(transports) ⇒
|
2012-11-22 13:33:48 +01:00
|
|
|
val responsibleTransports = transports.filter { case (t, _) ⇒ t.isResponsibleFor(remote) }
|
2012-09-12 11:18:42 +02:00
|
|
|
|
|
|
|
|
responsibleTransports.size match {
|
|
|
|
|
case 0 ⇒
|
|
|
|
|
throw new RemoteTransportException(
|
2013-02-08 13:13:52 +01:00
|
|
|
s"No transport is responsible for address: [$remote] although protocol [${remote.protocol}] is available." +
|
2012-09-12 11:18:42 +02:00
|
|
|
" Make sure at least one transport is configured to be responsible for the address.",
|
|
|
|
|
null)
|
|
|
|
|
|
|
|
|
|
case 1 ⇒
|
|
|
|
|
responsibleTransports.head._2
|
|
|
|
|
|
|
|
|
|
case _ ⇒
|
|
|
|
|
throw new RemoteTransportException(
|
2013-01-14 10:04:49 +01:00
|
|
|
s"Multiple transports are available for [$remote]: [${responsibleTransports.mkString(",")}]. " +
|
2012-09-12 11:18:42 +02:00
|
|
|
"Remoting cannot decide which transport to use to reach the remote system. Change your configuration " +
|
|
|
|
|
"so that only one transport is responsible for the address.",
|
|
|
|
|
null)
|
|
|
|
|
}
|
2012-11-23 10:15:19 +01:00
|
|
|
case None ⇒ throw new RemoteTransportException(
|
2013-03-25 08:42:48 +01:00
|
|
|
s"No transport is loaded for protocol: [${remote.protocol}], available protocols: [${transportMapping.keys.mkString(", ")}]", null)
|
2012-09-12 11:18:42 +02:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
2012-11-23 10:15:19 +01:00
|
|
|
case class RegisterTransportActor(props: Props, name: String)
|
|
|
|
|
|
2012-12-11 13:08:36 +01:00
|
|
|
private[Remoting] class TransportSupervisor extends Actor {
|
|
|
|
|
override def supervisorStrategy = OneForOneStrategy() {
|
|
|
|
|
case NonFatal(e) ⇒ Restart
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
def receive = {
|
|
|
|
|
case RegisterTransportActor(props, name) ⇒ sender ! context.actorOf(props, name)
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
2012-09-12 11:18:42 +02:00
|
|
|
}
|
|
|
|
|
|
2013-02-08 13:13:52 +01:00
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
2012-09-12 11:18:42 +02:00
|
|
|
private[remote] class Remoting(_system: ExtendedActorSystem, _provider: RemoteActorRefProvider) extends RemoteTransport(_system, _provider) {
|
|
|
|
|
|
2012-12-12 12:14:43 +01:00
|
|
|
@volatile private var endpointManager: Option[ActorRef] = None
|
2012-11-21 14:18:24 +01:00
|
|
|
@volatile private var transportMapping: Map[String, Set[(Transport, Address)]] = _
|
2012-12-12 12:14:43 +01:00
|
|
|
// This is effectively a write-once variable similar to a lazy val. The reason for not using a lazy val is exception
|
|
|
|
|
// handling.
|
2012-09-12 11:18:42 +02:00
|
|
|
@volatile var addresses: Set[Address] = _
|
2012-12-12 15:04:44 +01:00
|
|
|
// This variable has the same semantics as the addresses variable, in the sense it is written once, and emulates
|
|
|
|
|
// a lazy val
|
|
|
|
|
@volatile var defaultAddress: Address = _
|
2012-11-21 14:18:24 +01:00
|
|
|
|
2013-01-17 16:19:31 +01:00
|
|
|
import provider.remoteSettings._
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2012-12-11 13:08:36 +01:00
|
|
|
val transportSupervisor = system.asInstanceOf[ActorSystemImpl].systemActorOf(Props[TransportSupervisor], "transports")
|
2012-11-23 10:15:19 +01:00
|
|
|
|
2012-09-12 11:18:42 +02:00
|
|
|
override def localAddressForRemote(remote: Address): Address = Remoting.localAddressForRemote(transportMapping, remote)
|
|
|
|
|
|
|
|
|
|
val log: LoggingAdapter = Logging(system.eventStream, "Remoting")
|
2013-01-17 16:19:31 +01:00
|
|
|
val eventPublisher = new EventPublisher(system, log, LogRemoteLifecycleEvents)
|
2012-09-12 11:18:42 +02:00
|
|
|
|
|
|
|
|
private def notifyError(msg: String, cause: Throwable): Unit =
|
|
|
|
|
eventPublisher.notifyListeners(RemotingErrorEvent(new RemoteTransportException(msg, cause)))
|
|
|
|
|
|
2012-12-14 16:09:38 +01:00
|
|
|
override def shutdown(): Future[Unit] = {
|
|
|
|
|
import scala.concurrent.ExecutionContext.Implicits.global
|
2012-12-12 12:14:43 +01:00
|
|
|
endpointManager match {
|
|
|
|
|
case Some(manager) ⇒
|
2013-01-17 16:19:31 +01:00
|
|
|
implicit val timeout = ShutdownTimeout
|
2012-12-14 16:09:38 +01:00
|
|
|
val stopped: Future[Boolean] = (manager ? ShutdownAndFlush).mapTo[Boolean]
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2012-12-14 16:09:38 +01:00
|
|
|
def finalize(): Unit = {
|
2012-12-13 14:27:34 +01:00
|
|
|
eventPublisher.notifyListeners(RemotingShutdownEvent)
|
2012-12-14 16:09:38 +01:00
|
|
|
endpointManager = None
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
stopped.onComplete {
|
|
|
|
|
case Success(flushSuccessful) ⇒
|
|
|
|
|
if (!flushSuccessful)
|
|
|
|
|
log.warning("Shutdown finished, but flushing timed out. Some messages might not have been sent. " +
|
2013-01-17 16:19:31 +01:00
|
|
|
"Increase akka.remote.flush-wait-on-shutdown to a larger value to avoid this.")
|
2012-12-14 16:09:38 +01:00
|
|
|
finalize()
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2012-12-14 16:09:38 +01:00
|
|
|
case Failure(e) ⇒
|
|
|
|
|
notifyError("Failure during shutdown of remoting.", e)
|
|
|
|
|
finalize()
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
stopped map { _ ⇒ () } // RARP needs only type Unit, not a boolean
|
|
|
|
|
case None ⇒
|
|
|
|
|
log.warning("Remoting is not running. Ignoring shutdown attempt.")
|
2012-12-19 15:12:29 +01:00
|
|
|
Future successful (())
|
2012-09-12 11:18:42 +02:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
// Start assumes that it cannot be followed by another start() without having a shutdown() first
|
|
|
|
|
override def start(): Unit = {
|
2012-12-12 12:14:43 +01:00
|
|
|
endpointManager match {
|
|
|
|
|
case None ⇒
|
|
|
|
|
log.info("Starting remoting")
|
|
|
|
|
val manager: ActorRef = system.asInstanceOf[ActorSystemImpl].systemActorOf(
|
|
|
|
|
Props(new EndpointManager(provider.remoteSettings.config, log)), Remoting.EndpointManagerName)
|
|
|
|
|
endpointManager = Some(manager)
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2012-12-12 12:14:43 +01:00
|
|
|
try {
|
2012-12-12 15:04:44 +01:00
|
|
|
val addressesPromise: Promise[Seq[(Transport, Address)]] = Promise()
|
2012-12-12 12:14:43 +01:00
|
|
|
manager ! Listen(addressesPromise)
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2013-01-09 12:21:31 +01:00
|
|
|
val transports: Seq[(Transport, Address)] = Await.result(addressesPromise.future,
|
2013-01-17 16:19:31 +01:00
|
|
|
StartupTimeout.duration)
|
2012-12-12 15:04:44 +01:00
|
|
|
if (transports.isEmpty) throw new RemoteTransportException("No transport drivers were loaded.", null)
|
|
|
|
|
|
2013-01-09 12:21:31 +01:00
|
|
|
transportMapping = transports.groupBy {
|
|
|
|
|
case (transport, _) ⇒ transport.schemeIdentifier
|
|
|
|
|
} map { case (k, v) ⇒ k -> v.toSet }
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2012-12-12 15:04:44 +01:00
|
|
|
defaultAddress = transports.head._2
|
2012-12-12 12:14:43 +01:00
|
|
|
addresses = transports.map { _._2 }.toSet
|
2012-12-07 16:03:04 +01:00
|
|
|
|
2013-01-17 16:19:31 +01:00
|
|
|
log.info("Remoting started; listening on addresses :" + addresses.mkString("[", ", ", "]"))
|
|
|
|
|
|
2012-12-12 12:14:43 +01:00
|
|
|
manager ! StartupFinished
|
|
|
|
|
eventPublisher.notifyListeners(RemotingListenEvent(addresses))
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2012-12-12 12:14:43 +01:00
|
|
|
} catch {
|
2013-01-11 09:55:42 +01:00
|
|
|
case e: TimeoutException ⇒
|
|
|
|
|
notifyError("Startup timed out", e)
|
|
|
|
|
throw e
|
|
|
|
|
case NonFatal(e) ⇒
|
|
|
|
|
notifyError("Startup failed", e)
|
|
|
|
|
throw e
|
2012-12-12 12:14:43 +01:00
|
|
|
}
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2012-12-12 12:14:43 +01:00
|
|
|
case Some(_) ⇒
|
|
|
|
|
log.warning("Remoting was already started. Ignoring start attempt.")
|
2012-09-12 11:18:42 +02:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
2012-12-12 12:14:43 +01:00
|
|
|
override def send(message: Any, senderOption: Option[ActorRef], recipient: RemoteActorRef): Unit = endpointManager match {
|
|
|
|
|
case Some(manager) ⇒ manager.tell(Send(message, senderOption, recipient), sender = senderOption getOrElse Actor.noSender)
|
2013-04-08 17:16:40 +02:00
|
|
|
case None ⇒ throw new RemoteTransportExceptionNoStackTrace("Attempted to send remote message but Remoting is not running.", null)
|
2012-12-12 12:14:43 +01:00
|
|
|
}
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2012-12-12 12:14:43 +01:00
|
|
|
override def managementCommand(cmd: Any): Future[Boolean] = endpointManager match {
|
|
|
|
|
case Some(manager) ⇒
|
2013-01-09 12:21:31 +01:00
|
|
|
import system.dispatcher
|
2013-01-17 16:19:31 +01:00
|
|
|
implicit val timeout = CommandAckTimeout
|
2013-01-09 12:21:31 +01:00
|
|
|
manager ? ManagementCommand(cmd) map { case ManagementCommandAck(status) ⇒ status }
|
2013-04-08 17:16:40 +02:00
|
|
|
case None ⇒ throw new RemoteTransportExceptionNoStackTrace("Attempted to send management command but Remoting is not running.", null)
|
2012-11-23 10:15:19 +01:00
|
|
|
}
|
|
|
|
|
|
2012-09-12 11:18:42 +02:00
|
|
|
// Not used anywhere only to keep compatibility with RemoteTransport interface
|
|
|
|
|
protected def useUntrustedMode: Boolean = provider.remoteSettings.UntrustedMode
|
|
|
|
|
|
|
|
|
|
// Not used anywhere only to keep compatibility with RemoteTransport interface
|
2013-01-17 16:19:31 +01:00
|
|
|
protected def logRemoteLifeCycleEvents: Boolean = LogRemoteLifecycleEvents
|
2012-09-12 11:18:42 +02:00
|
|
|
|
|
|
|
|
}
|
|
|
|
|
|
2013-02-08 13:13:52 +01:00
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
2012-09-12 11:18:42 +02:00
|
|
|
private[remote] object EndpointManager {
|
|
|
|
|
|
2012-12-13 14:27:34 +01:00
|
|
|
// Messages between Remoting and EndpointManager
|
2012-09-12 11:18:42 +02:00
|
|
|
sealed trait RemotingCommand
|
2012-12-12 15:04:44 +01:00
|
|
|
case class Listen(addressesPromise: Promise[Seq[(Transport, Address)]]) extends RemotingCommand
|
2012-11-22 13:33:48 +01:00
|
|
|
case object StartupFinished extends RemotingCommand
|
2012-12-13 14:27:34 +01:00
|
|
|
case object ShutdownAndFlush extends RemotingCommand
|
2012-09-12 11:18:42 +02:00
|
|
|
case class Send(message: Any, senderOption: Option[ActorRef], recipient: RemoteActorRef) extends RemotingCommand {
|
|
|
|
|
override def toString = s"Remote message $senderOption -> $recipient"
|
|
|
|
|
}
|
2013-01-09 12:21:31 +01:00
|
|
|
case class ManagementCommand(cmd: Any) extends RemotingCommand
|
|
|
|
|
case class ManagementCommandAck(status: Boolean)
|
2012-11-23 10:15:19 +01:00
|
|
|
|
2012-12-13 14:27:34 +01:00
|
|
|
// Messages internal to EndpointManager
|
|
|
|
|
case object Prune
|
2012-12-12 15:04:44 +01:00
|
|
|
case class ListensResult(addressesPromise: Promise[Seq[(Transport, Address)]],
|
|
|
|
|
results: Seq[(Transport, Address, Promise[AssociationEventListener])])
|
|
|
|
|
|
2013-01-14 10:04:49 +01:00
|
|
|
sealed trait EndpointPolicy {
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2013-01-14 10:04:49 +01:00
|
|
|
/**
|
|
|
|
|
* Indicates that the policy does not contain an active endpoint, but it is a tombstone of a previous failure
|
|
|
|
|
*/
|
|
|
|
|
def isTombstone: Boolean
|
|
|
|
|
}
|
|
|
|
|
case class Pass(endpoint: ActorRef) extends EndpointPolicy {
|
|
|
|
|
override def isTombstone: Boolean = false
|
|
|
|
|
}
|
|
|
|
|
case class Gated(timeOfRelease: Deadline) extends EndpointPolicy {
|
|
|
|
|
override def isTombstone: Boolean = true
|
|
|
|
|
}
|
|
|
|
|
case class Quarantined(reason: Throwable) extends EndpointPolicy {
|
|
|
|
|
override def isTombstone: Boolean = true
|
|
|
|
|
}
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2013-01-14 10:04:49 +01:00
|
|
|
// Not threadsafe -- only to be used in HeadActor
|
|
|
|
|
class EndpointRegistry {
|
|
|
|
|
private var addressToWritable = HashMap[Address, EndpointPolicy]()
|
|
|
|
|
private var writableToAddress = HashMap[ActorRef, Address]()
|
|
|
|
|
private var addressToReadonly = HashMap[Address, ActorRef]()
|
|
|
|
|
private var readonlyToAddress = HashMap[ActorRef, Address]()
|
|
|
|
|
|
|
|
|
|
def registerWritableEndpoint(address: Address, endpoint: ActorRef): ActorRef = addressToWritable.get(address) match {
|
|
|
|
|
case Some(Pass(e)) ⇒
|
2013-02-08 13:13:52 +01:00
|
|
|
throw new IllegalArgumentException(s"Attempting to overwrite existing endpoint [$e] with [$endpoint]")
|
2013-01-14 10:04:49 +01:00
|
|
|
case _ ⇒
|
|
|
|
|
addressToWritable += address -> Pass(endpoint)
|
|
|
|
|
writableToAddress += endpoint -> address
|
|
|
|
|
endpoint
|
2012-11-21 16:39:04 +01:00
|
|
|
}
|
|
|
|
|
|
2013-01-14 10:04:49 +01:00
|
|
|
def registerReadOnlyEndpoint(address: Address, endpoint: ActorRef): ActorRef = {
|
|
|
|
|
addressToReadonly += address -> endpoint
|
|
|
|
|
readonlyToAddress += endpoint -> address
|
|
|
|
|
endpoint
|
2012-11-22 13:33:48 +01:00
|
|
|
}
|
|
|
|
|
|
2013-01-17 14:00:01 +01:00
|
|
|
def unregisterEndpoint(endpoint: ActorRef): Unit =
|
|
|
|
|
if (isWritable(endpoint)) {
|
|
|
|
|
val address = writableToAddress(endpoint)
|
|
|
|
|
addressToWritable.get(address) match {
|
|
|
|
|
case Some(policy) if policy.isTombstone ⇒ // There is already a tombstone directive, leave it there
|
|
|
|
|
case _ ⇒ addressToWritable -= address
|
|
|
|
|
}
|
|
|
|
|
writableToAddress -= endpoint
|
|
|
|
|
} else if (isReadOnly(endpoint)) {
|
|
|
|
|
addressToReadonly -= readonlyToAddress(endpoint)
|
|
|
|
|
readonlyToAddress -= endpoint
|
2012-09-12 11:18:42 +02:00
|
|
|
}
|
|
|
|
|
|
2013-01-14 10:04:49 +01:00
|
|
|
def writableEndpointWithPolicyFor(address: Address): Option[EndpointPolicy] = addressToWritable.get(address)
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2013-01-14 10:04:49 +01:00
|
|
|
def hasWritableEndpointFor(address: Address): Boolean = writableEndpointWithPolicyFor(address) match {
|
|
|
|
|
case Some(Pass(_)) ⇒ true
|
|
|
|
|
case _ ⇒ false
|
2012-11-21 16:39:04 +01:00
|
|
|
}
|
|
|
|
|
|
2013-01-14 10:04:49 +01:00
|
|
|
def readOnlyEndpointFor(address: Address): Option[ActorRef] = addressToReadonly.get(address)
|
|
|
|
|
|
|
|
|
|
def isWritable(endpoint: ActorRef): Boolean = writableToAddress contains endpoint
|
2012-12-12 12:29:36 +01:00
|
|
|
|
2013-01-14 10:04:49 +01:00
|
|
|
def isReadOnly(endpoint: ActorRef): Boolean = readonlyToAddress contains endpoint
|
|
|
|
|
|
|
|
|
|
def isQuarantined(address: Address): Boolean = writableEndpointWithPolicyFor(address) match {
|
|
|
|
|
case Some(Quarantined(_)) ⇒ true
|
|
|
|
|
case _ ⇒ false
|
2012-09-12 11:18:42 +02:00
|
|
|
}
|
|
|
|
|
|
2013-01-17 14:00:01 +01:00
|
|
|
def markAsFailed(endpoint: ActorRef, timeOfRelease: Deadline): Unit =
|
|
|
|
|
if (isWritable(endpoint)) {
|
|
|
|
|
addressToWritable += writableToAddress(endpoint) -> Gated(timeOfRelease)
|
|
|
|
|
writableToAddress -= endpoint
|
|
|
|
|
} else if (isReadOnly(endpoint)) {
|
|
|
|
|
addressToReadonly -= readonlyToAddress(endpoint)
|
|
|
|
|
readonlyToAddress -= endpoint
|
|
|
|
|
}
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2013-01-14 10:04:49 +01:00
|
|
|
def markAsQuarantined(address: Address, reason: Throwable): Unit = addressToWritable += address -> Quarantined(reason)
|
2012-12-12 12:29:36 +01:00
|
|
|
|
2013-01-14 10:04:49 +01:00
|
|
|
def allEndpoints: collection.Iterable[ActorRef] = writableToAddress.keys ++ readonlyToAddress.keys
|
|
|
|
|
|
|
|
|
|
def pruneGatedEntries(): Unit = {
|
|
|
|
|
addressToWritable = addressToWritable.filter {
|
|
|
|
|
case (_, Gated(timeOfRelease)) ⇒ timeOfRelease.hasTimeLeft
|
|
|
|
|
case _ ⇒ true
|
2012-09-12 11:18:42 +02:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
2013-02-08 13:13:52 +01:00
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
2012-09-12 11:18:42 +02:00
|
|
|
private[remote] class EndpointManager(conf: Config, log: LoggingAdapter) extends Actor {
|
|
|
|
|
|
|
|
|
|
import EndpointManager._
|
|
|
|
|
import context.dispatcher
|
|
|
|
|
|
2013-01-17 16:19:31 +01:00
|
|
|
val settings = new RemoteSettings(conf)
|
2012-09-12 11:18:42 +02:00
|
|
|
val extendedSystem = context.system.asInstanceOf[ExtendedActorSystem]
|
2012-11-22 13:33:48 +01:00
|
|
|
val endpointId: Iterator[Int] = Iterator from 0
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2013-01-17 16:19:31 +01:00
|
|
|
val eventPublisher = new EventPublisher(context.system, log, settings.LogRemoteLifecycleEvents)
|
2012-09-12 11:18:42 +02:00
|
|
|
|
|
|
|
|
// Mapping between addresses and endpoint actors. If passive connections are turned off, incoming connections
|
|
|
|
|
// will be not part of this map!
|
|
|
|
|
val endpoints = new EndpointRegistry
|
|
|
|
|
// Mapping between transports and the local addresses they listen to
|
|
|
|
|
var transportMapping: Map[Address, Transport] = Map()
|
|
|
|
|
|
2013-01-14 10:04:49 +01:00
|
|
|
def retryGateEnabled = settings.RetryGateClosedFor > Duration.Zero
|
|
|
|
|
val pruneInterval: FiniteDuration = if (retryGateEnabled) settings.RetryGateClosedFor * 2 else Duration.Zero
|
2012-11-22 13:33:48 +01:00
|
|
|
val pruneTimerCancellable: Option[Cancellable] = if (retryGateEnabled)
|
2013-01-14 10:04:49 +01:00
|
|
|
Some(context.system.scheduler.schedule(pruneInterval, pruneInterval, self, Prune))
|
2012-09-12 11:18:42 +02:00
|
|
|
else None
|
|
|
|
|
|
2013-03-07 18:08:07 +01:00
|
|
|
override val supervisorStrategy =
|
|
|
|
|
OneForOneStrategy(settings.MaximumRetriesInWindow, settings.RetryWindow, loggingEnabled = false) {
|
|
|
|
|
case InvalidAssociation(localAddress, remoteAddress, _) ⇒
|
|
|
|
|
log.error("Tried to associate with invalid remote address [{}]. " +
|
|
|
|
|
"Address is now quarantined, all messages to this address will be delivered to dead letters.", remoteAddress)
|
|
|
|
|
endpoints.markAsFailed(sender, Deadline.now + settings.UnknownAddressGateClosedFor)
|
2012-09-12 11:18:42 +02:00
|
|
|
Stop
|
2013-03-07 18:08:07 +01:00
|
|
|
|
|
|
|
|
case NonFatal(e) ⇒
|
|
|
|
|
|
|
|
|
|
// logging
|
|
|
|
|
e match {
|
|
|
|
|
case _: EndpointDisassociatedException | _: EndpointAssociationException ⇒ // no logging
|
|
|
|
|
case _ ⇒ log.error(e, e.getMessage)
|
|
|
|
|
}
|
|
|
|
|
|
|
|
|
|
// Retrying immediately if the retry gate is disabled, and it is an endpoint used for writing.
|
|
|
|
|
if (!retryGateEnabled && endpoints.isWritable(sender)) {
|
|
|
|
|
// This strategy keeps all the messages in the stash of the endpoint so restart will transfer the queue
|
|
|
|
|
// to the restarted endpoint -- thus no messages are lost
|
|
|
|
|
Restart
|
|
|
|
|
} else {
|
|
|
|
|
// This strategy throws away all the messages enqueued in the endpoint (in its stash), registers the time of failure,
|
|
|
|
|
// keeps throwing away messages until the retry gate becomes open (time specified in RetryGateClosedFor)
|
|
|
|
|
endpoints.markAsFailed(sender, Deadline.now + settings.RetryGateClosedFor)
|
|
|
|
|
Stop
|
|
|
|
|
}
|
|
|
|
|
}
|
2012-09-12 11:18:42 +02:00
|
|
|
|
|
|
|
|
def receive = {
|
2012-12-12 12:14:43 +01:00
|
|
|
case Listen(addressesPromise) ⇒
|
2012-12-12 15:04:44 +01:00
|
|
|
listens map { ListensResult(addressesPromise, _) } pipeTo self
|
|
|
|
|
case ListensResult(addressesPromise, results) ⇒
|
|
|
|
|
transportMapping = results.groupBy {
|
|
|
|
|
case (_, transportAddress, _) ⇒ transportAddress
|
|
|
|
|
} map {
|
|
|
|
|
case (a, t) if t.size > 1 ⇒
|
|
|
|
|
throw new RemoteTransportException(s"There are more than one transports listening on local address [$a]", null)
|
|
|
|
|
case (a, t) ⇒ a -> t.head._1
|
|
|
|
|
}
|
|
|
|
|
// Register to each transport as listener and collect mapping to addresses
|
|
|
|
|
val transportsAndAddresses = results map {
|
|
|
|
|
case (transport, address, promise) ⇒
|
2012-12-18 12:54:17 +01:00
|
|
|
promise.success(ActorAssociationEventListener(self))
|
2012-12-12 15:04:44 +01:00
|
|
|
transport -> address
|
2012-12-12 12:14:43 +01:00
|
|
|
}
|
2012-12-12 15:04:44 +01:00
|
|
|
addressesPromise.success(transportsAndAddresses)
|
2013-01-09 12:21:31 +01:00
|
|
|
case ManagementCommand(_) ⇒
|
|
|
|
|
sender ! ManagementCommandAck(false)
|
2012-12-13 14:27:34 +01:00
|
|
|
case StartupFinished ⇒
|
|
|
|
|
context.become(accepting)
|
|
|
|
|
case ShutdownAndFlush ⇒
|
|
|
|
|
sender ! true
|
|
|
|
|
context.stop(self) // Nothing to flush at this point
|
2012-09-12 11:18:42 +02:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
val accepting: Receive = {
|
2013-01-09 12:21:31 +01:00
|
|
|
case ManagementCommand(cmd) ⇒
|
|
|
|
|
val allStatuses = transportMapping.values map { transport ⇒
|
|
|
|
|
transport.managementCommand(cmd)
|
|
|
|
|
}
|
|
|
|
|
Future.fold(allStatuses)(true)(_ && _) map ManagementCommandAck pipeTo sender
|
2012-11-23 10:15:19 +01:00
|
|
|
|
2012-09-12 11:18:42 +02:00
|
|
|
case s @ Send(message, senderOption, recipientRef) ⇒
|
|
|
|
|
val recipientAddress = recipientRef.path.address
|
|
|
|
|
|
2013-01-14 10:04:49 +01:00
|
|
|
def createAndRegisterWritingEndpoint(): ActorRef = endpoints.registerWritableEndpoint(recipientAddress, createEndpoint(
|
|
|
|
|
recipientAddress,
|
|
|
|
|
recipientRef.localAddressToUse,
|
|
|
|
|
transportMapping(recipientRef.localAddressToUse),
|
|
|
|
|
settings,
|
|
|
|
|
None))
|
|
|
|
|
|
|
|
|
|
endpoints.writableEndpointWithPolicyFor(recipientAddress) match {
|
|
|
|
|
case Some(Pass(endpoint)) ⇒
|
2012-11-21 16:39:04 +01:00
|
|
|
endpoint ! s
|
2013-01-14 10:04:49 +01:00
|
|
|
case Some(Gated(timeOfRelease)) ⇒
|
|
|
|
|
if (timeOfRelease.isOverdue()) createAndRegisterWritingEndpoint() ! s
|
|
|
|
|
else forwardToDeadLetters(s)
|
|
|
|
|
case Some(Quarantined(_)) ⇒
|
|
|
|
|
forwardToDeadLetters(s)
|
2012-11-21 16:39:04 +01:00
|
|
|
case None ⇒
|
2013-01-14 10:04:49 +01:00
|
|
|
createAndRegisterWritingEndpoint() ! s
|
2012-09-12 11:18:42 +02:00
|
|
|
|
|
|
|
|
}
|
|
|
|
|
|
2013-01-14 10:04:49 +01:00
|
|
|
case InboundAssociation(handle) ⇒ endpoints.readOnlyEndpointFor(handle.remoteAddress) match {
|
2012-11-21 16:39:04 +01:00
|
|
|
case Some(endpoint) ⇒ endpoint ! EndpointWriter.TakeOver(handle)
|
|
|
|
|
case None ⇒
|
2013-01-14 10:04:49 +01:00
|
|
|
if (endpoints.isQuarantined(handle.remoteAddress)) handle.disassociate()
|
|
|
|
|
else {
|
|
|
|
|
eventPublisher.notifyListeners(AssociatedEvent(handle.localAddress, handle.remoteAddress, true))
|
|
|
|
|
val endpoint = createEndpoint(
|
|
|
|
|
handle.remoteAddress,
|
|
|
|
|
handle.localAddress,
|
|
|
|
|
transportMapping(handle.localAddress),
|
|
|
|
|
settings,
|
|
|
|
|
Some(handle))
|
|
|
|
|
if (settings.UsePassiveConnections && !endpoints.hasWritableEndpointFor(handle.remoteAddress))
|
|
|
|
|
endpoints.registerWritableEndpoint(handle.remoteAddress, endpoint)
|
|
|
|
|
else
|
|
|
|
|
endpoints.registerReadOnlyEndpoint(handle.remoteAddress, endpoint)
|
|
|
|
|
}
|
2012-11-21 16:39:04 +01:00
|
|
|
}
|
2012-12-13 14:27:34 +01:00
|
|
|
case Terminated(endpoint) ⇒
|
2013-01-14 10:04:49 +01:00
|
|
|
endpoints.unregisterEndpoint(endpoint)
|
2012-12-13 14:27:34 +01:00
|
|
|
case Prune ⇒
|
2013-01-14 10:04:49 +01:00
|
|
|
endpoints.pruneGatedEntries()
|
2012-12-13 14:27:34 +01:00
|
|
|
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 {
|
2013-03-30 01:03:17 +01:00
|
|
|
gracefulStop(_, settings.FlushWait, EndpointWriter.FlushAndStop)
|
2012-12-13 14:27:34 +01:00
|
|
|
} 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()
|
2012-12-19 15:12:29 +01:00
|
|
|
case Terminated(_) ⇒ // why should we care now?
|
2012-12-13 14:27:34 +01:00
|
|
|
}
|
|
|
|
|
|
2013-02-04 12:41:58 +01:00
|
|
|
private def forwardToDeadLetters(s: Send): Unit = {
|
|
|
|
|
val sender = s.senderOption.getOrElse(extendedSystem.deadLetters)
|
|
|
|
|
extendedSystem.deadLetters.tell(DeadLetter(s.message, sender, s.recipient), sender)
|
|
|
|
|
}
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2012-12-12 15:04:44 +01:00
|
|
|
private def listens: Future[Seq[(Transport, Address, Promise[AssociationEventListener])]] = {
|
2012-12-14 13:45:55 +01:00
|
|
|
/*
|
|
|
|
|
* Constructs chains of adapters on top of each driver as given in configuration. The resulting structure looks
|
|
|
|
|
* like the following:
|
|
|
|
|
* AkkaProtocolTransport <- Adapter <- ... <- Adapter <- Driver
|
|
|
|
|
*
|
|
|
|
|
* The transports variable contains only the heads of each chains (the AkkaProtocolTransport instances).
|
|
|
|
|
*/
|
|
|
|
|
val transports: Seq[AkkaProtocolTransport] = for ((fqn, adapters, config) ← settings.Transports) yield {
|
2012-09-12 11:18:42 +02:00
|
|
|
|
|
|
|
|
val args = Seq(classOf[ExtendedActorSystem] -> context.system, classOf[Config] -> config)
|
|
|
|
|
|
2012-12-14 13:45:55 +01:00
|
|
|
// Loads the driver -- the bottom element of the chain.
|
|
|
|
|
// The chain at this point:
|
|
|
|
|
// Driver
|
2012-11-23 10:15:19 +01:00
|
|
|
val driver = extendedSystem.dynamicAccess
|
2012-09-12 11:18:42 +02:00
|
|
|
.createInstanceFor[Transport](fqn, args).recover({
|
|
|
|
|
|
|
|
|
|
case exception ⇒ throw new IllegalArgumentException(
|
|
|
|
|
(s"Cannot instantiate transport [$fqn]. " +
|
|
|
|
|
"Make sure it extends [akka.remote.transport.Transport] and has constructor with " +
|
|
|
|
|
"[akka.actor.ExtendedActorSystem] and [com.typesafe.config.Config] parameters"), exception)
|
|
|
|
|
|
|
|
|
|
}).get
|
|
|
|
|
|
2012-12-14 13:45:55 +01:00
|
|
|
// Iteratively decorates the bottom level driver with a list of adapters.
|
|
|
|
|
// The chain at this point:
|
|
|
|
|
// Adapter <- ... <- Adapter <- Driver
|
2012-11-23 10:15:19 +01:00
|
|
|
val wrappedTransport =
|
|
|
|
|
adapters.map { TransportAdaptersExtension.get(context.system).getAdapterProvider(_) }.foldLeft(driver) {
|
|
|
|
|
(t: Transport, provider: TransportAdapterProvider) ⇒
|
2012-12-14 13:45:55 +01:00
|
|
|
// The TransportAdapterProvider will wrap the given Transport and returns with a wrapped one
|
2013-02-08 13:13:52 +01:00
|
|
|
provider.create(t, context.system.asInstanceOf[ExtendedActorSystem])
|
2012-11-23 10:15:19 +01:00
|
|
|
}
|
|
|
|
|
|
2012-12-14 13:45:55 +01:00
|
|
|
// Apply AkkaProtocolTransport wrapper to the end of the chain
|
|
|
|
|
// The chain at this point:
|
|
|
|
|
// AkkaProtocolTransport <- Adapter <- ... <- Adapter <- Driver
|
2012-09-12 11:18:42 +02:00
|
|
|
new AkkaProtocolTransport(wrappedTransport, context.system, new AkkaProtocolSettings(conf), AkkaPduProtobufCodec)
|
|
|
|
|
}
|
|
|
|
|
|
2012-12-12 15:04:44 +01:00
|
|
|
// Collect all transports, listen addresses and listener promises in one future
|
|
|
|
|
Future.sequence(transports.map { transport ⇒
|
|
|
|
|
transport.listen map { case (address, listenerPromise) ⇒ (transport, address, listenerPromise) }
|
|
|
|
|
})
|
2012-09-12 11:18:42 +02:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
private def createEndpoint(remoteAddress: Address,
|
|
|
|
|
localAddress: Address,
|
2012-12-12 12:14:43 +01:00
|
|
|
transport: Transport,
|
2013-01-17 16:19:31 +01:00
|
|
|
endpointSettings: RemoteSettings,
|
2012-09-12 11:18:42 +02:00
|
|
|
handleOption: Option[AssociationHandle]): ActorRef = {
|
2012-12-07 16:03:04 +01:00
|
|
|
assert(transportMapping contains localAddress)
|
2012-09-12 11:18:42 +02:00
|
|
|
|
2012-12-07 16:03:04 +01:00
|
|
|
context.watch(context.actorOf(Props(
|
2012-09-12 11:18:42 +02:00
|
|
|
new EndpointWriter(
|
|
|
|
|
handleOption,
|
|
|
|
|
localAddress,
|
|
|
|
|
remoteAddress,
|
2012-12-12 12:14:43 +01:00
|
|
|
transport,
|
|
|
|
|
endpointSettings,
|
2012-09-12 11:18:42 +02:00
|
|
|
AkkaPduProtobufCodec))
|
2013-01-17 16:19:31 +01:00
|
|
|
.withDispatcher("akka.remote.writer-dispatcher"),
|
2012-12-12 14:49:38 +01:00
|
|
|
"endpointWriter-" + AddressUrlEncoder(remoteAddress) + "-" + endpointId.next()))
|
2012-09-12 11:18:42 +02:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
override def postStop(): Unit = {
|
|
|
|
|
pruneTimerCancellable.foreach { _.cancel() }
|
|
|
|
|
transportMapping.values foreach { transport ⇒
|
2012-11-22 13:33:48 +01:00
|
|
|
try transport.shutdown() catch {
|
2012-09-12 11:18:42 +02:00
|
|
|
case NonFatal(e) ⇒
|
2012-12-07 16:03:04 +01:00
|
|
|
log.error(e, s"Unable to shut down the underlying transport: [$transport]")
|
2012-09-12 11:18:42 +02:00
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
|
2012-12-19 15:12:29 +01:00
|
|
|
}
|