package libp2pquic

import (
	
	
	
	
	
	
	
	

	
	ic 
	
	
	
	tpt 
	p2ptls 
	

	logging 
	ma 
	mafmt 
	manet 
	
)

const ListenOrder = 1

var log = logging.Logger("quic-transport")

var ErrHolePunching = errors.New("hole punching attempted; no active dial")

var HolePunchTimeout = 5 * time.Second

// The Transport implements the tpt.Transport interface for QUIC connections.
type transport struct {
	privKey     ic.PrivKey
	localPeer   peer.ID
	identity    *p2ptls.Identity
	connManager *quicreuse.ConnManager
	gater       connmgr.ConnectionGater
	rcmgr       network.ResourceManager

	holePunchingMx sync.Mutex
	holePunching   map[holePunchKey]*activeHolePunch

	rndMx sync.Mutex
	rnd   rand.Rand

	connMx sync.Mutex
	conns  map[quic.Connection]*conn

	listenersMu sync.Mutex
	// map of UDPAddr as string to a virtualListeners
	listeners map[string][]*virtualListener
}

var _ tpt.Transport = &transport{}

type holePunchKey struct {
	addr string
	peer peer.ID
}

type activeHolePunch struct {
	connCh    chan tpt.CapableConn
	fulfilled bool
}

// NewTransport creates a new QUIC transport
func ( ic.PrivKey,  *quicreuse.ConnManager,  pnet.PSK,  connmgr.ConnectionGater,  network.ResourceManager) (tpt.Transport, error) {
	if len() > 0 {
		log.Error("QUIC doesn't support private networks yet.")
		return nil, errors.New("QUIC doesn't support private networks yet")
	}
	,  := peer.IDFromPrivateKey()
	if  != nil {
		return nil, 
	}
	,  := p2ptls.NewIdentity()
	if  != nil {
		return nil, 
	}

	if  == nil {
		 = &network.NullResourceManager{}
	}

	return &transport{
		privKey:      ,
		localPeer:    ,
		identity:     ,
		connManager:  ,
		gater:        ,
		rcmgr:        ,
		conns:        make(map[quic.Connection]*conn),
		holePunching: make(map[holePunchKey]*activeHolePunch),
		rnd:          *rand.New(rand.NewSource(time.Now().UnixNano())),

		listeners: make(map[string][]*virtualListener),
	}, nil
}

func ( *transport) () int {
	return ListenOrder
}

// Dial dials a new QUIC connection
func ( *transport) ( context.Context,  ma.Multiaddr,  peer.ID) ( tpt.CapableConn,  error) {
	if , ,  := network.GetSimultaneousConnect();  && ! {
		return .holePunch(, , )
	}

	,  := .rcmgr.OpenConnection(network.DirOutbound, false, )
	if  != nil {
		log.Debugw("resource manager blocked outgoing connection", "peer", , "addr", , "error", )
		return nil, 
	}

	,  := .dialWithScope(, , , )
	if  != nil {
		.Done()
		return nil, 
	}
	return , nil
}

func ( *transport) ( context.Context,  ma.Multiaddr,  peer.ID,  network.ConnManagementScope) (tpt.CapableConn, error) {
	if  := .SetPeer();  != nil {
		log.Debugw("resource manager blocked outgoing connection for peer", "peer", , "addr", , "error", )
		return nil, 
	}

	,  := .identity.ConfigForPeer()
	 = quicreuse.WithAssociation(, )
	,  := .connManager.DialQUIC(, , , .allowWindowIncrease)
	if  != nil {
		return nil, 
	}

	// Should be ready by this point, don't block.
	var  ic.PubKey
	select {
	case  = <-:
	default:
	}
	if  == nil {
		.CloseWithError(1, "")
		return nil, errors.New("p2p/transport/quic BUG: expected remote pub key to be set")
	}

	,  := quicreuse.ToQuicMultiaddr(.LocalAddr(), .ConnectionState().Version)
	if  != nil {
		.CloseWithError(1, "")
		return nil, 
	}
	 := &conn{
		quicConn:        ,
		transport:       ,
		scope:           ,
		localPeer:       .localPeer,
		localMultiaddr:  ,
		remotePubKey:    ,
		remotePeerID:    ,
		remoteMultiaddr: ,
	}
	if .gater != nil && !.gater.InterceptSecured(network.DirOutbound, , ) {
		.CloseWithError(quic.ApplicationErrorCode(network.ConnGated), "connection gated")
		return nil, fmt.Errorf("secured connection gated")
	}
	.addConn(, )
	return , nil
}

func ( *transport) ( quic.Connection,  *conn) {
	.connMx.Lock()
	.conns[] = 
	.connMx.Unlock()
}

func ( *transport) ( quic.Connection) {
	.connMx.Lock()
	delete(.conns, )
	.connMx.Unlock()
}

func ( *transport) ( context.Context,  ma.Multiaddr,  peer.ID) (tpt.CapableConn, error) {
	, ,  := manet.DialArgs()
	if  != nil {
		return nil, 
	}
	,  := net.ResolveUDPAddr(, )
	if  != nil {
		return nil, 
	}
	,  := .connManager.TransportWithAssociationForDial(, , )
	if  != nil {
		return nil, 
	}
	defer .DecreaseCount()

	,  := context.WithTimeout(, HolePunchTimeout)
	defer ()

	 := holePunchKey{addr: .String(), peer: }
	.holePunchingMx.Lock()
	if ,  := .holePunching[];  {
		.holePunchingMx.Unlock()
		return nil, fmt.Errorf("already punching hole for %s", )
	}
	 := make(chan tpt.CapableConn, 1)
	.holePunching[] = &activeHolePunch{connCh: }
	.holePunchingMx.Unlock()

	var  *time.Timer
	defer func() {
		if  != nil {
			.Stop()
		}
	}()

	 := make([]byte, 64)
	var  error
:
	for  := 0; ; ++ {
		.rndMx.Lock()
		,  := .rnd.Read()
		.rndMx.Unlock()
		if  != nil {
			 = 
			break
		}
		if ,  := .WriteTo(, );  != nil {
			 = 
			break
		}

		 := 10 * ( + 1) * ( + 1) // in ms
		if  > 200 {
			 = 200
		}
		 := 10*time.Millisecond + time.Duration(rand.Intn())*time.Millisecond
		if  == nil {
			 = time.NewTimer()
		} else {
			.Reset()
		}
		select {
		case  := <-:
			.holePunchingMx.Lock()
			delete(.holePunching, )
			.holePunchingMx.Unlock()
			return , nil
		case <-.C:
		case <-.Done():
			 = ErrHolePunching
			break 
		}
	}
	// we only arrive here if punchErr != nil
	.holePunchingMx.Lock()
	defer func() {
		delete(.holePunching, )
		.holePunchingMx.Unlock()
	}()
	select {
	case  := <-.holePunching[].connCh:
		return , nil
	default:
		return nil, 
	}
}

// Don't use mafmt.QUIC as we don't want to dial DNS addresses. Just /ip{4,6}/udp/quic-v1
var dialMatcher = mafmt.And(mafmt.IP, mafmt.Base(ma.P_UDP), mafmt.Base(ma.P_QUIC_V1))

// CanDial determines if we can dial to an address
func ( *transport) ( ma.Multiaddr) bool {
	return dialMatcher.Matches()
}

// Listen listens for new QUIC connections on the passed multiaddr.
func ( *transport) ( ma.Multiaddr) (tpt.Listener, error) {
	var  tls.Config
	.GetConfigForClient = func( *tls.ClientHelloInfo) (*tls.Config, error) {
		// return a tls.Config that verifies the peer's certificate chain.
		// Note that since we have no way of associating an incoming QUIC connection with
		// the peer ID calculated here, we don't actually receive the peer's public key
		// from the key chan.
		,  := .identity.ConfigForPeer("")
		return , nil
	}
	.NextProtos = []string{"libp2p"}
	, ,  := quicreuse.FromQuicMultiaddr()
	if  != nil {
		return nil, 
	}

	.listenersMu.Lock()
	defer .listenersMu.Unlock()
	 := .listeners[.String()]
	var  *listener
	var  *acceptLoopRunner
	if len() != 0 {
		// We already have an underlying listener, let's use it
		 = [0].listener
		 = [0].acceptRunnner
		// Make sure our underlying listener is listening on the specified QUIC version
		if ,  := .localMultiaddrs[]; ! {
			return nil, fmt.Errorf("can't listen on quic version %v, underlying listener doesn't support it", )
		}
	} else {
		,  := .connManager.ListenQUICAndAssociate(, , &, .allowWindowIncrease)
		if  != nil {
			return nil, 
		}
		,  := newListener(, , .localPeer, .privKey, .rcmgr)
		if  != nil {
			_ = .Close()
			return nil, 
		}
		 = &

		 = &acceptLoopRunner{
			acceptSem: make(chan struct{}, 1),
			muxer:     make(map[quic.Version]chan acceptVal),
		}
	}

	 := &virtualListener{
		listener:      ,
		version:       ,
		udpAddr:       .String(),
		t:             ,
		acceptRunnner: ,
		acceptChan:    .AcceptForVersion(),
	}

	 = append(, )
	.listeners[.String()] = 

	return , nil
}

func ( *transport) ( quic.Connection,  uint64) bool {
	// If the QUIC connection tries to increase the window before we've inserted it
	// into our connections map (which we do right after dialing / accepting it),
	// we have no way to account for that memory. This should be very rare.
	// Block this attempt. The connection can request more memory later.
	.connMx.Lock()
	,  := .conns[]
	.connMx.Unlock()
	if ! {
		return false
	}
	return .allowWindowIncrease()
}

// Proxy returns true if this transport proxies.
func ( *transport) () bool {
	return false
}

// Protocols returns the set of protocols handled by this transport.
func ( *transport) () []int {
	return .connManager.Protocols()
}

func ( *transport) () string {
	return "QUIC"
}

func ( *transport) () error {
	return nil
}

func ( *transport) ( *virtualListener) error {
	.listenersMu.Lock()
	defer .listenersMu.Unlock()

	var  error
	 := .listeners[.udpAddr]
	if len() == 1 {
		// This is the last virtual listener here, so we can close the underlying listener
		 = .listener.Close()
		delete(.listeners, .udpAddr)
		return 
	}

	for  := 0;  < len(); ++ {
		// Swap remove
		if  == [] {
			[] = [len()-1]
			 = [:len()-1]
			.listeners[.udpAddr] = 
			break
		}
	}

	return nil

}