2014-11-28 10:41:57 +01:00
|
|
|
/**
|
|
|
|
|
* Copyright (C) 2009-2014 Typesafe Inc. <http://www.typesafe.com>
|
|
|
|
|
*/
|
2014-12-16 11:48:42 +01:00
|
|
|
package akka.stream.scaladsl
|
2014-11-28 10:41:57 +01:00
|
|
|
|
|
|
|
|
import java.net.{ InetSocketAddress, URLEncoder }
|
2014-11-27 22:57:10 +01:00
|
|
|
import scala.collection.immutable
|
|
|
|
|
import scala.concurrent.{ Promise, ExecutionContext, Future }
|
2014-12-16 11:48:42 +01:00
|
|
|
import scala.concurrent.duration.Duration
|
|
|
|
|
import scala.util.{ Failure, Success }
|
|
|
|
|
import scala.util.control.NoStackTrace
|
|
|
|
|
import akka.actor.Actor
|
|
|
|
|
import akka.actor.ActorRef
|
|
|
|
|
import akka.actor.ActorSystem
|
|
|
|
|
import akka.actor.ExtendedActorSystem
|
|
|
|
|
import akka.actor.ExtensionId
|
|
|
|
|
import akka.actor.ExtensionIdProvider
|
|
|
|
|
import akka.actor.Props
|
2014-11-27 22:57:10 +01:00
|
|
|
import akka.io.Inet.SocketOption
|
|
|
|
|
import akka.io.Tcp
|
2015-01-27 18:29:20 +01:00
|
|
|
import akka.stream.FlowMaterializer
|
|
|
|
|
import akka.stream.ActorFlowMaterializer
|
2014-11-27 22:57:10 +01:00
|
|
|
import akka.stream.impl._
|
2014-12-16 11:48:42 +01:00
|
|
|
import akka.stream.scaladsl._
|
|
|
|
|
import akka.util.ByteString
|
|
|
|
|
import org.reactivestreams.{ Processor, Subscriber, Subscription }
|
|
|
|
|
import akka.stream.impl.io.TcpStreamActor
|
|
|
|
|
import akka.stream.impl.io.TcpListenStreamActor
|
|
|
|
|
import akka.stream.impl.io.DelayedInitProcessor
|
|
|
|
|
import akka.stream.impl.io.StreamTcpManager
|
2014-11-28 10:41:57 +01:00
|
|
|
|
2014-12-16 11:48:42 +01:00
|
|
|
object StreamTcp extends ExtensionId[StreamTcp] with ExtensionIdProvider {
|
2014-11-28 10:41:57 +01:00
|
|
|
|
|
|
|
|
/**
|
2014-11-27 22:57:10 +01:00
|
|
|
* Represents a prospective TCP server binding.
|
2014-11-28 10:41:57 +01:00
|
|
|
*/
|
2014-11-27 22:57:10 +01:00
|
|
|
trait ServerBinding {
|
|
|
|
|
/**
|
|
|
|
|
* The local address of the endpoint bound by the materialization of the `connections` [[Source]]
|
|
|
|
|
* whose [[MaterializedMap]] is passed as parameter.
|
|
|
|
|
*/
|
|
|
|
|
def localAddress(materializedMap: MaterializedMap): Future[InetSocketAddress]
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* The stream of accepted incoming connections.
|
|
|
|
|
* Can be materialized several times but only one subscription can be "live" at one time, i.e.
|
|
|
|
|
* subsequent materializations will reject subscriptions with an [[BindFailedException]] if the previous
|
|
|
|
|
* materialization still has an uncancelled subscription.
|
|
|
|
|
* Cancelling the subscription to a materialization of this source will cause the listening port to be unbound.
|
|
|
|
|
*/
|
|
|
|
|
def connections: Source[IncomingConnection]
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Asynchronously triggers the unbinding of the port that was bound by the materialization of the `connections`
|
|
|
|
|
* [[Source]] whose [[MaterializedMap]] is passed as parameter.
|
|
|
|
|
*
|
2014-12-16 11:48:42 +01:00
|
|
|
* The produced [[scala.concurrent.Future]] is fulfilled when the unbinding has been completed.
|
2014-11-27 22:57:10 +01:00
|
|
|
*/
|
|
|
|
|
def unbind(materializedMap: MaterializedMap): Future[Unit]
|
|
|
|
|
}
|
2014-11-28 10:41:57 +01:00
|
|
|
|
|
|
|
|
/**
|
2014-11-27 22:57:10 +01:00
|
|
|
* Represents an accepted incoming TCP connection.
|
2014-11-28 10:41:57 +01:00
|
|
|
*/
|
2014-11-27 22:57:10 +01:00
|
|
|
trait IncomingConnection {
|
|
|
|
|
/**
|
|
|
|
|
* The local address this connection is bound to.
|
|
|
|
|
*/
|
|
|
|
|
def localAddress: InetSocketAddress
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* The remote address this connection is bound to.
|
|
|
|
|
*/
|
|
|
|
|
def remoteAddress: InetSocketAddress
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Handles the connection using the given flow, which is materialized exactly once and the respective
|
|
|
|
|
* [[MaterializedMap]] returned.
|
|
|
|
|
*
|
|
|
|
|
* Convenience shortcut for: `flow.join(handler).run()`.
|
|
|
|
|
*/
|
|
|
|
|
def handleWith(handler: Flow[ByteString, ByteString])(implicit materializer: FlowMaterializer): MaterializedMap
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* A flow representing the client on the other side of the connection.
|
|
|
|
|
* This flow can be materialized only once.
|
|
|
|
|
*/
|
|
|
|
|
def flow: Flow[ByteString, ByteString]
|
|
|
|
|
}
|
2014-11-28 10:41:57 +01:00
|
|
|
|
|
|
|
|
/**
|
2014-11-27 22:57:10 +01:00
|
|
|
* Represents a prospective outgoing TCP connection.
|
2014-11-28 10:41:57 +01:00
|
|
|
*/
|
2014-12-16 11:48:42 +01:00
|
|
|
trait OutgoingConnection {
|
2014-11-27 22:57:10 +01:00
|
|
|
/**
|
|
|
|
|
* The remote address this connection is or will be bound to.
|
|
|
|
|
*/
|
|
|
|
|
def remoteAddress: InetSocketAddress
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* The local address of the endpoint bound by the materialization of the connection materialization
|
|
|
|
|
* whose [[MaterializedMap]] is passed as parameter.
|
|
|
|
|
*/
|
|
|
|
|
def localAddress(mMap: MaterializedMap): Future[InetSocketAddress]
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* Handles the connection using the given flow.
|
|
|
|
|
* This method can be called several times, every call will materialize the given flow exactly once thereby
|
|
|
|
|
* triggering a new connection attempt to the `remoteAddress`.
|
|
|
|
|
* If the connection cannot be established the materialized stream will immediately be terminated
|
2014-12-16 11:48:42 +01:00
|
|
|
* with a [[akka.stream.StreamTcpException]].
|
2014-11-27 22:57:10 +01:00
|
|
|
*
|
|
|
|
|
* Convenience shortcut for: `flow.join(handler).run()`.
|
|
|
|
|
*/
|
|
|
|
|
def handleWith(handler: Flow[ByteString, ByteString])(implicit materializer: FlowMaterializer): MaterializedMap
|
|
|
|
|
|
|
|
|
|
/**
|
|
|
|
|
* A flow representing the server on the other side of the connection.
|
|
|
|
|
* This flow can be materialized several times, every materialization will open a new connection to the
|
|
|
|
|
* `remoteAddress`. If the connection cannot be established the materialized stream will immediately be terminated
|
2014-12-16 11:48:42 +01:00
|
|
|
* with a [[akka.stream.StreamTcpException]].
|
2014-11-27 22:57:10 +01:00
|
|
|
*/
|
|
|
|
|
def flow: Flow[ByteString, ByteString]
|
|
|
|
|
}
|
|
|
|
|
|
2014-12-16 11:48:42 +01:00
|
|
|
/**
|
|
|
|
|
* INTERNAL API
|
|
|
|
|
*/
|
|
|
|
|
private[akka] class PreMaterializedOutgoingKey extends Key[Future[InetSocketAddress]] {
|
|
|
|
|
override def materialize(map: MaterializedMap) =
|
|
|
|
|
throw new IllegalStateException("This key has already been materialized by the TCP Processor")
|
|
|
|
|
}
|
2014-11-27 22:57:10 +01:00
|
|
|
|
2014-12-16 11:48:42 +01:00
|
|
|
def apply()(implicit system: ActorSystem): StreamTcp = super.apply(system)
|
2014-11-27 22:57:10 +01:00
|
|
|
|
2014-12-16 11:48:42 +01:00
|
|
|
override def get(system: ActorSystem): StreamTcp = super.get(system)
|
2014-11-27 22:57:10 +01:00
|
|
|
|
|
|
|
|
def lookup() = StreamTcp
|
|
|
|
|
|
2014-12-16 11:48:42 +01:00
|
|
|
def createExtension(system: ExtendedActorSystem): StreamTcp = new StreamTcp(system)
|
2014-11-27 22:57:10 +01:00
|
|
|
}
|
|
|
|
|
|
2014-12-16 11:48:42 +01:00
|
|
|
class StreamTcp(system: ExtendedActorSystem) extends akka.actor.Extension {
|
2014-11-27 22:57:10 +01:00
|
|
|
import StreamTcp._
|
|
|
|
|
import system.dispatcher
|
|
|
|
|
|
|
|
|
|
private val manager: ActorRef = system.systemActorOf(Props[StreamTcpManager], name = "IO-TCP-STREAM")
|
2014-11-28 10:41:57 +01:00
|
|
|
|
|
|
|
|
/**
|
2014-12-16 11:48:42 +01:00
|
|
|
* Creates a [[StreamTcp.ServerBinding]] instance which represents a prospective TCP server binding on the given `endpoint`.
|
2014-11-28 10:41:57 +01:00
|
|
|
*/
|
2014-11-27 22:57:10 +01:00
|
|
|
def bind(endpoint: InetSocketAddress,
|
|
|
|
|
backlog: Int = 100,
|
|
|
|
|
options: immutable.Traversable[SocketOption] = Nil,
|
|
|
|
|
idleTimeout: Duration = Duration.Inf): ServerBinding = {
|
2014-12-02 16:38:14 +01:00
|
|
|
val connectionSource = new KeyedActorFlowSource[IncomingConnection, (Future[InetSocketAddress], Future[() ⇒ Future[Unit]])] {
|
2014-11-27 22:57:10 +01:00
|
|
|
override def attach(flowSubscriber: Subscriber[IncomingConnection],
|
2015-01-27 18:29:20 +01:00
|
|
|
materializer: ActorFlowMaterializer,
|
2014-11-27 22:57:10 +01:00
|
|
|
flowName: String): MaterializedType = {
|
|
|
|
|
val localAddressPromise = Promise[InetSocketAddress]()
|
|
|
|
|
val unbindPromise = Promise[() ⇒ Future[Unit]]()
|
|
|
|
|
manager ! StreamTcpManager.Bind(localAddressPromise, unbindPromise, flowSubscriber, endpoint, backlog, options,
|
|
|
|
|
idleTimeout)
|
|
|
|
|
localAddressPromise.future -> unbindPromise.future
|
2014-11-28 10:41:57 +01:00
|
|
|
}
|
2014-11-27 22:57:10 +01:00
|
|
|
}
|
|
|
|
|
new ServerBinding {
|
|
|
|
|
def localAddress(mm: MaterializedMap) = mm.get(connectionSource)._1
|
|
|
|
|
def connections = connectionSource
|
|
|
|
|
def unbind(mm: MaterializedMap): Future[Unit] = mm.get(connectionSource)._2.flatMap(_())
|
|
|
|
|
}
|
2014-11-28 10:41:57 +01:00
|
|
|
}
|
|
|
|
|
|
|
|
|
|
/**
|
2014-12-16 11:48:42 +01:00
|
|
|
* Creates an [[StreamTcp.OutgoingConnection]] instance representing a prospective TCP client connection to the given endpoint.
|
2014-11-28 10:41:57 +01:00
|
|
|
*/
|
2014-11-27 22:57:10 +01:00
|
|
|
def outgoingConnection(remoteAddress: InetSocketAddress,
|
|
|
|
|
localAddress: Option[InetSocketAddress] = None,
|
|
|
|
|
options: immutable.Traversable[SocketOption] = Nil,
|
|
|
|
|
connectTimeout: Duration = Duration.Inf,
|
|
|
|
|
idleTimeout: Duration = Duration.Inf): OutgoingConnection = {
|
|
|
|
|
val remoteAddr = remoteAddress
|
|
|
|
|
val key = new PreMaterializedOutgoingKey()
|
|
|
|
|
val stream = Pipe(key) { () ⇒
|
|
|
|
|
val processorPromise = Promise[Processor[ByteString, ByteString]]()
|
|
|
|
|
val localAddressPromise = Promise[InetSocketAddress]()
|
|
|
|
|
manager ! StreamTcpManager.Connect(processorPromise, localAddressPromise, remoteAddress, localAddress, options,
|
|
|
|
|
connectTimeout, idleTimeout)
|
|
|
|
|
(new DelayedInitProcessor[ByteString, ByteString](processorPromise.future), localAddressPromise.future)
|
|
|
|
|
}
|
|
|
|
|
new OutgoingConnection {
|
|
|
|
|
def remoteAddress = remoteAddr
|
|
|
|
|
def localAddress(mm: MaterializedMap) = mm.get(key)
|
|
|
|
|
def flow = stream
|
|
|
|
|
def handleWith(handler: Flow[ByteString, ByteString])(implicit fm: FlowMaterializer) =
|
|
|
|
|
flow.join(handler).run()
|
|
|
|
|
}
|
|
|
|
|
}
|
|
|
|
|
}
|
2014-11-28 10:41:57 +01:00
|
|
|
|