|
|
@@ -41,6 +41,7 @@ import (
|
|
|
"net"
|
|
|
"strings"
|
|
|
"sync"
|
|
|
+ "sync/atomic"
|
|
|
"time"
|
|
|
|
|
|
"golang.org/x/net/context"
|
|
|
@@ -49,18 +50,23 @@ import (
|
|
|
"google.golang.org/grpc/codes"
|
|
|
"google.golang.org/grpc/credentials"
|
|
|
"google.golang.org/grpc/grpclog"
|
|
|
+ "google.golang.org/grpc/keepalive"
|
|
|
"google.golang.org/grpc/metadata"
|
|
|
"google.golang.org/grpc/peer"
|
|
|
+ "google.golang.org/grpc/stats"
|
|
|
)
|
|
|
|
|
|
// http2Client implements the ClientTransport interface with HTTP2.
|
|
|
type http2Client struct {
|
|
|
- target string // server name/addr
|
|
|
- userAgent string
|
|
|
- md interface{}
|
|
|
- conn net.Conn // underlying communication channel
|
|
|
- authInfo credentials.AuthInfo // auth info about the connection
|
|
|
- nextID uint32 // the next stream ID to be used
|
|
|
+ ctx context.Context
|
|
|
+ target string // server name/addr
|
|
|
+ userAgent string
|
|
|
+ md interface{}
|
|
|
+ conn net.Conn // underlying communication channel
|
|
|
+ remoteAddr net.Addr
|
|
|
+ localAddr net.Addr
|
|
|
+ authInfo credentials.AuthInfo // auth info about the connection
|
|
|
+ nextID uint32 // the next stream ID to be used
|
|
|
|
|
|
// writableChan synchronizes write access to the transport.
|
|
|
// A writer acquires the write lock by sending a value on writableChan
|
|
|
@@ -76,6 +82,8 @@ type http2Client struct {
|
|
|
// goAway is closed to notify the upper layer (i.e., addrConn.transportMonitor)
|
|
|
// that the server sent GoAway on this transport.
|
|
|
goAway chan struct{}
|
|
|
+ // awakenKeepalive is used to wake up keepalive when after it has gone dormant.
|
|
|
+ awakenKeepalive chan struct{}
|
|
|
|
|
|
framer *framer
|
|
|
hBuf *bytes.Buffer // the buffer for HPACK encoding
|
|
|
@@ -95,6 +103,13 @@ type http2Client struct {
|
|
|
|
|
|
creds []credentials.PerRPCCredentials
|
|
|
|
|
|
+ // Boolean to keep track of reading activity on transport.
|
|
|
+ // 1 is true and 0 is false.
|
|
|
+ activity uint32 // Accessed atomically.
|
|
|
+ kp keepalive.ClientParameters
|
|
|
+
|
|
|
+ statsHandler stats.Handler
|
|
|
+
|
|
|
mu sync.Mutex // guard the following variables
|
|
|
state transportState // the state of underlying connection
|
|
|
activeStreams map[uint32]*Stream
|
|
|
@@ -150,6 +165,9 @@ func newHTTP2Client(ctx context.Context, addr TargetInfo, opts ConnectOptions) (
|
|
|
scheme := "http"
|
|
|
conn, err := dial(ctx, opts.Dialer, addr.Addr)
|
|
|
if err != nil {
|
|
|
+ if opts.FailOnNonTempDialError {
|
|
|
+ return nil, connectionErrorf(isTemporary(err), err, "transport: %v", err)
|
|
|
+ }
|
|
|
return nil, connectionErrorf(true, err, "transport: %v", err)
|
|
|
}
|
|
|
// Any further errors will close the underlying connection
|
|
|
@@ -169,23 +187,31 @@ func newHTTP2Client(ctx context.Context, addr TargetInfo, opts ConnectOptions) (
|
|
|
return nil, connectionErrorf(temp, err, "transport: %v", err)
|
|
|
}
|
|
|
}
|
|
|
- ua := primaryUA
|
|
|
- if opts.UserAgent != "" {
|
|
|
- ua = opts.UserAgent + " " + ua
|
|
|
+ kp := opts.KeepaliveParams
|
|
|
+ // Validate keepalive parameters.
|
|
|
+ if kp.Time == 0 {
|
|
|
+ kp.Time = defaultKeepaliveTime
|
|
|
+ }
|
|
|
+ if kp.Timeout == 0 {
|
|
|
+ kp.Timeout = defaultKeepaliveTimeout
|
|
|
}
|
|
|
var buf bytes.Buffer
|
|
|
t := &http2Client{
|
|
|
- target: addr.Addr,
|
|
|
- userAgent: ua,
|
|
|
- md: addr.Metadata,
|
|
|
- conn: conn,
|
|
|
- authInfo: authInfo,
|
|
|
+ ctx: ctx,
|
|
|
+ target: addr.Addr,
|
|
|
+ userAgent: opts.UserAgent,
|
|
|
+ md: addr.Metadata,
|
|
|
+ conn: conn,
|
|
|
+ remoteAddr: conn.RemoteAddr(),
|
|
|
+ localAddr: conn.LocalAddr(),
|
|
|
+ authInfo: authInfo,
|
|
|
// The client initiated stream id is odd starting from 1.
|
|
|
nextID: 1,
|
|
|
writableChan: make(chan int, 1),
|
|
|
shutdownChan: make(chan struct{}),
|
|
|
errorChan: make(chan struct{}),
|
|
|
goAway: make(chan struct{}),
|
|
|
+ awakenKeepalive: make(chan struct{}, 1),
|
|
|
framer: newFramer(conn),
|
|
|
hBuf: &buf,
|
|
|
hEnc: hpack.NewEncoder(&buf),
|
|
|
@@ -196,8 +222,24 @@ func newHTTP2Client(ctx context.Context, addr TargetInfo, opts ConnectOptions) (
|
|
|
state: reachable,
|
|
|
activeStreams: make(map[uint32]*Stream),
|
|
|
creds: opts.PerRPCCredentials,
|
|
|
- maxStreams: math.MaxInt32,
|
|
|
+ maxStreams: defaultMaxStreamsClient,
|
|
|
+ streamsQuota: newQuotaPool(defaultMaxStreamsClient),
|
|
|
streamSendQuota: defaultWindowSize,
|
|
|
+ kp: kp,
|
|
|
+ statsHandler: opts.StatsHandler,
|
|
|
+ }
|
|
|
+ // Make sure awakenKeepalive can't be written upon.
|
|
|
+ // keepalive routine will make it writable, if need be.
|
|
|
+ t.awakenKeepalive <- struct{}{}
|
|
|
+ if t.statsHandler != nil {
|
|
|
+ t.ctx = t.statsHandler.TagConn(t.ctx, &stats.ConnTagInfo{
|
|
|
+ RemoteAddr: t.remoteAddr,
|
|
|
+ LocalAddr: t.localAddr,
|
|
|
+ })
|
|
|
+ connBegin := &stats.ConnBegin{
|
|
|
+ Client: true,
|
|
|
+ }
|
|
|
+ t.statsHandler.HandleConn(t.ctx, connBegin)
|
|
|
}
|
|
|
// Start the reader goroutine for incoming message. Each transport has
|
|
|
// a dedicated goroutine which reads HTTP2 frame from network. Then it
|
|
|
@@ -233,6 +275,9 @@ func newHTTP2Client(ctx context.Context, addr TargetInfo, opts ConnectOptions) (
|
|
|
}
|
|
|
}
|
|
|
go t.controller()
|
|
|
+ if t.kp.Time != infinity {
|
|
|
+ go t.keepalive()
|
|
|
+ }
|
|
|
t.writableChan <- 0
|
|
|
return t, nil
|
|
|
}
|
|
|
@@ -270,12 +315,13 @@ func (t *http2Client) newStream(ctx context.Context, callHdr *CallHdr) *Stream {
|
|
|
// streams.
|
|
|
func (t *http2Client) NewStream(ctx context.Context, callHdr *CallHdr) (_ *Stream, err error) {
|
|
|
pr := &peer.Peer{
|
|
|
- Addr: t.conn.RemoteAddr(),
|
|
|
+ Addr: t.remoteAddr,
|
|
|
}
|
|
|
// Attach Auth info if there is any.
|
|
|
if t.authInfo != nil {
|
|
|
pr.AuthInfo = t.authInfo
|
|
|
}
|
|
|
+ userCtx := ctx
|
|
|
ctx = peer.NewContext(ctx, pr)
|
|
|
authData := make(map[string]string)
|
|
|
for _, c := range t.creds {
|
|
|
@@ -313,21 +359,18 @@ func (t *http2Client) NewStream(ctx context.Context, callHdr *CallHdr) (_ *Strea
|
|
|
t.mu.Unlock()
|
|
|
return nil, ErrConnClosing
|
|
|
}
|
|
|
- checkStreamsQuota := t.streamsQuota != nil
|
|
|
t.mu.Unlock()
|
|
|
- if checkStreamsQuota {
|
|
|
- sq, err := wait(ctx, nil, nil, t.shutdownChan, t.streamsQuota.acquire())
|
|
|
- if err != nil {
|
|
|
- return nil, err
|
|
|
- }
|
|
|
- // Returns the quota balance back.
|
|
|
- if sq > 1 {
|
|
|
- t.streamsQuota.add(sq - 1)
|
|
|
- }
|
|
|
+ sq, err := wait(ctx, nil, nil, t.shutdownChan, t.streamsQuota.acquire())
|
|
|
+ if err != nil {
|
|
|
+ return nil, err
|
|
|
+ }
|
|
|
+ // Returns the quota balance back.
|
|
|
+ if sq > 1 {
|
|
|
+ t.streamsQuota.add(sq - 1)
|
|
|
}
|
|
|
if _, err := wait(ctx, nil, nil, t.shutdownChan, t.writableChan); err != nil {
|
|
|
// Return the quota back now because there is no stream returned to the caller.
|
|
|
- if _, ok := err.(StreamError); ok && checkStreamsQuota {
|
|
|
+ if _, ok := err.(StreamError); ok {
|
|
|
t.streamsQuota.add(1)
|
|
|
}
|
|
|
return nil, err
|
|
|
@@ -335,9 +378,7 @@ func (t *http2Client) NewStream(ctx context.Context, callHdr *CallHdr) (_ *Strea
|
|
|
t.mu.Lock()
|
|
|
if t.state == draining {
|
|
|
t.mu.Unlock()
|
|
|
- if checkStreamsQuota {
|
|
|
- t.streamsQuota.add(1)
|
|
|
- }
|
|
|
+ t.streamsQuota.add(1)
|
|
|
// Need to make t writable again so that the rpc in flight can still proceed.
|
|
|
t.writableChan <- 0
|
|
|
return nil, ErrStreamDrain
|
|
|
@@ -347,18 +388,19 @@ func (t *http2Client) NewStream(ctx context.Context, callHdr *CallHdr) (_ *Strea
|
|
|
return nil, ErrConnClosing
|
|
|
}
|
|
|
s := t.newStream(ctx, callHdr)
|
|
|
+ s.clientStatsCtx = userCtx
|
|
|
t.activeStreams[s.id] = s
|
|
|
-
|
|
|
- // This stream is not counted when applySetings(...) initialize t.streamsQuota.
|
|
|
- // Reset t.streamsQuota to the right value.
|
|
|
- var reset bool
|
|
|
- if !checkStreamsQuota && t.streamsQuota != nil {
|
|
|
- reset = true
|
|
|
+ // If the number of active streams change from 0 to 1, then check if keepalive
|
|
|
+ // has gone dormant. If so, wake it up.
|
|
|
+ if len(t.activeStreams) == 1 {
|
|
|
+ select {
|
|
|
+ case t.awakenKeepalive <- struct{}{}:
|
|
|
+ t.framer.writePing(false, false, [8]byte{})
|
|
|
+ default:
|
|
|
+ }
|
|
|
}
|
|
|
+
|
|
|
t.mu.Unlock()
|
|
|
- if reset {
|
|
|
- t.streamsQuota.reset(-1)
|
|
|
- }
|
|
|
|
|
|
// HPACK encodes various headers. Note that once WriteField(...) is
|
|
|
// called, the corresponding headers/continuation frame has to be sent
|
|
|
@@ -413,6 +455,7 @@ func (t *http2Client) NewStream(ctx context.Context, callHdr *CallHdr) (_ *Strea
|
|
|
}
|
|
|
}
|
|
|
first := true
|
|
|
+ bufLen := t.hBuf.Len()
|
|
|
// Sends the headers in a single batch even when they span multiple frames.
|
|
|
for !endHeaders {
|
|
|
size := t.hBuf.Len()
|
|
|
@@ -447,6 +490,17 @@ func (t *http2Client) NewStream(ctx context.Context, callHdr *CallHdr) (_ *Strea
|
|
|
return nil, connectionErrorf(true, err, "transport: %v", err)
|
|
|
}
|
|
|
}
|
|
|
+ if t.statsHandler != nil {
|
|
|
+ outHeader := &stats.OutHeader{
|
|
|
+ Client: true,
|
|
|
+ WireLength: bufLen,
|
|
|
+ FullMethod: callHdr.Method,
|
|
|
+ RemoteAddr: t.remoteAddr,
|
|
|
+ LocalAddr: t.localAddr,
|
|
|
+ Compression: callHdr.SendCompress,
|
|
|
+ }
|
|
|
+ t.statsHandler.HandleRPC(s.clientStatsCtx, outHeader)
|
|
|
+ }
|
|
|
t.writableChan <- 0
|
|
|
return s, nil
|
|
|
}
|
|
|
@@ -454,15 +508,11 @@ func (t *http2Client) NewStream(ctx context.Context, callHdr *CallHdr) (_ *Strea
|
|
|
// CloseStream clears the footprint of a stream when the stream is not needed any more.
|
|
|
// This must not be executed in reader's goroutine.
|
|
|
func (t *http2Client) CloseStream(s *Stream, err error) {
|
|
|
- var updateStreams bool
|
|
|
t.mu.Lock()
|
|
|
if t.activeStreams == nil {
|
|
|
t.mu.Unlock()
|
|
|
return
|
|
|
}
|
|
|
- if t.streamsQuota != nil {
|
|
|
- updateStreams = true
|
|
|
- }
|
|
|
delete(t.activeStreams, s.id)
|
|
|
if t.state == draining && len(t.activeStreams) == 0 {
|
|
|
// The transport is draining and s is the last live stream on t.
|
|
|
@@ -471,10 +521,27 @@ func (t *http2Client) CloseStream(s *Stream, err error) {
|
|
|
return
|
|
|
}
|
|
|
t.mu.Unlock()
|
|
|
- if updateStreams {
|
|
|
- t.streamsQuota.add(1)
|
|
|
- }
|
|
|
+ // rstStream is true in case the stream is being closed at the client-side
|
|
|
+ // and the server needs to be intimated about it by sending a RST_STREAM
|
|
|
+ // frame.
|
|
|
+ // To make sure this frame is written to the wire before the headers of the
|
|
|
+ // next stream waiting for streamsQuota, we add to streamsQuota pool only
|
|
|
+ // after having acquired the writableChan to send RST_STREAM out (look at
|
|
|
+ // the controller() routine).
|
|
|
+ var rstStream bool
|
|
|
+ var rstError http2.ErrCode
|
|
|
+ defer func() {
|
|
|
+ // In case, the client doesn't have to send RST_STREAM to server
|
|
|
+ // we can safely add back to streamsQuota pool now.
|
|
|
+ if !rstStream {
|
|
|
+ t.streamsQuota.add(1)
|
|
|
+ return
|
|
|
+ }
|
|
|
+ t.controlBuf.put(&resetStream{s.id, rstError})
|
|
|
+ }()
|
|
|
s.mu.Lock()
|
|
|
+ rstStream = s.rstStream
|
|
|
+ rstError = s.rstError
|
|
|
if q := s.fc.resetPendingData(); q > 0 {
|
|
|
if n := t.fc.onRead(q); n > 0 {
|
|
|
t.controlBuf.put(&windowUpdate{0, n})
|
|
|
@@ -490,8 +557,9 @@ func (t *http2Client) CloseStream(s *Stream, err error) {
|
|
|
}
|
|
|
s.state = streamDone
|
|
|
s.mu.Unlock()
|
|
|
- if se, ok := err.(StreamError); ok && se.Code != codes.DeadlineExceeded {
|
|
|
- t.controlBuf.put(&resetStream{s.id, http2.ErrCodeCancel})
|
|
|
+ if _, ok := err.(StreamError); ok {
|
|
|
+ rstStream = true
|
|
|
+ rstError = http2.ErrCodeCancel
|
|
|
}
|
|
|
}
|
|
|
|
|
|
@@ -525,6 +593,12 @@ func (t *http2Client) Close() (err error) {
|
|
|
s.mu.Unlock()
|
|
|
s.write(recvMsg{err: ErrConnClosing})
|
|
|
}
|
|
|
+ if t.statsHandler != nil {
|
|
|
+ connEnd := &stats.ConnEnd{
|
|
|
+ Client: true,
|
|
|
+ }
|
|
|
+ t.statsHandler.HandleConn(t.ctx, connEnd)
|
|
|
+ }
|
|
|
return
|
|
|
}
|
|
|
|
|
|
@@ -582,19 +656,14 @@ func (t *http2Client) Write(s *Stream, data []byte, opts *Options) error {
|
|
|
var p []byte
|
|
|
if r.Len() > 0 {
|
|
|
size := http2MaxFrameLen
|
|
|
- s.sendQuotaPool.add(0)
|
|
|
// Wait until the stream has some quota to send the data.
|
|
|
sq, err := wait(s.ctx, s.done, s.goAway, t.shutdownChan, s.sendQuotaPool.acquire())
|
|
|
if err != nil {
|
|
|
return err
|
|
|
}
|
|
|
- t.sendQuotaPool.add(0)
|
|
|
// Wait until the transport has some quota to send the data.
|
|
|
tq, err := wait(s.ctx, s.done, s.goAway, t.shutdownChan, t.sendQuotaPool.acquire())
|
|
|
if err != nil {
|
|
|
- if _, ok := err.(StreamError); ok || err == io.EOF {
|
|
|
- t.sendQuotaPool.cancel()
|
|
|
- }
|
|
|
return err
|
|
|
}
|
|
|
if sq < size {
|
|
|
@@ -704,7 +773,7 @@ func (t *http2Client) updateWindow(s *Stream, n uint32) {
|
|
|
}
|
|
|
|
|
|
func (t *http2Client) handleData(f *http2.DataFrame) {
|
|
|
- size := len(f.Data())
|
|
|
+ size := f.Header().Length
|
|
|
if err := t.fc.onData(uint32(size)); err != nil {
|
|
|
t.notifyError(connectionErrorf(true, err, "%v", err))
|
|
|
return
|
|
|
@@ -718,6 +787,11 @@ func (t *http2Client) handleData(f *http2.DataFrame) {
|
|
|
return
|
|
|
}
|
|
|
if size > 0 {
|
|
|
+ if f.Header().Flags.Has(http2.FlagDataPadded) {
|
|
|
+ if w := t.fc.onRead(uint32(size) - uint32(len(f.Data()))); w > 0 {
|
|
|
+ t.controlBuf.put(&windowUpdate{0, w})
|
|
|
+ }
|
|
|
+ }
|
|
|
s.mu.Lock()
|
|
|
if s.state == streamDone {
|
|
|
s.mu.Unlock()
|
|
|
@@ -731,19 +805,27 @@ func (t *http2Client) handleData(f *http2.DataFrame) {
|
|
|
s.state = streamDone
|
|
|
s.statusCode = codes.Internal
|
|
|
s.statusDesc = err.Error()
|
|
|
+ s.rstStream = true
|
|
|
+ s.rstError = http2.ErrCodeFlowControl
|
|
|
close(s.done)
|
|
|
s.mu.Unlock()
|
|
|
s.write(recvMsg{err: io.EOF})
|
|
|
- t.controlBuf.put(&resetStream{s.id, http2.ErrCodeFlowControl})
|
|
|
return
|
|
|
}
|
|
|
+ if f.Header().Flags.Has(http2.FlagDataPadded) {
|
|
|
+ if w := s.fc.onRead(uint32(size) - uint32(len(f.Data()))); w > 0 {
|
|
|
+ t.controlBuf.put(&windowUpdate{s.id, w})
|
|
|
+ }
|
|
|
+ }
|
|
|
s.mu.Unlock()
|
|
|
// TODO(bradfitz, zhaoq): A copy is required here because there is no
|
|
|
// guarantee f.Data() is consumed before the arrival of next frame.
|
|
|
// Can this copy be eliminated?
|
|
|
- data := make([]byte, size)
|
|
|
- copy(data, f.Data())
|
|
|
- s.write(recvMsg{data: data})
|
|
|
+ if len(f.Data()) > 0 {
|
|
|
+ data := make([]byte, len(f.Data()))
|
|
|
+ copy(data, f.Data())
|
|
|
+ s.write(recvMsg{data: data})
|
|
|
+ }
|
|
|
}
|
|
|
// The server has closed the stream without sending trailers. Record that
|
|
|
// the read direction is closed, and set the status appropriately.
|
|
|
@@ -874,6 +956,24 @@ func (t *http2Client) operateHeaders(frame *http2.MetaHeadersFrame) {
|
|
|
}
|
|
|
|
|
|
endStream := frame.StreamEnded()
|
|
|
+ var isHeader bool
|
|
|
+ defer func() {
|
|
|
+ if t.statsHandler != nil {
|
|
|
+ if isHeader {
|
|
|
+ inHeader := &stats.InHeader{
|
|
|
+ Client: true,
|
|
|
+ WireLength: int(frame.Header().Length),
|
|
|
+ }
|
|
|
+ t.statsHandler.HandleRPC(s.clientStatsCtx, inHeader)
|
|
|
+ } else {
|
|
|
+ inTrailer := &stats.InTrailer{
|
|
|
+ Client: true,
|
|
|
+ WireLength: int(frame.Header().Length),
|
|
|
+ }
|
|
|
+ t.statsHandler.HandleRPC(s.clientStatsCtx, inTrailer)
|
|
|
+ }
|
|
|
+ }
|
|
|
+ }()
|
|
|
|
|
|
s.mu.Lock()
|
|
|
if !endStream {
|
|
|
@@ -885,6 +985,7 @@ func (t *http2Client) operateHeaders(frame *http2.MetaHeadersFrame) {
|
|
|
}
|
|
|
close(s.headerChan)
|
|
|
s.headerDone = true
|
|
|
+ isHeader = true
|
|
|
}
|
|
|
if !endStream || s.state == streamDone {
|
|
|
s.mu.Unlock()
|
|
|
@@ -925,6 +1026,7 @@ func (t *http2Client) reader() {
|
|
|
t.notifyError(err)
|
|
|
return
|
|
|
}
|
|
|
+ atomic.CompareAndSwapUint32(&t.activity, 0, 1)
|
|
|
sf, ok := frame.(*http2.SettingsFrame)
|
|
|
if !ok {
|
|
|
t.notifyError(err)
|
|
|
@@ -935,6 +1037,7 @@ func (t *http2Client) reader() {
|
|
|
// loop to keep reading incoming messages on this transport.
|
|
|
for {
|
|
|
frame, err := t.framer.readFrame()
|
|
|
+ atomic.CompareAndSwapUint32(&t.activity, 0, 1)
|
|
|
if err != nil {
|
|
|
// Abort an active stream if the http2.Framer returns a
|
|
|
// http2.StreamError. This can happen only if the server's response
|
|
|
@@ -986,21 +1089,15 @@ func (t *http2Client) applySettings(ss []http2.Setting) {
|
|
|
s.Val = math.MaxInt32
|
|
|
}
|
|
|
t.mu.Lock()
|
|
|
- reset := t.streamsQuota != nil
|
|
|
- if !reset {
|
|
|
- t.streamsQuota = newQuotaPool(int(s.Val) - len(t.activeStreams))
|
|
|
- }
|
|
|
ms := t.maxStreams
|
|
|
t.maxStreams = int(s.Val)
|
|
|
t.mu.Unlock()
|
|
|
- if reset {
|
|
|
- t.streamsQuota.reset(int(s.Val) - ms)
|
|
|
- }
|
|
|
+ t.streamsQuota.add(int(s.Val) - ms)
|
|
|
case http2.SettingInitialWindowSize:
|
|
|
t.mu.Lock()
|
|
|
for _, stream := range t.activeStreams {
|
|
|
// Adjust the sending quota for each stream.
|
|
|
- stream.sendQuotaPool.reset(int(s.Val - t.streamSendQuota))
|
|
|
+ stream.sendQuotaPool.add(int(s.Val - t.streamSendQuota))
|
|
|
}
|
|
|
t.streamSendQuota = s.Val
|
|
|
t.mu.Unlock()
|
|
|
@@ -1028,6 +1125,12 @@ func (t *http2Client) controller() {
|
|
|
t.framer.writeSettings(true, i.ss...)
|
|
|
}
|
|
|
case *resetStream:
|
|
|
+ // If the server needs to be to intimated about stream closing,
|
|
|
+ // then we need to make sure the RST_STREAM frame is written to
|
|
|
+ // the wire before the headers of the next stream waiting on
|
|
|
+ // streamQuota. We ensure this by adding to the streamsQuota pool
|
|
|
+ // only after having acquired the writableChan to send RST_STREAM.
|
|
|
+ t.streamsQuota.add(1)
|
|
|
t.framer.writeRSTStream(true, i.streamID, i.code)
|
|
|
case *flushIO:
|
|
|
t.framer.flushWrite()
|
|
|
@@ -1047,6 +1150,61 @@ func (t *http2Client) controller() {
|
|
|
}
|
|
|
}
|
|
|
|
|
|
+// keepalive running in a separate goroutune makes sure the connection is alive by sending pings.
|
|
|
+func (t *http2Client) keepalive() {
|
|
|
+ p := &ping{data: [8]byte{}}
|
|
|
+ timer := time.NewTimer(t.kp.Time)
|
|
|
+ for {
|
|
|
+ select {
|
|
|
+ case <-timer.C:
|
|
|
+ if atomic.CompareAndSwapUint32(&t.activity, 1, 0) {
|
|
|
+ timer.Reset(t.kp.Time)
|
|
|
+ continue
|
|
|
+ }
|
|
|
+ // Check if keepalive should go dormant.
|
|
|
+ t.mu.Lock()
|
|
|
+ if len(t.activeStreams) < 1 && !t.kp.PermitWithoutStream {
|
|
|
+ // Make awakenKeepalive writable.
|
|
|
+ <-t.awakenKeepalive
|
|
|
+ t.mu.Unlock()
|
|
|
+ select {
|
|
|
+ case <-t.awakenKeepalive:
|
|
|
+ // If the control gets here a ping has been sent
|
|
|
+ // need to reset the timer with keepalive.Timeout.
|
|
|
+ case <-t.shutdownChan:
|
|
|
+ return
|
|
|
+ }
|
|
|
+ } else {
|
|
|
+ t.mu.Unlock()
|
|
|
+ // Send ping.
|
|
|
+ t.controlBuf.put(p)
|
|
|
+ }
|
|
|
+
|
|
|
+ // By the time control gets here a ping has been sent one way or the other.
|
|
|
+ timer.Reset(t.kp.Timeout)
|
|
|
+ select {
|
|
|
+ case <-timer.C:
|
|
|
+ if atomic.CompareAndSwapUint32(&t.activity, 1, 0) {
|
|
|
+ timer.Reset(t.kp.Time)
|
|
|
+ continue
|
|
|
+ }
|
|
|
+ t.Close()
|
|
|
+ return
|
|
|
+ case <-t.shutdownChan:
|
|
|
+ if !timer.Stop() {
|
|
|
+ <-timer.C
|
|
|
+ }
|
|
|
+ return
|
|
|
+ }
|
|
|
+ case <-t.shutdownChan:
|
|
|
+ if !timer.Stop() {
|
|
|
+ <-timer.C
|
|
|
+ }
|
|
|
+ return
|
|
|
+ }
|
|
|
+ }
|
|
|
+}
|
|
|
+
|
|
|
func (t *http2Client) Error() <-chan struct{} {
|
|
|
return t.errorChan
|
|
|
}
|