lnd: switch over to using brontide for p2p connections
This commit modifies the existing p2p connection authentication and encryption scheme to now use the newly designed ‘brontide’ authenticated key agreement scheme for all connections. Additionally, within the daemon lnwire.NetAddress is now used within all peers which encapsulates host information, a node’s identity public key relevant services, and supported bitcoin nets.
This commit is contained in:
parent
474f0afceb
commit
4fe23a8b3e
@ -200,7 +200,7 @@ func (f *fundingManager) NumPendingChannels() uint32 {
|
|||||||
|
|
||||||
type pendingChannel struct {
|
type pendingChannel struct {
|
||||||
peerId int32
|
peerId int32
|
||||||
lightningID [32]byte
|
identityPub *btcec.PublicKey
|
||||||
channelPoint *wire.OutPoint
|
channelPoint *wire.OutPoint
|
||||||
capacity btcutil.Amount
|
capacity btcutil.Amount
|
||||||
localBalance btcutil.Amount
|
localBalance btcutil.Amount
|
||||||
@ -285,7 +285,7 @@ func (f *fundingManager) handlePendingChannels(msg *pendingChansReq) {
|
|||||||
|
|
||||||
pendingChan := &pendingChannel{
|
pendingChan := &pendingChannel{
|
||||||
peerId: peerID,
|
peerId: peerID,
|
||||||
lightningID: peer.lightningID,
|
identityPub: peer.addr.IdentityKey,
|
||||||
channelPoint: res.FundingOutpoint(),
|
channelPoint: res.FundingOutpoint(),
|
||||||
capacity: localFund + remoteFund,
|
capacity: localFund + remoteFund,
|
||||||
localBalance: localFund,
|
localBalance: localFund,
|
||||||
@ -340,7 +340,7 @@ func (f *fundingManager) handleFundingRequest(fmsg *fundingRequestMsg) {
|
|||||||
// channel ourselves.
|
// channel ourselves.
|
||||||
// TODO(roasbeef): passing num confs 1 is irrelevant here, make signed?
|
// TODO(roasbeef): passing num confs 1 is irrelevant here, make signed?
|
||||||
reservation, err := f.wallet.InitChannelReservation(amt, 0,
|
reservation, err := f.wallet.InitChannelReservation(amt, 0,
|
||||||
fmsg.peer.identityPub, fmsg.peer.lightningAddr.NetAddr, 1, delay)
|
fmsg.peer.addr.IdentityKey, fmsg.peer.addr.Address, 1, delay)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
// TODO(roasbeef): push ErrorGeneric message
|
// TODO(roasbeef): push ErrorGeneric message
|
||||||
fndgLog.Errorf("Unable to initialize reservation: %v", err)
|
fndgLog.Errorf("Unable to initialize reservation: %v", err)
|
||||||
@ -614,7 +614,8 @@ func (f *fundingManager) handleFundingSignComplete(fmsg *fundingSignCompleteMsg)
|
|||||||
// finding.
|
// finding.
|
||||||
chanInfo := openChan.StateSnapshot()
|
chanInfo := openChan.StateSnapshot()
|
||||||
capacity := int64(chanInfo.LocalBalance + chanInfo.RemoteBalance)
|
capacity := int64(chanInfo.LocalBalance + chanInfo.RemoteBalance)
|
||||||
vertex := hex.EncodeToString(fmsg.peer.identityPub.SerializeCompressed())
|
pubSerialized := fmsg.peer.addr.IdentityKey.SerializeCompressed()
|
||||||
|
vertex := hex.EncodeToString(pubSerialized)
|
||||||
fmsg.peer.server.routingMgr.OpenChannel(
|
fmsg.peer.server.routingMgr.OpenChannel(
|
||||||
graph.NewID(vertex),
|
graph.NewID(vertex),
|
||||||
graph.NewEdgeID(fundingPoint.String()),
|
graph.NewEdgeID(fundingPoint.String()),
|
||||||
@ -686,7 +687,7 @@ func (f *fundingManager) handleFundingOpen(fmsg *fundingOpenMsg) {
|
|||||||
// Notify the L3 routing manager of the newly active channel link.
|
// Notify the L3 routing manager of the newly active channel link.
|
||||||
capacity := int64(resCtx.reservation.OurContribution().FundingAmount +
|
capacity := int64(resCtx.reservation.OurContribution().FundingAmount +
|
||||||
resCtx.reservation.TheirContribution().FundingAmount)
|
resCtx.reservation.TheirContribution().FundingAmount)
|
||||||
vertex := hex.EncodeToString(fmsg.peer.identityPub.SerializeCompressed())
|
vertex := hex.EncodeToString(fmsg.peer.addr.IdentityKey.SerializeCompressed())
|
||||||
fmsg.peer.server.routingMgr.OpenChannel(
|
fmsg.peer.server.routingMgr.OpenChannel(
|
||||||
graph.NewID(vertex),
|
graph.NewID(vertex),
|
||||||
graph.NewEdgeID(resCtx.reservation.FundingOutpoint().String()),
|
graph.NewEdgeID(resCtx.reservation.FundingOutpoint().String()),
|
||||||
@ -715,7 +716,7 @@ func (f *fundingManager) initFundingWorkflow(targetPeer *peer, req *openChanReq)
|
|||||||
func (f *fundingManager) handleInitFundingMsg(msg *initFundingMsg) {
|
func (f *fundingManager) handleInitFundingMsg(msg *initFundingMsg) {
|
||||||
var (
|
var (
|
||||||
// TODO(roasbeef): add delay
|
// TODO(roasbeef): add delay
|
||||||
nodeID = msg.peer.identityPub
|
nodeID = msg.peer.addr.IdentityKey
|
||||||
localAmt = msg.localFundingAmt
|
localAmt = msg.localFundingAmt
|
||||||
remoteAmt = msg.remoteFundingAmt
|
remoteAmt = msg.remoteFundingAmt
|
||||||
capacity = localAmt + remoteAmt
|
capacity = localAmt + remoteAmt
|
||||||
@ -724,13 +725,13 @@ func (f *fundingManager) handleInitFundingMsg(msg *initFundingMsg) {
|
|||||||
|
|
||||||
fndgLog.Infof("Initiating fundingRequest(localAmt=%v, remoteAmt=%v, "+
|
fndgLog.Infof("Initiating fundingRequest(localAmt=%v, remoteAmt=%v, "+
|
||||||
"capacity=%v, numConfs=%v, addr=%v)", localAmt, remoteAmt,
|
"capacity=%v, numConfs=%v, addr=%v)", localAmt, remoteAmt,
|
||||||
capacity, numConfs, msg.peer.lightningAddr.NetAddr)
|
capacity, numConfs, msg.peer.addr.Address)
|
||||||
|
|
||||||
// Initialize a funding reservation with the local wallet. If the
|
// Initialize a funding reservation with the local wallet. If the
|
||||||
// wallet doesn't have enough funds to commit to this channel, then
|
// wallet doesn't have enough funds to commit to this channel, then
|
||||||
// the request will fail, and be aborted.
|
// the request will fail, and be aborted.
|
||||||
reservation, err := f.wallet.InitChannelReservation(capacity, localAmt,
|
reservation, err := f.wallet.InitChannelReservation(capacity, localAmt,
|
||||||
nodeID, msg.peer.lightningAddr.NetAddr, uint16(numConfs), 4)
|
nodeID, msg.peer.addr.Address, uint16(numConfs), 4)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
msg.err <- err
|
msg.err <- err
|
||||||
return
|
return
|
||||||
|
@ -125,8 +125,10 @@ type htlcSwitch struct {
|
|||||||
interfaceMtx sync.RWMutex
|
interfaceMtx sync.RWMutex
|
||||||
interfaces map[wire.ShaHash][]*link
|
interfaces map[wire.ShaHash][]*link
|
||||||
|
|
||||||
// onionIndex is a secondary index used to properly forward a message
|
// onionIndex is an index used to properly forward a message
|
||||||
// to the next hop within a Sphinx circuit.
|
// to the next hop within a Sphinx circuit. Within the sphinx packets,
|
||||||
|
// the "next-hop" destination is encoded as the hash160 of the node's
|
||||||
|
// public key serialized in compressed format.
|
||||||
onionMtx sync.RWMutex
|
onionMtx sync.RWMutex
|
||||||
onionIndex map[[ripemd160.Size]byte][]*link
|
onionIndex map[[ripemd160.Size]byte][]*link
|
||||||
|
|
||||||
@ -442,6 +444,10 @@ func (h *htlcSwitch) handleRegisterLink(req *registerLinkMsg) {
|
|||||||
chanPoint: chanPoint,
|
chanPoint: chanPoint,
|
||||||
}
|
}
|
||||||
|
|
||||||
|
// First update the channel index with this new channel point. The
|
||||||
|
// channel index will be used to quickly lookup channels in order to:
|
||||||
|
// close them, update their link capacity, or possibly during multi-hop
|
||||||
|
// HTLC forwarding.
|
||||||
h.chanIndexMtx.Lock()
|
h.chanIndexMtx.Lock()
|
||||||
h.chanIndex[*chanPoint] = newLink
|
h.chanIndex[*chanPoint] = newLink
|
||||||
h.chanIndexMtx.Unlock()
|
h.chanIndexMtx.Unlock()
|
||||||
@ -452,8 +458,11 @@ func (h *htlcSwitch) handleRegisterLink(req *registerLinkMsg) {
|
|||||||
h.interfaces[interfaceID] = append(h.interfaces[interfaceID], newLink)
|
h.interfaces[interfaceID] = append(h.interfaces[interfaceID], newLink)
|
||||||
h.interfaceMtx.Unlock()
|
h.interfaceMtx.Unlock()
|
||||||
|
|
||||||
|
// Next, update the onion index which is used to look up the
|
||||||
|
// settle/clear links during multi-hop payments and to dispatch
|
||||||
|
// outgoing payments initiated by a local sub-system.
|
||||||
var onionId [ripemd160.Size]byte
|
var onionId [ripemd160.Size]byte
|
||||||
copy(onionId[:], btcutil.Hash160(req.peer.identityPub.SerializeCompressed()))
|
copy(onionId[:], btcutil.Hash160(req.peer.addr.IdentityKey.SerializeCompressed()))
|
||||||
|
|
||||||
h.onionMtx.Lock()
|
h.onionMtx.Lock()
|
||||||
h.onionIndex[onionId] = h.interfaces[interfaceID]
|
h.onionIndex[onionId] = h.interfaces[interfaceID]
|
||||||
|
42
peer.go
42
peer.go
@ -15,7 +15,6 @@ import (
|
|||||||
"github.com/davecgh/go-spew/spew"
|
"github.com/davecgh/go-spew/spew"
|
||||||
"github.com/lightningnetwork/lightning-onion"
|
"github.com/lightningnetwork/lightning-onion"
|
||||||
"github.com/lightningnetwork/lnd/channeldb"
|
"github.com/lightningnetwork/lnd/channeldb"
|
||||||
"github.com/lightningnetwork/lnd/lndc"
|
|
||||||
"github.com/lightningnetwork/lnd/lnrpc"
|
"github.com/lightningnetwork/lnd/lnrpc"
|
||||||
"github.com/lightningnetwork/lnd/lnwallet"
|
"github.com/lightningnetwork/lnd/lnwallet"
|
||||||
"github.com/lightningnetwork/lnd/lnwire"
|
"github.com/lightningnetwork/lnd/lnwire"
|
||||||
@ -66,8 +65,7 @@ type peer struct {
|
|||||||
|
|
||||||
conn net.Conn
|
conn net.Conn
|
||||||
|
|
||||||
identityPub *btcec.PublicKey
|
addr *lnwire.NetAddress
|
||||||
lightningAddr *lndc.LNAdr
|
|
||||||
lightningID wire.ShaHash
|
lightningID wire.ShaHash
|
||||||
|
|
||||||
inbound bool
|
inbound bool
|
||||||
@ -88,9 +86,6 @@ type peer struct {
|
|||||||
satoshisSent uint64
|
satoshisSent uint64
|
||||||
satoshisReceived uint64
|
satoshisReceived uint64
|
||||||
|
|
||||||
// chainNet is the Bitcoin network to which this peer is anchored to.
|
|
||||||
chainNet wire.BitcoinNet
|
|
||||||
|
|
||||||
// sendQueue is the channel which is used to queue outgoing to be
|
// sendQueue is the channel which is used to queue outgoing to be
|
||||||
// written onto the wire. Note that this channel is unbuffered.
|
// written onto the wire. Note that this channel is unbuffered.
|
||||||
sendQueue chan outgoinMsg
|
sendQueue chan outgoinMsg
|
||||||
@ -150,16 +145,17 @@ type peer struct {
|
|||||||
|
|
||||||
// newPeer creates a new peer from an establish connection object, and a
|
// newPeer creates a new peer from an establish connection object, and a
|
||||||
// pointer to the main server.
|
// pointer to the main server.
|
||||||
func newPeer(conn net.Conn, server *server, btcNet wire.BitcoinNet, inbound bool) (*peer, error) {
|
func newPeer(conn net.Conn, server *server, addr *lnwire.NetAddress,
|
||||||
lndcConn := conn.(*lndc.LNDConn)
|
inbound bool) (*peer, error) {
|
||||||
nodePub := lndcConn.RemotePub
|
|
||||||
|
nodePub := addr.IdentityKey
|
||||||
|
|
||||||
p := &peer{
|
p := &peer{
|
||||||
conn: conn,
|
conn: conn,
|
||||||
identityPub: nodePub,
|
|
||||||
lightningID: wire.ShaHash(fastsha256.Sum256(nodePub.SerializeCompressed())),
|
lightningID: wire.ShaHash(fastsha256.Sum256(nodePub.SerializeCompressed())),
|
||||||
|
addr: addr,
|
||||||
|
|
||||||
id: atomic.AddInt32(&numNodes, 1),
|
id: atomic.AddInt32(&numNodes, 1),
|
||||||
chainNet: btcNet,
|
|
||||||
inbound: inbound,
|
inbound: inbound,
|
||||||
|
|
||||||
server: server,
|
server: server,
|
||||||
@ -184,15 +180,6 @@ func newPeer(conn net.Conn, server *server, btcNet wire.BitcoinNet, inbound bool
|
|||||||
quit: make(chan struct{}),
|
quit: make(chan struct{}),
|
||||||
}
|
}
|
||||||
|
|
||||||
// TODO(roasbeef): re-write after lnaddr revamp, shouldn't need to use
|
|
||||||
// type assertions
|
|
||||||
var err error
|
|
||||||
tcpAddr := lndcConn.Conn.(*net.TCPConn).RemoteAddr().(*net.TCPAddr)
|
|
||||||
p.lightningAddr, err = lndc.NewLnAdr(tcpAddr, nodePub, activeNetParams.Params)
|
|
||||||
if err != nil {
|
|
||||||
return nil, err
|
|
||||||
}
|
|
||||||
|
|
||||||
// Initiate the pending channel identifier properly depending on if this
|
// Initiate the pending channel identifier properly depending on if this
|
||||||
// node is inbound or outbound. This value will be used in an increasing
|
// node is inbound or outbound. This value will be used in an increasing
|
||||||
// manner to track pending channels.
|
// manner to track pending channels.
|
||||||
@ -204,7 +191,7 @@ func newPeer(conn net.Conn, server *server, btcNet wire.BitcoinNet, inbound bool
|
|||||||
|
|
||||||
// Fetch and then load all the active channels we have with this
|
// Fetch and then load all the active channels we have with this
|
||||||
// remote peer from the database.
|
// remote peer from the database.
|
||||||
activeChans, err := server.chanDB.FetchOpenChannels(p.identityPub)
|
activeChans, err := server.chanDB.FetchOpenChannels(p.addr.IdentityKey)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
peerLog.Errorf("unable to fetch active chans "+
|
peerLog.Errorf("unable to fetch active chans "+
|
||||||
"for peer %v: %v", p, err)
|
"for peer %v: %v", p, err)
|
||||||
@ -313,7 +300,7 @@ func (p *peer) Disconnect() {
|
|||||||
// Tell the switch to unregister all links associated with this
|
// Tell the switch to unregister all links associated with this
|
||||||
// peer. Passing nil as the target link indicates that all links
|
// peer. Passing nil as the target link indicates that all links
|
||||||
// associated with this interface should be closed.
|
// associated with this interface should be closed.
|
||||||
p.server.htlcSwitch.UnregisterLink(p.identityPub, nil)
|
p.server.htlcSwitch.UnregisterLink(p.addr.IdentityKey, nil)
|
||||||
|
|
||||||
p.server.donePeers <- p
|
p.server.donePeers <- p
|
||||||
}()
|
}()
|
||||||
@ -328,7 +315,8 @@ func (p *peer) String() string {
|
|||||||
// any additional raw payload.
|
// any additional raw payload.
|
||||||
func (p *peer) readNextMessage() (lnwire.Message, []byte, error) {
|
func (p *peer) readNextMessage() (lnwire.Message, []byte, error) {
|
||||||
// TODO(roasbeef): use our own net magic?
|
// TODO(roasbeef): use our own net magic?
|
||||||
n, nextMsg, rawPayload, err := lnwire.ReadMessage(p.conn, 0, p.chainNet)
|
n, nextMsg, rawPayload, err := lnwire.ReadMessage(p.conn, 0,
|
||||||
|
p.addr.ChainNet)
|
||||||
atomic.AddUint64(&p.bytesReceived, uint64(n))
|
atomic.AddUint64(&p.bytesReceived, uint64(n))
|
||||||
if err != nil {
|
if err != nil {
|
||||||
return nil, nil, err
|
return nil, nil, err
|
||||||
@ -403,7 +391,7 @@ out:
|
|||||||
*lnwire.RoutingTableTransferMessage:
|
*lnwire.RoutingTableTransferMessage:
|
||||||
|
|
||||||
// Convert to base routing message and set sender and receiver
|
// Convert to base routing message and set sender and receiver
|
||||||
vertex := hex.EncodeToString(p.identityPub.SerializeCompressed())
|
vertex := hex.EncodeToString(p.addr.IdentityKey.SerializeCompressed())
|
||||||
p.server.routingMgr.ReceiveRoutingMessage(msg, graph.NewID(vertex))
|
p.server.routingMgr.ReceiveRoutingMessage(msg, graph.NewID(vertex))
|
||||||
}
|
}
|
||||||
|
|
||||||
@ -457,7 +445,7 @@ func (p *peer) writeMessage(msg lnwire.Message) error {
|
|||||||
return spew.Sdump(msg)
|
return spew.Sdump(msg)
|
||||||
}))
|
}))
|
||||||
|
|
||||||
n, err := lnwire.WriteMessage(p.conn, msg, 0, p.chainNet)
|
n, err := lnwire.WriteMessage(p.conn, msg, 0, p.addr.ChainNet)
|
||||||
atomic.AddUint64(&p.bytesSent, uint64(n))
|
atomic.AddUint64(&p.bytesSent, uint64(n))
|
||||||
|
|
||||||
return err
|
return err
|
||||||
@ -846,7 +834,7 @@ func wipeChannel(p *peer, channel *lnwallet.LightningChannel) error {
|
|||||||
|
|
||||||
// Instruct the Htlc Switch to close this link as the channel is no
|
// Instruct the Htlc Switch to close this link as the channel is no
|
||||||
// longer active.
|
// longer active.
|
||||||
p.server.htlcSwitch.UnregisterLink(p.identityPub, chanID)
|
p.server.htlcSwitch.UnregisterLink(p.addr.IdentityKey, chanID)
|
||||||
htlcWireLink, ok := p.htlcManagers[*chanID]
|
htlcWireLink, ok := p.htlcManagers[*chanID]
|
||||||
if !ok {
|
if !ok {
|
||||||
return nil
|
return nil
|
||||||
@ -1274,6 +1262,7 @@ func (p *peer) handleUpstreamMsg(state *commitmentState, msg lnwire.Message) {
|
|||||||
for _, htlc := range htlcsToForward {
|
for _, htlc := range htlcsToForward {
|
||||||
// We don't need to forward any HTLC's that we
|
// We don't need to forward any HTLC's that we
|
||||||
// just settled above.
|
// just settled above.
|
||||||
|
// TODO(roasbeef): key by index insteaad?
|
||||||
if _, ok := settledPayments[htlc.RHash]; ok {
|
if _, ok := settledPayments[htlc.RHash]; ok {
|
||||||
continue
|
continue
|
||||||
}
|
}
|
||||||
@ -1322,6 +1311,7 @@ func (p *peer) handleUpstreamMsg(state *commitmentState, msg lnwire.Message) {
|
|||||||
|
|
||||||
// Notify the invoiceRegistry of the invoices we just settled
|
// Notify the invoiceRegistry of the invoices we just settled
|
||||||
// with this latest commitment update.
|
// with this latest commitment update.
|
||||||
|
// TODO(roasbeef): wait until next transition?
|
||||||
for invoice, _ := range settledPayments {
|
for invoice, _ := range settledPayments {
|
||||||
err := p.server.invoices.SettleInvoice(wire.ShaHash(invoice))
|
err := p.server.invoices.SettleInvoice(wire.ShaHash(invoice))
|
||||||
if err != nil {
|
if err != nil {
|
||||||
|
46
rpcserver.go
46
rpcserver.go
@ -5,6 +5,7 @@ import (
|
|||||||
"encoding/hex"
|
"encoding/hex"
|
||||||
"fmt"
|
"fmt"
|
||||||
"io"
|
"io"
|
||||||
|
"net"
|
||||||
"time"
|
"time"
|
||||||
|
|
||||||
"sync"
|
"sync"
|
||||||
@ -15,7 +16,6 @@ import (
|
|||||||
"github.com/davecgh/go-spew/spew"
|
"github.com/davecgh/go-spew/spew"
|
||||||
"github.com/lightningnetwork/lightning-onion"
|
"github.com/lightningnetwork/lightning-onion"
|
||||||
"github.com/lightningnetwork/lnd/channeldb"
|
"github.com/lightningnetwork/lnd/channeldb"
|
||||||
"github.com/lightningnetwork/lnd/lndc"
|
|
||||||
"github.com/lightningnetwork/lnd/lnrpc"
|
"github.com/lightningnetwork/lnd/lnrpc"
|
||||||
"github.com/lightningnetwork/lnd/lnwallet"
|
"github.com/lightningnetwork/lnd/lnwallet"
|
||||||
"github.com/lightningnetwork/lnd/lnwire"
|
"github.com/lightningnetwork/lnd/lnwire"
|
||||||
@ -194,14 +194,25 @@ func (r *rpcServer) ConnectPeer(ctx context.Context,
|
|||||||
return nil, fmt.Errorf("need: lnc pubkeyhash@hostname")
|
return nil, fmt.Errorf("need: lnc pubkeyhash@hostname")
|
||||||
}
|
}
|
||||||
|
|
||||||
idAtHost := fmt.Sprintf("%v@%v", in.Addr.PubKeyHash, in.Addr.Host)
|
pubkeyHex, err := hex.DecodeString(in.Addr.Pubkey)
|
||||||
rpcsLog.Debugf("[connectpeer] peer=%v", idAtHost)
|
|
||||||
|
|
||||||
peerAddr, err := lndc.LnAddrFromString(idAtHost, activeNetParams.Params)
|
|
||||||
if err != nil {
|
if err != nil {
|
||||||
rpcsLog.Errorf("(connectpeer): error parsing ln addr: %v", err)
|
|
||||||
return nil, err
|
return nil, err
|
||||||
}
|
}
|
||||||
|
pubkey, err := btcec.ParsePubKey(pubkeyHex, btcec.S256())
|
||||||
|
if err != nil {
|
||||||
|
return nil, err
|
||||||
|
}
|
||||||
|
|
||||||
|
host, err := net.ResolveTCPAddr("tcp", in.Addr.Host)
|
||||||
|
if err != nil {
|
||||||
|
return nil, err
|
||||||
|
}
|
||||||
|
|
||||||
|
peerAddr := &lnwire.NetAddress{
|
||||||
|
IdentityKey: pubkey,
|
||||||
|
Address: host,
|
||||||
|
ChainNet: activeNetParams.Net,
|
||||||
|
}
|
||||||
|
|
||||||
peerID, err := r.server.ConnectToPeer(peerAddr)
|
peerID, err := r.server.ConnectToPeer(peerAddr)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
@ -209,6 +220,7 @@ func (r *rpcServer) ConnectPeer(ctx context.Context,
|
|||||||
return nil, err
|
return nil, err
|
||||||
}
|
}
|
||||||
|
|
||||||
|
// TODO(roasbeef): add pubkey return
|
||||||
rpcsLog.Debugf("Connected to peer: %v", peerAddr.String())
|
rpcsLog.Debugf("Connected to peer: %v", peerAddr.String())
|
||||||
return &lnrpc.ConnectPeerResponse{peerID}, nil
|
return &lnrpc.ConnectPeerResponse{peerID}, nil
|
||||||
}
|
}
|
||||||
@ -224,8 +236,13 @@ func (r *rpcServer) OpenChannel(in *lnrpc.OpenChannelRequest,
|
|||||||
|
|
||||||
localFundingAmt := btcutil.Amount(in.LocalFundingAmount)
|
localFundingAmt := btcutil.Amount(in.LocalFundingAmount)
|
||||||
remoteFundingAmt := btcutil.Amount(in.RemoteFundingAmount)
|
remoteFundingAmt := btcutil.Amount(in.RemoteFundingAmount)
|
||||||
|
nodepubKey, err := btcec.ParsePubKey(in.NodePubkey, btcec.S256())
|
||||||
|
if err != nil {
|
||||||
|
return err
|
||||||
|
}
|
||||||
|
|
||||||
updateChan, errChan := r.server.OpenChannel(in.TargetPeerId,
|
updateChan, errChan := r.server.OpenChannel(in.TargetPeerId,
|
||||||
in.TargetNode, localFundingAmt, remoteFundingAmt, in.NumConfs)
|
nodepubKey, localFundingAmt, remoteFundingAmt, in.NumConfs)
|
||||||
|
|
||||||
var outpoint wire.OutPoint
|
var outpoint wire.OutPoint
|
||||||
out:
|
out:
|
||||||
@ -233,8 +250,8 @@ out:
|
|||||||
select {
|
select {
|
||||||
case err := <-errChan:
|
case err := <-errChan:
|
||||||
rpcsLog.Errorf("unable to open channel to "+
|
rpcsLog.Errorf("unable to open channel to "+
|
||||||
"lightningID(%x) nor peerID(%v): %v",
|
"identityPub(%x) nor peerID(%v): %v",
|
||||||
in.TargetNode, in.TargetPeerId, err)
|
nodepubKey, in.TargetPeerId, err)
|
||||||
return err
|
return err
|
||||||
case fundingUpdate := <-updateChan:
|
case fundingUpdate := <-updateChan:
|
||||||
rpcsLog.Tracef("[openchannel] sending update: %v",
|
rpcsLog.Tracef("[openchannel] sending update: %v",
|
||||||
@ -333,17 +350,11 @@ func (r *rpcServer) GetInfo(ctx context.Context,
|
|||||||
}
|
}
|
||||||
|
|
||||||
pendingChannels := r.server.fundingMgr.NumPendingChannels()
|
pendingChannels := r.server.fundingMgr.NumPendingChannels()
|
||||||
|
|
||||||
idPub := r.server.identityPriv.PubKey().SerializeCompressed()
|
idPub := r.server.identityPriv.PubKey().SerializeCompressed()
|
||||||
idAddr, err := btcutil.NewAddressPubKeyHash(btcutil.Hash160(idPub), activeNetParams.Params)
|
|
||||||
if err != nil {
|
|
||||||
return nil, err
|
|
||||||
}
|
|
||||||
|
|
||||||
return &lnrpc.GetInfoResponse{
|
return &lnrpc.GetInfoResponse{
|
||||||
LightningId: hex.EncodeToString(r.server.lightningID[:]),
|
LightningId: hex.EncodeToString(r.server.lightningID[:]),
|
||||||
IdentityPubkey: hex.EncodeToString(idPub),
|
IdentityPubkey: hex.EncodeToString(idPub),
|
||||||
IdentityAddress: idAddr.String(),
|
|
||||||
NumPendingChannels: pendingChannels,
|
NumPendingChannels: pendingChannels,
|
||||||
NumActiveChannels: activeChannels,
|
NumActiveChannels: activeChannels,
|
||||||
NumPeers: uint32(len(serverPeers)),
|
NumPeers: uint32(len(serverPeers)),
|
||||||
@ -364,7 +375,7 @@ func (r *rpcServer) ListPeers(ctx context.Context,
|
|||||||
for _, serverPeer := range serverPeers {
|
for _, serverPeer := range serverPeers {
|
||||||
// TODO(roasbeef): add a snapshot method which grabs peer read mtx
|
// TODO(roasbeef): add a snapshot method which grabs peer read mtx
|
||||||
|
|
||||||
nodePub := serverPeer.identityPub.SerializeCompressed()
|
nodePub := serverPeer.addr.IdentityKey.SerializeCompressed()
|
||||||
peer := &lnrpc.Peer{
|
peer := &lnrpc.Peer{
|
||||||
PubKey: hex.EncodeToString(nodePub),
|
PubKey: hex.EncodeToString(nodePub),
|
||||||
PeerId: serverPeer.id,
|
PeerId: serverPeer.id,
|
||||||
@ -432,9 +443,10 @@ func (r *rpcServer) PendingChannels(ctx context.Context,
|
|||||||
pendingOpenChans := r.server.fundingMgr.PendingChannels()
|
pendingOpenChans := r.server.fundingMgr.PendingChannels()
|
||||||
for _, pendingOpen := range pendingOpenChans {
|
for _, pendingOpen := range pendingOpenChans {
|
||||||
// TODO(roasbeef): add confirmation progress
|
// TODO(roasbeef): add confirmation progress
|
||||||
|
pub := pendingOpen.identityPub.SerializeCompressed()
|
||||||
pendingChan := &lnrpc.PendingChannelResponse_PendingChannel{
|
pendingChan := &lnrpc.PendingChannelResponse_PendingChannel{
|
||||||
PeerId: pendingOpen.peerId,
|
PeerId: pendingOpen.peerId,
|
||||||
LightningId: hex.EncodeToString(pendingOpen.lightningID[:]),
|
IdentityKey: hex.EncodeToString(pub),
|
||||||
ChannelPoint: pendingOpen.channelPoint.String(),
|
ChannelPoint: pendingOpen.channelPoint.String(),
|
||||||
Capacity: int64(pendingOpen.capacity),
|
Capacity: int64(pendingOpen.capacity),
|
||||||
LocalBalance: int64(pendingOpen.localBalance),
|
LocalBalance: int64(pendingOpen.localBalance),
|
||||||
|
86
server.go
86
server.go
@ -1,7 +1,6 @@
|
|||||||
package main
|
package main
|
||||||
|
|
||||||
import (
|
import (
|
||||||
"bytes"
|
|
||||||
"encoding/hex"
|
"encoding/hex"
|
||||||
"fmt"
|
"fmt"
|
||||||
"net"
|
"net"
|
||||||
@ -10,11 +9,12 @@ import (
|
|||||||
|
|
||||||
"github.com/btcsuite/fastsha256"
|
"github.com/btcsuite/fastsha256"
|
||||||
"github.com/lightningnetwork/lightning-onion"
|
"github.com/lightningnetwork/lightning-onion"
|
||||||
|
"github.com/lightningnetwork/lnd/brontide"
|
||||||
"github.com/lightningnetwork/lnd/chainntnfs"
|
"github.com/lightningnetwork/lnd/chainntnfs"
|
||||||
"github.com/lightningnetwork/lnd/channeldb"
|
"github.com/lightningnetwork/lnd/channeldb"
|
||||||
"github.com/lightningnetwork/lnd/lndc"
|
|
||||||
"github.com/lightningnetwork/lnd/lnrpc"
|
"github.com/lightningnetwork/lnd/lnrpc"
|
||||||
"github.com/lightningnetwork/lnd/lnwallet"
|
"github.com/lightningnetwork/lnd/lnwallet"
|
||||||
|
"github.com/lightningnetwork/lnd/lnwire"
|
||||||
"github.com/roasbeef/btcd/btcec"
|
"github.com/roasbeef/btcd/btcec"
|
||||||
"github.com/roasbeef/btcutil"
|
"github.com/roasbeef/btcutil"
|
||||||
|
|
||||||
@ -81,7 +81,7 @@ func newServer(listenAddrs []string, notifier chainntnfs.ChainNotifier,
|
|||||||
|
|
||||||
listeners := make([]net.Listener, len(listenAddrs))
|
listeners := make([]net.Listener, len(listenAddrs))
|
||||||
for i, addr := range listenAddrs {
|
for i, addr := range listenAddrs {
|
||||||
listeners[i], err = lndc.NewListener(privKey, addr)
|
listeners[i], err = brontide.NewListener(privKey, addr)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
return nil, err
|
return nil, err
|
||||||
}
|
}
|
||||||
@ -250,7 +250,7 @@ func (s *server) removePeer(p *peer) {
|
|||||||
// particular peer. This message also houses an error channel which will be
|
// particular peer. This message also houses an error channel which will be
|
||||||
// used to report success/failure.
|
// used to report success/failure.
|
||||||
type connectPeerMsg struct {
|
type connectPeerMsg struct {
|
||||||
addr *lndc.LNAdr
|
addr *lnwire.NetAddress
|
||||||
resp chan int32
|
resp chan int32
|
||||||
err chan error
|
err chan error
|
||||||
}
|
}
|
||||||
@ -266,7 +266,7 @@ type listPeersMsg struct {
|
|||||||
// relative peer ID, or a global lightning ID.
|
// relative peer ID, or a global lightning ID.
|
||||||
type openChanReq struct {
|
type openChanReq struct {
|
||||||
targetPeerID int32
|
targetPeerID int32
|
||||||
targetNodeID [32]byte
|
targetPubkey *btcec.PublicKey
|
||||||
|
|
||||||
// TODO(roasbeef): make enums in lnwire
|
// TODO(roasbeef): make enums in lnwire
|
||||||
channelType uint8
|
channelType uint8
|
||||||
@ -318,7 +318,7 @@ out:
|
|||||||
|
|
||||||
var targetPeer *peer
|
var targetPeer *peer
|
||||||
for _, peer := range s.peers { // TODO: threadsafe api
|
for _, peer := range s.peers { // TODO: threadsafe api
|
||||||
nodePub := peer.identityPub.SerializeCompressed()
|
nodePub := peer.addr.IdentityKey.SerializeCompressed()
|
||||||
idStr := hex.EncodeToString(nodePub)
|
idStr := hex.EncodeToString(nodePub)
|
||||||
|
|
||||||
// We found the the target
|
// We found the the target
|
||||||
@ -361,51 +361,39 @@ func (s *server) handleConnectPeer(msg *connectPeerMsg) {
|
|||||||
|
|
||||||
// Ensure we're not already connected to this
|
// Ensure we're not already connected to this
|
||||||
// peer.
|
// peer.
|
||||||
|
targetPub := msg.addr.IdentityKey
|
||||||
for _, peer := range s.peers {
|
for _, peer := range s.peers {
|
||||||
if peer.lightningAddr.String() == addr.String() {
|
if peer.addr.IdentityKey.IsEqual(targetPub) {
|
||||||
msg.err <- fmt.Errorf(
|
msg.err <- fmt.Errorf(
|
||||||
"already connected to peer: %v",
|
"already connected to peer: %v",
|
||||||
peer.lightningAddr,
|
peer.addr,
|
||||||
)
|
)
|
||||||
msg.resp <- -1
|
msg.resp <- -1
|
||||||
return
|
return
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
// Launch a goroutine to connect to the requested
|
// Launch a goroutine to connect to the requested peer so we can
|
||||||
// peer so we can continue to handle queries.
|
// continue to handle queries.
|
||||||
|
//
|
||||||
// TODO(roasbeef): semaphore to limit the number of goroutines for
|
// TODO(roasbeef): semaphore to limit the number of goroutines for
|
||||||
// async requests.
|
// async requests.
|
||||||
go func() {
|
go func() {
|
||||||
// For the lndc crypto handshake, we
|
srvrLog.Debugf("connecting to %v", addr)
|
||||||
// either need a compressed pubkey, or a
|
|
||||||
// 20-byte pkh.
|
|
||||||
var remoteId []byte
|
|
||||||
if addr.PubKey == nil {
|
|
||||||
remoteId = addr.Base58Adr.ScriptAddress()
|
|
||||||
} else {
|
|
||||||
remoteId = addr.PubKey.SerializeCompressed()
|
|
||||||
}
|
|
||||||
|
|
||||||
srvrLog.Debugf("connecting to %v", hex.EncodeToString(remoteId))
|
// Attempt to connect to the remote node. If the we can't make
|
||||||
// Attempt to connect to the remote
|
// the connection, or the crypto negotation breaks down, then
|
||||||
// node. If the we can't make the
|
// return an error to the caller.
|
||||||
// connection, or the crypto negotation
|
conn, err := brontide.Dial(s.identityPriv, addr)
|
||||||
// breaks down, then return an error to the
|
if err != nil {
|
||||||
// caller.
|
|
||||||
ipAddr := addr.NetAddr.String()
|
|
||||||
conn := lndc.NewConn(nil)
|
|
||||||
if err := conn.Dial(
|
|
||||||
s.identityPriv, ipAddr, remoteId); err != nil {
|
|
||||||
msg.err <- err
|
msg.err <- err
|
||||||
msg.resp <- -1
|
msg.resp <- -1
|
||||||
return
|
return
|
||||||
}
|
}
|
||||||
|
|
||||||
// Now that we've established a connection,
|
// Now that we've established a connection, create a peer, and
|
||||||
// create a peer, and it to the set of
|
// it to the set of currently active peers.
|
||||||
// currently active peers.
|
peer, err := newPeer(conn, s, msg.addr, false)
|
||||||
peer, err := newPeer(conn, s, activeNetParams.Net, false)
|
|
||||||
if err != nil {
|
if err != nil {
|
||||||
srvrLog.Errorf("unable to create peer %v", err)
|
srvrLog.Errorf("unable to create peer %v", err)
|
||||||
conn.Close()
|
conn.Close()
|
||||||
@ -414,6 +402,9 @@ func (s *server) handleConnectPeer(msg *connectPeerMsg) {
|
|||||||
return
|
return
|
||||||
}
|
}
|
||||||
|
|
||||||
|
// TODO(roasbeef): update IP address for link-node
|
||||||
|
// * also mark last-seen, do it one single transaction?
|
||||||
|
|
||||||
peer.Start()
|
peer.Start()
|
||||||
s.newPeers <- peer
|
s.newPeers <- peer
|
||||||
|
|
||||||
@ -431,16 +422,17 @@ func (s *server) handleOpenChanReq(req *openChanReq) {
|
|||||||
var targetPeer *peer
|
var targetPeer *peer
|
||||||
for _, peer := range s.peers { // TODO(roasbeef): threadsafe api
|
for _, peer := range s.peers { // TODO(roasbeef): threadsafe api
|
||||||
// We found the the target
|
// We found the the target
|
||||||
if req.targetPeerID == peer.id ||
|
if peer.addr.IdentityKey.IsEqual(req.targetPubkey) ||
|
||||||
bytes.Equal(req.targetNodeID[:], peer.lightningID[:]) {
|
req.targetPeerID == peer.id {
|
||||||
targetPeer = peer
|
targetPeer = peer
|
||||||
break
|
break
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
if targetPeer == nil {
|
if targetPeer == nil {
|
||||||
req.err <- fmt.Errorf("unable to find peer lightningID(%v), "+
|
req.err <- fmt.Errorf("unable to find peer nodeID(%x), "+
|
||||||
"peerID(%v)", req.targetNodeID, req.targetPeerID)
|
"peerID(%v)", req.targetPubkey.SerializeCompressed(),
|
||||||
|
req.targetPeerID)
|
||||||
return
|
return
|
||||||
}
|
}
|
||||||
|
|
||||||
@ -455,7 +447,7 @@ func (s *server) handleOpenChanReq(req *openChanReq) {
|
|||||||
// ConnectToPeer requests that the server connect to a Lightning Network peer
|
// ConnectToPeer requests that the server connect to a Lightning Network peer
|
||||||
// at the specified address. This function will *block* until either a
|
// at the specified address. This function will *block* until either a
|
||||||
// connection is established, or the initial handshake process fails.
|
// connection is established, or the initial handshake process fails.
|
||||||
func (s *server) ConnectToPeer(addr *lndc.LNAdr) (int32, error) {
|
func (s *server) ConnectToPeer(addr *lnwire.NetAddress) (int32, error) {
|
||||||
reply := make(chan int32, 1)
|
reply := make(chan int32, 1)
|
||||||
errChan := make(chan error, 1)
|
errChan := make(chan error, 1)
|
||||||
|
|
||||||
@ -466,7 +458,8 @@ func (s *server) ConnectToPeer(addr *lndc.LNAdr) (int32, error) {
|
|||||||
|
|
||||||
// OpenChannel sends a request to the server to open a channel to the specified
|
// OpenChannel sends a request to the server to open a channel to the specified
|
||||||
// peer identified by ID with the passed channel funding paramters.
|
// peer identified by ID with the passed channel funding paramters.
|
||||||
func (s *server) OpenChannel(peerID int32, nodeID []byte, localAmt, remoteAmt btcutil.Amount,
|
func (s *server) OpenChannel(peerID int32, nodeKey *btcec.PublicKey,
|
||||||
|
localAmt, remoteAmt btcutil.Amount,
|
||||||
numConfs uint32) (chan *lnrpc.OpenStatusUpdate, chan error) {
|
numConfs uint32) (chan *lnrpc.OpenStatusUpdate, chan error) {
|
||||||
|
|
||||||
errChan := make(chan error, 1)
|
errChan := make(chan error, 1)
|
||||||
@ -474,13 +467,13 @@ func (s *server) OpenChannel(peerID int32, nodeID []byte, localAmt, remoteAmt bt
|
|||||||
|
|
||||||
req := &openChanReq{
|
req := &openChanReq{
|
||||||
targetPeerID: peerID,
|
targetPeerID: peerID,
|
||||||
|
targetPubkey: nodeKey,
|
||||||
localFundingAmt: localAmt,
|
localFundingAmt: localAmt,
|
||||||
remoteFundingAmt: remoteAmt,
|
remoteFundingAmt: remoteAmt,
|
||||||
numConfs: numConfs,
|
numConfs: numConfs,
|
||||||
updates: updateChan,
|
updates: updateChan,
|
||||||
err: errChan,
|
err: errChan,
|
||||||
}
|
}
|
||||||
copy(req.targetNodeID[:], nodeID)
|
|
||||||
|
|
||||||
s.queries <- req
|
s.queries <- req
|
||||||
|
|
||||||
@ -514,13 +507,24 @@ func (s *server) listener(l net.Listener) {
|
|||||||
}
|
}
|
||||||
|
|
||||||
srvrLog.Tracef("New inbound connection from %v", conn.RemoteAddr())
|
srvrLog.Tracef("New inbound connection from %v", conn.RemoteAddr())
|
||||||
peer, err := newPeer(conn, s, activeNetParams.Net, true)
|
|
||||||
|
brontideConn := conn.(*brontide.Conn)
|
||||||
|
peerAddr := &lnwire.NetAddress{
|
||||||
|
IdentityKey: brontideConn.RemotePub(),
|
||||||
|
Address: conn.RemoteAddr().(*net.TCPAddr),
|
||||||
|
ChainNet: activeNetParams.Net,
|
||||||
|
}
|
||||||
|
|
||||||
|
peer, err := newPeer(conn, s, peerAddr, true)
|
||||||
if err != nil {
|
if err != nil {
|
||||||
srvrLog.Errorf("unable to create peer: %v", err)
|
srvrLog.Errorf("unable to create peer: %v", err)
|
||||||
conn.Close()
|
conn.Close()
|
||||||
continue
|
continue
|
||||||
}
|
}
|
||||||
|
|
||||||
|
// TODO(roasbeef): update IP address for link-node
|
||||||
|
// * also mark last-seen, do it one single transaction?
|
||||||
|
|
||||||
peer.Start()
|
peer.Start()
|
||||||
s.newPeers <- peer
|
s.newPeers <- peer
|
||||||
}
|
}
|
||||||
|
Loading…
Reference in New Issue
Block a user