=str #16597 initial steps with idleTimeout
This commit is contained in:
parent
99a9c5964e
commit
2c2228c241
12 changed files with 124 additions and 72 deletions
|
|
@ -7,31 +7,26 @@ package akka.http.impl.engine.server
|
|||
import java.net.InetSocketAddress
|
||||
import java.util.Random
|
||||
|
||||
import akka.http.ServerSettings
|
||||
import akka.http.scaladsl.model.ws.Message
|
||||
import akka.stream.io._
|
||||
import org.reactivestreams.{ Subscriber, Publisher }
|
||||
import scala.util.control.NonFatal
|
||||
import akka.util.ByteString
|
||||
import akka.actor.{ ActorRef, Deploy, Props }
|
||||
import akka.event.LoggingAdapter
|
||||
import akka.actor.{ Deploy, ActorRef, Props }
|
||||
import akka.stream._
|
||||
import akka.stream.scaladsl._
|
||||
import akka.stream.stage.PushPullStage
|
||||
import akka.http.impl.engine.parsing._
|
||||
import akka.http.impl.engine.rendering.{ ResponseRenderingOutput, ResponseRenderingContext, HttpResponseRendererFactory }
|
||||
import akka.http.ServerSettings
|
||||
import akka.http.impl.engine.TokenSourceActor
|
||||
import akka.http.impl.engine.parsing.ParserOutput._
|
||||
import akka.http.impl.engine.parsing._
|
||||
import akka.http.impl.engine.rendering.{ HttpResponseRendererFactory, ResponseRenderingContext, ResponseRenderingOutput }
|
||||
import akka.http.impl.engine.ws.Websocket.SwitchToWebsocketToken
|
||||
import akka.http.impl.engine.ws._
|
||||
import akka.http.impl.util._
|
||||
import akka.http.scaladsl.Http
|
||||
import akka.http.scaladsl.model._
|
||||
import akka.http.impl.util._
|
||||
import akka.http.impl.engine.ws._
|
||||
import Websocket.SwitchToWebsocketToken
|
||||
import ParserOutput._
|
||||
import akka.stream.stage.GraphStage
|
||||
import akka.stream.stage.GraphStageLogic
|
||||
import akka.stream.stage.OutHandler
|
||||
import akka.stream.stage.InHandler
|
||||
import akka.http.impl.engine.rendering.ResponseRenderingContext
|
||||
import akka.stream._
|
||||
import akka.stream.io._
|
||||
import akka.stream.scaladsl._
|
||||
import akka.stream.stage._
|
||||
import akka.util.ByteString
|
||||
import org.reactivestreams.{ Publisher, Subscriber }
|
||||
|
||||
import scala.util.control.NonFatal
|
||||
|
||||
/**
|
||||
* INTERNAL API
|
||||
|
|
|
|||
|
|
@ -4,18 +4,19 @@
|
|||
|
||||
package akka.http.impl.util
|
||||
|
||||
import java.io.InputStream
|
||||
import java.util.concurrent.atomic.{ AtomicReference, AtomicBoolean }
|
||||
import akka.stream.impl.StreamLayout.Module
|
||||
import akka.stream.impl.{ SourceModule, SinkModule, PublisherSink }
|
||||
import org.reactivestreams.{ Subscription, Processor, Subscriber, Publisher }
|
||||
import scala.collection.immutable
|
||||
import scala.concurrent.{ Promise, ExecutionContext, Future }
|
||||
import akka.util.ByteString
|
||||
import java.util.concurrent.atomic.{ AtomicBoolean, AtomicReference }
|
||||
|
||||
import akka.http.scaladsl.model.RequestEntity
|
||||
import akka.stream._
|
||||
import akka.stream.impl.StreamLayout.Module
|
||||
import akka.stream.impl.{ PublisherSink, SinkModule, SourceModule }
|
||||
import akka.stream.scaladsl._
|
||||
import akka.stream.stage._
|
||||
import akka.util.ByteString
|
||||
import org.reactivestreams.{ Processor, Publisher, Subscriber, Subscription }
|
||||
|
||||
import scala.collection.immutable
|
||||
import scala.concurrent.{ ExecutionContext, Future, Promise }
|
||||
|
||||
/**
|
||||
* INTERNAL API
|
||||
|
|
@ -305,7 +306,7 @@ private[http] object StreamUtils {
|
|||
}
|
||||
|
||||
/**
|
||||
* Similar to Source.lazyEmpty but doesn't rely on materialization. Can only be used once.
|
||||
* Similar to Source.maybe but doesn't rely on materialization. Can only be used once.
|
||||
*/
|
||||
trait OneTimeValve {
|
||||
def source[T]: Source[T, Unit]
|
||||
|
|
|
|||
|
|
@ -5,9 +5,8 @@
|
|||
package akka.http.scaladsl
|
||||
|
||||
import java.net.InetSocketAddress
|
||||
import java.security.SecureRandom
|
||||
import java.util.concurrent.ConcurrentHashMap
|
||||
import java.util.{ Collection ⇒ JCollection, Random }
|
||||
import java.util.{ Collection ⇒ JCollection }
|
||||
import javax.net.ssl.{ SSLContext, SSLParameters }
|
||||
|
||||
import akka.actor._
|
||||
|
|
@ -15,8 +14,8 @@ import akka.event.LoggingAdapter
|
|||
import akka.http._
|
||||
import akka.http.impl.engine.client._
|
||||
import akka.http.impl.engine.server._
|
||||
import akka.http.impl.util.{ ReadTheDocumentationException, Java6Compat, StreamUtils }
|
||||
import akka.http.impl.engine.ws.WebsocketClientBlueprint
|
||||
import akka.http.impl.util.{ Java6Compat, ReadTheDocumentationException, StreamUtils }
|
||||
import akka.http.scaladsl.model._
|
||||
import akka.http.scaladsl.model.headers.Host
|
||||
import akka.http.scaladsl.model.ws.{ WebsocketUpgradeResponse, WebsocketRequest, Message }
|
||||
|
|
@ -25,7 +24,6 @@ import akka.japi
|
|||
import akka.stream.Materializer
|
||||
import akka.stream.io._
|
||||
import akka.stream.scaladsl._
|
||||
import akka.util.ByteString
|
||||
import com.typesafe.config.Config
|
||||
|
||||
import scala.collection.immutable
|
||||
|
|
|
|||
|
|
@ -10,6 +10,7 @@ import akka.http.impl.engine.ws.ByteStringSinkProbe
|
|||
import akka.stream.io.{ SendBytes, SslTlsOutbound, SessionBytes }
|
||||
|
||||
import scala.concurrent.duration.FiniteDuration
|
||||
import scala.concurrent.duration._
|
||||
|
||||
import akka.actor.ActorSystem
|
||||
import akka.event.NoLogging
|
||||
|
|
@ -32,7 +33,8 @@ abstract class HttpServerTestSetupBase {
|
|||
val requests = TestSubscriber.probe[HttpRequest]
|
||||
val responses = TestPublisher.probe[HttpResponse]()
|
||||
|
||||
def settings = ServerSettings(system).copy(serverHeader = Some(Server(List(ProductVersion("akka-http", "test")))))
|
||||
def settings = ServerSettings(system)
|
||||
.copy(serverHeader = Some(Server(List(ProductVersion("akka-http", "test")))))
|
||||
def remoteAddress: Option[InetSocketAddress] = None
|
||||
|
||||
val (netIn, netOut) = {
|
||||
|
|
@ -68,6 +70,8 @@ abstract class HttpServerTestSetupBase {
|
|||
|
||||
def expectRequest: HttpRequest = requests.requestNext()
|
||||
def expectNoRequest(max: FiniteDuration): Unit = requests.expectNoMsg(max)
|
||||
def expectSubscribe(): Unit = netOut.expectComplete()
|
||||
def expectSubscribeAndNetworkClose(): Unit = netOut.expectSubscriptionAndComplete()
|
||||
def expectNetworkClose(): Unit = netOut.expectComplete()
|
||||
|
||||
def send(data: ByteString): Unit = netIn.sendNext(data)
|
||||
|
|
|
|||
|
|
@ -5,7 +5,7 @@
|
|||
package akka.http.impl.engine.ws
|
||||
|
||||
import akka.actor.ActorSystem
|
||||
import akka.stream.scaladsl.{ Source, Sink }
|
||||
import akka.stream.scaladsl.Sink
|
||||
import akka.stream.testkit.TestSubscriber
|
||||
import akka.util.ByteString
|
||||
|
||||
|
|
@ -23,6 +23,7 @@ trait ByteStringSinkProbe {
|
|||
def expectNoBytes(): Unit
|
||||
def expectNoBytes(timeout: FiniteDuration): Unit
|
||||
|
||||
def expectSubscriptionAndComplete(): Unit
|
||||
def expectComplete(): Unit
|
||||
def expectError(): Throwable
|
||||
def expectError(cause: Throwable): Unit
|
||||
|
|
@ -62,6 +63,7 @@ object ByteStringSinkProbe {
|
|||
def expectUtf8EncodedString(string: String): Unit =
|
||||
expectBytes(ByteString(string, "utf8"))
|
||||
|
||||
def expectSubscriptionAndComplete(): Unit = probe.expectSubscriptionAndComplete()
|
||||
def expectComplete(): Unit = probe.expectComplete()
|
||||
def expectError(): Throwable = probe.expectError()
|
||||
def expectError(cause: Throwable): Unit = probe.expectError(cause)
|
||||
|
|
|
|||
|
|
@ -1,5 +1,5 @@
|
|||
/**
|
||||
* Copyright (C) 2009-2014 Typesafe Inc. <http://www.typesafe.com>
|
||||
* Copyright (C) 2009-2015 Typesafe Inc. <http://www.typesafe.com>
|
||||
*/
|
||||
|
||||
package akka.http.scaladsl
|
||||
|
|
@ -8,6 +8,7 @@ import java.io.{ BufferedReader, BufferedWriter, InputStreamReader, OutputStream
|
|||
import java.net.Socket
|
||||
import akka.http.scaladsl.Http.ServerBinding
|
||||
import akka.http.{ ClientConnectionSettings, ServerSettings }
|
||||
import akka.util.ByteString
|
||||
import com.typesafe.config.{ Config, ConfigFactory }
|
||||
import scala.annotation.tailrec
|
||||
import scala.concurrent.{ Promise, Future, Await }
|
||||
|
|
@ -24,7 +25,7 @@ import akka.http.scaladsl.model._
|
|||
import akka.http.scaladsl.model.headers._
|
||||
import akka.http.impl.util._
|
||||
|
||||
import scala.util.Success
|
||||
import scala.util.{ Failure, Try, Success }
|
||||
|
||||
class ClientServerSpec extends WordSpec with Matchers with BeforeAndAfterAll {
|
||||
val testConf: Config = ConfigFactory.parseString("""
|
||||
|
|
@ -138,6 +139,43 @@ class ClientServerSpec extends WordSpec with Matchers with BeforeAndAfterAll {
|
|||
Await.result(b1.unbind(), 1.second)
|
||||
}
|
||||
|
||||
"close connection with idle client after idleTimeout" in {
|
||||
val (_, hostname, port) = TestUtils.temporaryServerHostnameAndPort()
|
||||
val s = ServerSettings(system)
|
||||
val theIdleTimeout = 300.millis
|
||||
val settings = s.copy(timeouts = s.timeouts.copy(idleTimeout = theIdleTimeout))
|
||||
|
||||
val receivedRequest = Promise[Long]()
|
||||
|
||||
def handle(req: HttpRequest): Future[HttpResponse] = {
|
||||
receivedRequest.complete(Success(System.nanoTime()))
|
||||
Promise().future // never complete the request with a response; 're waiting for the timeout to happen, nothing else
|
||||
}
|
||||
|
||||
val binding = Http().bindAndHandleAsync(handle, hostname, port, settings = settings)
|
||||
val b1 = Await.result(binding, 3.seconds)
|
||||
|
||||
def runIdleRequest(uri: Uri): Future[HttpResponse] = {
|
||||
val itNeverEnds = Chunked.fromData(ContentTypes.`text/plain`, Source.maybe[ByteString])
|
||||
Http().outgoingConnection(hostname, port)
|
||||
.runWith(Source.single(HttpRequest(PUT, uri, entity = itNeverEnds)), Sink.head)
|
||||
._2
|
||||
}
|
||||
|
||||
val clientsResponseFuture = runIdleRequest("/")
|
||||
|
||||
// await for the server to get the request
|
||||
val serverReceivedRequestAtNanos = Await.result(receivedRequest.future, 2.seconds)
|
||||
|
||||
// waiting for the timeout to happen on the client
|
||||
Try(Await.result(clientsResponseFuture, 2.second)).recoverWith {
|
||||
case _: StreamTcpException ⇒ Success(System.nanoTime())
|
||||
case other: Throwable ⇒ Failure(other)
|
||||
}.get
|
||||
val diff = System.nanoTime() - serverReceivedRequestAtNanos
|
||||
diff should be > theIdleTimeout.toNanos
|
||||
}
|
||||
|
||||
"log materialization errors in `bindAndHandle`" which {
|
||||
|
||||
"are triggered in `transform`" in Utils.assertAllStagesStopped {
|
||||
|
|
|
|||
Loading…
Add table
Add a link
Reference in a new issue