Browse Source

Merge pull request #1048 from tendermint/p2p-id

P2P ID
pull/1107/head
Ethan Buchman 7 years ago
committed by GitHub
parent
commit
8b3fb743cf
No known key found for this signature in database GPG Key ID: 4AEE18F83AFDEB23
35 changed files with 546 additions and 300 deletions
  1. +3
    -3
      benchmarks/codec_test.go
  2. +17
    -16
      blockchain/pool.go
  3. +8
    -7
      blockchain/pool_test.go
  4. +5
    -5
      blockchain/reactor.go
  5. +6
    -6
      blockchain/reactor_test.go
  6. +11
    -0
      config/config.go
  7. +3
    -0
      config/toml.go
  8. +5
    -5
      consensus/reactor.go
  9. +6
    -6
      consensus/replay.go
  10. +24
    -23
      consensus/state.go
  11. +8
    -7
      consensus/types/height_vote_set.go
  12. +1
    -1
      consensus/wal.go
  13. +13
    -10
      node/node.go
  14. +0
    -1
      p2p/README.md
  15. +19
    -15
      p2p/addrbook.go
  16. +6
    -2
      p2p/addrbook_test.go
  17. +0
    -6
      p2p/connection.go
  18. +113
    -0
      p2p/key.go
  19. +50
    -0
      p2p/key_test.go
  20. +52
    -27
      p2p/netaddress.go
  21. +19
    -2
      p2p/netaddress_test.go
  22. +17
    -18
      p2p/peer.go
  23. +12
    -12
      p2p/peer_set.go
  24. +7
    -6
      p2p/peer_set_test.go
  25. +14
    -13
      p2p/peer_test.go
  26. +23
    -34
      p2p/pex_reactor.go
  27. +5
    -4
      p2p/pex_reactor_test.go
  28. +8
    -8
      p2p/secret_connection.go
  29. +7
    -8
      p2p/secret_connection_test.go
  30. +46
    -34
      p2p/switch.go
  31. +7
    -7
      p2p/switch_test.go
  32. +23
    -8
      p2p/types.go
  33. +3
    -2
      rpc/core/consensus.go
  34. +1
    -1
      rpc/core/types/responses.go
  35. +4
    -3
      types/vote_set.go

+ 3
- 3
benchmarks/codec_test.go View File

@ -17,7 +17,7 @@ func BenchmarkEncodeStatusWire(b *testing.B) {
pubKey := crypto.GenPrivKeyEd25519().PubKey()
status := &ctypes.ResultStatus{
NodeInfo: &p2p.NodeInfo{
PubKey: pubKey.Unwrap().(crypto.PubKeyEd25519),
PubKey: pubKey,
Moniker: "SOMENAME",
Network: "SOMENAME",
RemoteAddr: "SOMEADDR",
@ -42,7 +42,7 @@ func BenchmarkEncodeStatusWire(b *testing.B) {
func BenchmarkEncodeNodeInfoWire(b *testing.B) {
b.StopTimer()
pubKey := crypto.GenPrivKeyEd25519().PubKey().Unwrap().(crypto.PubKeyEd25519)
pubKey := crypto.GenPrivKeyEd25519().PubKey()
nodeInfo := &p2p.NodeInfo{
PubKey: pubKey,
Moniker: "SOMENAME",
@ -63,7 +63,7 @@ func BenchmarkEncodeNodeInfoWire(b *testing.B) {
func BenchmarkEncodeNodeInfoBinary(b *testing.B) {
b.StopTimer()
pubKey := crypto.GenPrivKeyEd25519().PubKey().Unwrap().(crypto.PubKeyEd25519)
pubKey := crypto.GenPrivKeyEd25519().PubKey()
nodeInfo := &p2p.NodeInfo{
PubKey: pubKey,
Moniker: "SOMENAME",


+ 17
- 16
blockchain/pool.go View File

@ -5,6 +5,7 @@ import (
"sync"
"time"
"github.com/tendermint/tendermint/p2p"
"github.com/tendermint/tendermint/types"
cmn "github.com/tendermint/tmlibs/common"
flow "github.com/tendermint/tmlibs/flowrate"
@ -56,16 +57,16 @@ type BlockPool struct {
height int64 // the lowest key in requesters.
numPending int32 // number of requests pending assignment or block response
// peers
peers map[string]*bpPeer
peers map[p2p.ID]*bpPeer
maxPeerHeight int64
requestsCh chan<- BlockRequest
timeoutsCh chan<- string
timeoutsCh chan<- p2p.ID
}
func NewBlockPool(start int64, requestsCh chan<- BlockRequest, timeoutsCh chan<- string) *BlockPool {
func NewBlockPool(start int64, requestsCh chan<- BlockRequest, timeoutsCh chan<- p2p.ID) *BlockPool {
bp := &BlockPool{
peers: make(map[string]*bpPeer),
peers: make(map[p2p.ID]*bpPeer),
requesters: make(map[int64]*bpRequester),
height: start,
@ -210,7 +211,7 @@ func (pool *BlockPool) RedoRequest(height int64) {
}
// TODO: ensure that blocks come in order for each peer.
func (pool *BlockPool) AddBlock(peerID string, block *types.Block, blockSize int) {
func (pool *BlockPool) AddBlock(peerID p2p.ID, block *types.Block, blockSize int) {
pool.mtx.Lock()
defer pool.mtx.Unlock()
@ -240,7 +241,7 @@ func (pool *BlockPool) MaxPeerHeight() int64 {
}
// Sets the peer's alleged blockchain height.
func (pool *BlockPool) SetPeerHeight(peerID string, height int64) {
func (pool *BlockPool) SetPeerHeight(peerID p2p.ID, height int64) {
pool.mtx.Lock()
defer pool.mtx.Unlock()
@ -258,14 +259,14 @@ func (pool *BlockPool) SetPeerHeight(peerID string, height int64) {
}
}
func (pool *BlockPool) RemovePeer(peerID string) {
func (pool *BlockPool) RemovePeer(peerID p2p.ID) {
pool.mtx.Lock()
defer pool.mtx.Unlock()
pool.removePeer(peerID)
}
func (pool *BlockPool) removePeer(peerID string) {
func (pool *BlockPool) removePeer(peerID p2p.ID) {
for _, requester := range pool.requesters {
if requester.getPeerID() == peerID {
if requester.getBlock() != nil {
@ -321,14 +322,14 @@ func (pool *BlockPool) requestersLen() int64 {
return int64(len(pool.requesters))
}
func (pool *BlockPool) sendRequest(height int64, peerID string) {
func (pool *BlockPool) sendRequest(height int64, peerID p2p.ID) {
if !pool.IsRunning() {
return
}
pool.requestsCh <- BlockRequest{height, peerID}
}
func (pool *BlockPool) sendTimeout(peerID string) {
func (pool *BlockPool) sendTimeout(peerID p2p.ID) {
if !pool.IsRunning() {
return
}
@ -357,7 +358,7 @@ func (pool *BlockPool) debug() string {
type bpPeer struct {
pool *BlockPool
id string
id p2p.ID
recvMonitor *flow.Monitor
height int64
@ -368,7 +369,7 @@ type bpPeer struct {
logger log.Logger
}
func newBPPeer(pool *BlockPool, peerID string, height int64) *bpPeer {
func newBPPeer(pool *BlockPool, peerID p2p.ID, height int64) *bpPeer {
peer := &bpPeer{
pool: pool,
id: peerID,
@ -434,7 +435,7 @@ type bpRequester struct {
redoCh chan struct{}
mtx sync.Mutex
peerID string
peerID p2p.ID
block *types.Block
}
@ -458,7 +459,7 @@ func (bpr *bpRequester) OnStart() error {
}
// Returns true if the peer matches
func (bpr *bpRequester) setBlock(block *types.Block, peerID string) bool {
func (bpr *bpRequester) setBlock(block *types.Block, peerID p2p.ID) bool {
bpr.mtx.Lock()
if bpr.block != nil || bpr.peerID != peerID {
bpr.mtx.Unlock()
@ -477,7 +478,7 @@ func (bpr *bpRequester) getBlock() *types.Block {
return bpr.block
}
func (bpr *bpRequester) getPeerID() string {
func (bpr *bpRequester) getPeerID() p2p.ID {
bpr.mtx.Lock()
defer bpr.mtx.Unlock()
return bpr.peerID
@ -551,5 +552,5 @@ OUTER_LOOP:
type BlockRequest struct {
Height int64
PeerID string
PeerID p2p.ID
}

+ 8
- 7
blockchain/pool_test.go View File

@ -5,6 +5,7 @@ import (
"testing"
"time"
"github.com/tendermint/tendermint/p2p"
"github.com/tendermint/tendermint/types"
cmn "github.com/tendermint/tmlibs/common"
"github.com/tendermint/tmlibs/log"
@ -15,14 +16,14 @@ func init() {
}
type testPeer struct {
id string
id p2p.ID
height int64
}
func makePeers(numPeers int, minHeight, maxHeight int64) map[string]testPeer {
peers := make(map[string]testPeer, numPeers)
func makePeers(numPeers int, minHeight, maxHeight int64) map[p2p.ID]testPeer {
peers := make(map[p2p.ID]testPeer, numPeers)
for i := 0; i < numPeers; i++ {
peerID := cmn.RandStr(12)
peerID := p2p.ID(cmn.RandStr(12))
height := minHeight + rand.Int63n(maxHeight-minHeight)
peers[peerID] = testPeer{peerID, height}
}
@ -32,7 +33,7 @@ func makePeers(numPeers int, minHeight, maxHeight int64) map[string]testPeer {
func TestBasic(t *testing.T) {
start := int64(42)
peers := makePeers(10, start+1, 1000)
timeoutsCh := make(chan string, 100)
timeoutsCh := make(chan p2p.ID, 100)
requestsCh := make(chan BlockRequest, 100)
pool := NewBlockPool(start, requestsCh, timeoutsCh)
pool.SetLogger(log.TestingLogger())
@ -89,7 +90,7 @@ func TestBasic(t *testing.T) {
func TestTimeout(t *testing.T) {
start := int64(42)
peers := makePeers(10, start+1, 1000)
timeoutsCh := make(chan string, 100)
timeoutsCh := make(chan p2p.ID, 100)
requestsCh := make(chan BlockRequest, 100)
pool := NewBlockPool(start, requestsCh, timeoutsCh)
pool.SetLogger(log.TestingLogger())
@ -127,7 +128,7 @@ func TestTimeout(t *testing.T) {
// Pull from channels
counter := 0
timedOut := map[string]struct{}{}
timedOut := map[p2p.ID]struct{}{}
for {
select {
case peerID := <-timeoutsCh:


+ 5
- 5
blockchain/reactor.go View File

@ -52,7 +52,7 @@ type BlockchainReactor struct {
pool *BlockPool
fastSync bool
requestsCh chan BlockRequest
timeoutsCh chan string
timeoutsCh chan p2p.ID
}
// NewBlockchainReactor returns new reactor instance.
@ -62,7 +62,7 @@ func NewBlockchainReactor(state sm.State, blockExec *sm.BlockExecutor, store *Bl
}
requestsCh := make(chan BlockRequest, defaultChannelCapacity)
timeoutsCh := make(chan string, defaultChannelCapacity)
timeoutsCh := make(chan p2p.ID, defaultChannelCapacity)
pool := NewBlockPool(
store.Height()+1,
requestsCh,
@ -131,7 +131,7 @@ func (bcR *BlockchainReactor) AddPeer(peer p2p.Peer) {
// RemovePeer implements Reactor by removing peer from the pool.
func (bcR *BlockchainReactor) RemovePeer(peer p2p.Peer, reason interface{}) {
bcR.pool.RemovePeer(peer.Key())
bcR.pool.RemovePeer(peer.ID())
}
// respondToPeer loads a block and sends it to the requesting peer,
@ -170,7 +170,7 @@ func (bcR *BlockchainReactor) Receive(chID byte, src p2p.Peer, msgBytes []byte)
}
case *bcBlockResponseMessage:
// Got a block.
bcR.pool.AddBlock(src.Key(), msg.Block, len(msgBytes))
bcR.pool.AddBlock(src.ID(), msg.Block, len(msgBytes))
case *bcStatusRequestMessage:
// Send peer our state.
queued := src.TrySend(BlockchainChannel, struct{ BlockchainMessage }{&bcStatusResponseMessage{bcR.store.Height()}})
@ -179,7 +179,7 @@ func (bcR *BlockchainReactor) Receive(chID byte, src p2p.Peer, msgBytes []byte)
}
case *bcStatusResponseMessage:
// Got a peer status. Unverified.
bcR.pool.SetPeerHeight(src.Key(), msg.Height)
bcR.pool.SetPeerHeight(src.ID(), msg.Height)
default:
bcR.Logger.Error(cmn.Fmt("Unknown message type %v", reflect.TypeOf(msg)))
}


+ 6
- 6
blockchain/reactor_test.go View File

@ -55,7 +55,7 @@ func TestNoBlockMessageResponse(t *testing.T) {
defer bcr.Stop()
// Add some peers in
peer := newbcrTestPeer(cmn.RandStr(12))
peer := newbcrTestPeer(p2p.ID(cmn.RandStr(12)))
bcr.AddPeer(peer)
chID := byte(0x01)
@ -113,16 +113,16 @@ func makeBlock(height int64, state sm.State) *types.Block {
// The Test peer
type bcrTestPeer struct {
cmn.Service
key string
ch chan interface{}
id p2p.ID
ch chan interface{}
}
var _ p2p.Peer = (*bcrTestPeer)(nil)
func newbcrTestPeer(key string) *bcrTestPeer {
func newbcrTestPeer(id p2p.ID) *bcrTestPeer {
return &bcrTestPeer{
Service: cmn.NewBaseService(nil, "bcrTestPeer", nil),
key: key,
id: id,
ch: make(chan interface{}, 2),
}
}
@ -144,7 +144,7 @@ func (tp *bcrTestPeer) TrySend(chID byte, value interface{}) bool {
func (tp *bcrTestPeer) Send(chID byte, data interface{}) bool { return tp.TrySend(chID, data) }
func (tp *bcrTestPeer) NodeInfo() *p2p.NodeInfo { return nil }
func (tp *bcrTestPeer) Status() p2p.ConnectionStatus { return p2p.ConnectionStatus{} }
func (tp *bcrTestPeer) Key() string { return tp.key }
func (tp *bcrTestPeer) ID() p2p.ID { return tp.id }
func (tp *bcrTestPeer) IsOutbound() bool { return false }
func (tp *bcrTestPeer) IsPersistent() bool { return true }
func (tp *bcrTestPeer) Get(s string) interface{} { return s }


+ 11
- 0
config/config.go View File

@ -20,10 +20,12 @@ var (
defaultConfigFileName = "config.toml"
defaultGenesisJSONName = "genesis.json"
defaultPrivValName = "priv_validator.json"
defaultNodeKeyName = "node_key.json"
defaultConfigFilePath = filepath.Join(defaultConfigDir, defaultConfigFileName)
defaultGenesisJSONPath = filepath.Join(defaultConfigDir, defaultGenesisJSONName)
defaultPrivValPath = filepath.Join(defaultConfigDir, defaultPrivValName)
defaultNodeKeyPath = filepath.Join(defaultConfigDir, defaultNodeKeyName)
)
// Config defines the top level configuration for a Tendermint node
@ -92,6 +94,9 @@ type BaseConfig struct {
// Path to the JSON file containing the private key to use as a validator in the consensus protocol
PrivValidator string `mapstructure:"priv_validator_file"`
// A JSON file containing the private key to use for p2p authenticated encryption
NodeKey string `mapstructure:"node_key_file"`
// A custom human readable name for this node
Moniker string `mapstructure:"moniker"`
@ -133,6 +138,7 @@ func DefaultBaseConfig() BaseConfig {
return BaseConfig{
Genesis: defaultGenesisJSONPath,
PrivValidator: defaultPrivValPath,
NodeKey: defaultNodeKeyPath,
Moniker: defaultMoniker,
ProxyApp: "tcp://127.0.0.1:46658",
ABCI: "socket",
@ -165,6 +171,11 @@ func (b BaseConfig) PrivValidatorFile() string {
return rootify(b.PrivValidator, b.RootDir)
}
// NodeKeyFile returns the full path to the node_key.json file
func (b BaseConfig) NodeKeyFile() string {
return rootify(b.NodeKey, b.RootDir)
}
// DBDir returns the full path to the database directory
func (b BaseConfig) DBDir() string {
return rootify(b.DBPath, b.RootDir)


+ 3
- 0
config/toml.go View File

@ -87,6 +87,9 @@ genesis_file = "{{ .BaseConfig.Genesis }}"
# Path to the JSON file containing the private key to use as a validator in the consensus protocol
priv_validator_file = "{{ .BaseConfig.PrivValidator }}"
# Path to the JSON file containing the private key to use for node authentication in the p2p protocol
node_key_file = "{{ .BaseConfig.NodeKey}}"
# Mechanism to connect to the ABCI application: socket | grpc
abci = "{{ .BaseConfig.ABCI }}"


+ 5
- 5
consensus/reactor.go View File

@ -205,7 +205,7 @@ func (conR *ConsensusReactor) Receive(chID byte, src p2p.Peer, msgBytes []byte)
return
}
// Peer claims to have a maj23 for some BlockID at H,R,S,
votes.SetPeerMaj23(msg.Round, msg.Type, ps.Peer.Key(), msg.BlockID)
votes.SetPeerMaj23(msg.Round, msg.Type, ps.Peer.ID(), msg.BlockID)
// Respond with a VoteSetBitsMessage showing which votes we have.
// (and consequently shows which we don't have)
var ourVotes *cmn.BitArray
@ -242,12 +242,12 @@ func (conR *ConsensusReactor) Receive(chID byte, src p2p.Peer, msgBytes []byte)
switch msg := msg.(type) {
case *ProposalMessage:
ps.SetHasProposal(msg.Proposal)
conR.conS.peerMsgQueue <- msgInfo{msg, src.Key()}
conR.conS.peerMsgQueue <- msgInfo{msg, src.ID()}
case *ProposalPOLMessage:
ps.ApplyProposalPOLMessage(msg)
case *BlockPartMessage:
ps.SetHasProposalBlockPart(msg.Height, msg.Round, msg.Part.Index)
conR.conS.peerMsgQueue <- msgInfo{msg, src.Key()}
conR.conS.peerMsgQueue <- msgInfo{msg, src.ID()}
default:
conR.Logger.Error(cmn.Fmt("Unknown message type %v", reflect.TypeOf(msg)))
}
@ -267,7 +267,7 @@ func (conR *ConsensusReactor) Receive(chID byte, src p2p.Peer, msgBytes []byte)
ps.EnsureVoteBitArrays(height-1, lastCommitSize)
ps.SetHasVote(msg.Vote)
cs.peerMsgQueue <- msgInfo{msg, src.Key()}
cs.peerMsgQueue <- msgInfo{msg, src.ID()}
default:
// don't punish (leave room for soft upgrades)
@ -1200,7 +1200,7 @@ func (ps *PeerState) StringIndented(indent string) string {
%s Key %v
%s PRS %v
%s}`,
indent, ps.Peer.Key(),
indent, ps.Peer.ID(),
indent, ps.PeerRoundState.StringIndented(indent+" "),
indent)
}


+ 6
- 6
consensus/replay.go View File

@ -61,21 +61,21 @@ func (cs *ConsensusState) readReplayMessage(msg *TimedWALMessage, newStepCh chan
}
}
case msgInfo:
peerKey := m.PeerKey
if peerKey == "" {
peerKey = "local"
peerID := m.PeerID
if peerID == "" {
peerID = "local"
}
switch msg := m.Msg.(type) {
case *ProposalMessage:
p := msg.Proposal
cs.Logger.Info("Replay: Proposal", "height", p.Height, "round", p.Round, "header",
p.BlockPartsHeader, "pol", p.POLRound, "peer", peerKey)
p.BlockPartsHeader, "pol", p.POLRound, "peer", peerID)
case *BlockPartMessage:
cs.Logger.Info("Replay: BlockPart", "height", msg.Height, "round", msg.Round, "peer", peerKey)
cs.Logger.Info("Replay: BlockPart", "height", msg.Height, "round", msg.Round, "peer", peerID)
case *VoteMessage:
v := msg.Vote
cs.Logger.Info("Replay: Vote", "height", v.Height, "round", v.Round, "type", v.Type,
"blockID", v.BlockID, "peer", peerKey)
"blockID", v.BlockID, "peer", peerID)
}
cs.handleMsg(m)


+ 24
- 23
consensus/state.go View File

@ -17,6 +17,7 @@ import (
cfg "github.com/tendermint/tendermint/config"
cstypes "github.com/tendermint/tendermint/consensus/types"
"github.com/tendermint/tendermint/p2p"
sm "github.com/tendermint/tendermint/state"
"github.com/tendermint/tendermint/types"
)
@ -46,8 +47,8 @@ var (
// msgs from the reactor which may update the state
type msgInfo struct {
Msg ConsensusMessage `json:"msg"`
PeerKey string `json:"peer_key"`
Msg ConsensusMessage `json:"msg"`
PeerID p2p.ID `json:"peer_key"`
}
// internally generated messages which may update the state
@ -303,17 +304,17 @@ func (cs *ConsensusState) OpenWAL(walFile string) (WAL, error) {
//------------------------------------------------------------
// Public interface for passing messages into the consensus state, possibly causing a state transition.
// If peerKey == "", the msg is considered internal.
// If peerID == "", the msg is considered internal.
// Messages are added to the appropriate queue (peer or internal).
// If the queue is full, the function may block.
// TODO: should these return anything or let callers just use events?
// AddVote inputs a vote.
func (cs *ConsensusState) AddVote(vote *types.Vote, peerKey string) (added bool, err error) {
if peerKey == "" {
func (cs *ConsensusState) AddVote(vote *types.Vote, peerID p2p.ID) (added bool, err error) {
if peerID == "" {
cs.internalMsgQueue <- msgInfo{&VoteMessage{vote}, ""}
} else {
cs.peerMsgQueue <- msgInfo{&VoteMessage{vote}, peerKey}
cs.peerMsgQueue <- msgInfo{&VoteMessage{vote}, peerID}
}
// TODO: wait for event?!
@ -321,12 +322,12 @@ func (cs *ConsensusState) AddVote(vote *types.Vote, peerKey string) (added bool,
}
// SetProposal inputs a proposal.
func (cs *ConsensusState) SetProposal(proposal *types.Proposal, peerKey string) error {
func (cs *ConsensusState) SetProposal(proposal *types.Proposal, peerID p2p.ID) error {
if peerKey == "" {
if peerID == "" {
cs.internalMsgQueue <- msgInfo{&ProposalMessage{proposal}, ""}
} else {
cs.peerMsgQueue <- msgInfo{&ProposalMessage{proposal}, peerKey}
cs.peerMsgQueue <- msgInfo{&ProposalMessage{proposal}, peerID}
}
// TODO: wait for event?!
@ -334,12 +335,12 @@ func (cs *ConsensusState) SetProposal(proposal *types.Proposal, peerKey string)
}
// AddProposalBlockPart inputs a part of the proposal block.
func (cs *ConsensusState) AddProposalBlockPart(height int64, round int, part *types.Part, peerKey string) error {
func (cs *ConsensusState) AddProposalBlockPart(height int64, round int, part *types.Part, peerID p2p.ID) error {
if peerKey == "" {
if peerID == "" {
cs.internalMsgQueue <- msgInfo{&BlockPartMessage{height, round, part}, ""}
} else {
cs.peerMsgQueue <- msgInfo{&BlockPartMessage{height, round, part}, peerKey}
cs.peerMsgQueue <- msgInfo{&BlockPartMessage{height, round, part}, peerID}
}
// TODO: wait for event?!
@ -347,13 +348,13 @@ func (cs *ConsensusState) AddProposalBlockPart(height int64, round int, part *ty
}
// SetProposalAndBlock inputs the proposal and all block parts.
func (cs *ConsensusState) SetProposalAndBlock(proposal *types.Proposal, block *types.Block, parts *types.PartSet, peerKey string) error {
if err := cs.SetProposal(proposal, peerKey); err != nil {
func (cs *ConsensusState) SetProposalAndBlock(proposal *types.Proposal, block *types.Block, parts *types.PartSet, peerID p2p.ID) error {
if err := cs.SetProposal(proposal, peerID); err != nil {
return err
}
for i := 0; i < parts.Total(); i++ {
part := parts.GetPart(i)
if err := cs.AddProposalBlockPart(proposal.Height, proposal.Round, part, peerKey); err != nil {
if err := cs.AddProposalBlockPart(proposal.Height, proposal.Round, part, peerID); err != nil {
return err
}
}
@ -561,7 +562,7 @@ func (cs *ConsensusState) handleMsg(mi msgInfo) {
defer cs.mtx.Unlock()
var err error
msg, peerKey := mi.Msg, mi.PeerKey
msg, peerID := mi.Msg, mi.PeerID
switch msg := msg.(type) {
case *ProposalMessage:
// will not cause transition.
@ -569,14 +570,14 @@ func (cs *ConsensusState) handleMsg(mi msgInfo) {
err = cs.setProposal(msg.Proposal)
case *BlockPartMessage:
// if the proposal is complete, we'll enterPrevote or tryFinalizeCommit
_, err = cs.addProposalBlockPart(msg.Height, msg.Part, peerKey != "")
_, err = cs.addProposalBlockPart(msg.Height, msg.Part, peerID != "")
if err != nil && msg.Round != cs.Round {
err = nil
}
case *VoteMessage:
// attempt to add the vote and dupeout the validator if its a duplicate signature
// if the vote gives us a 2/3-any or 2/3-one, we transition
err := cs.tryAddVote(msg.Vote, peerKey)
err := cs.tryAddVote(msg.Vote, peerID)
if err == ErrAddingVote {
// TODO: punish peer
}
@ -591,7 +592,7 @@ func (cs *ConsensusState) handleMsg(mi msgInfo) {
cs.Logger.Error("Unknown msg type", reflect.TypeOf(msg))
}
if err != nil {
cs.Logger.Error("Error with msg", "type", reflect.TypeOf(msg), "peer", peerKey, "err", err, "msg", msg)
cs.Logger.Error("Error with msg", "type", reflect.TypeOf(msg), "peer", peerID, "err", err, "msg", msg)
}
}
@ -1308,8 +1309,8 @@ func (cs *ConsensusState) addProposalBlockPart(height int64, part *types.Part, v
}
// Attempt to add the vote. if its a duplicate signature, dupeout the validator
func (cs *ConsensusState) tryAddVote(vote *types.Vote, peerKey string) error {
_, err := cs.addVote(vote, peerKey)
func (cs *ConsensusState) tryAddVote(vote *types.Vote, peerID p2p.ID) error {
_, err := cs.addVote(vote, peerID)
if err != nil {
// If the vote height is off, we'll just ignore it,
// But if it's a conflicting sig, add it to the cs.evpool.
@ -1335,7 +1336,7 @@ func (cs *ConsensusState) tryAddVote(vote *types.Vote, peerKey string) error {
//-----------------------------------------------------------------------------
func (cs *ConsensusState) addVote(vote *types.Vote, peerKey string) (added bool, err error) {
func (cs *ConsensusState) addVote(vote *types.Vote, peerID p2p.ID) (added bool, err error) {
cs.Logger.Debug("addVote", "voteHeight", vote.Height, "voteType", vote.Type, "valIndex", vote.ValidatorIndex, "csHeight", cs.Height)
// A precommit for the previous height?
@ -1365,7 +1366,7 @@ func (cs *ConsensusState) addVote(vote *types.Vote, peerKey string) (added bool,
// A prevote/precommit for this height?
if vote.Height == cs.Height {
height := cs.Height
added, err = cs.Votes.AddVote(vote, peerKey)
added, err = cs.Votes.AddVote(vote, peerID)
if added {
cs.eventBus.PublishEventVote(types.EventDataVote{vote})


+ 8
- 7
consensus/types/height_vote_set.go View File

@ -4,6 +4,7 @@ import (
"strings"
"sync"
"github.com/tendermint/tendermint/p2p"
"github.com/tendermint/tendermint/types"
cmn "github.com/tendermint/tmlibs/common"
)
@ -35,7 +36,7 @@ type HeightVoteSet struct {
mtx sync.Mutex
round int // max tracked round
roundVoteSets map[int]RoundVoteSet // keys: [0...round]
peerCatchupRounds map[string][]int // keys: peer.Key; values: at most 2 rounds
peerCatchupRounds map[p2p.ID][]int // keys: peer.ID; values: at most 2 rounds
}
func NewHeightVoteSet(chainID string, height int64, valSet *types.ValidatorSet) *HeightVoteSet {
@ -53,7 +54,7 @@ func (hvs *HeightVoteSet) Reset(height int64, valSet *types.ValidatorSet) {
hvs.height = height
hvs.valSet = valSet
hvs.roundVoteSets = make(map[int]RoundVoteSet)
hvs.peerCatchupRounds = make(map[string][]int)
hvs.peerCatchupRounds = make(map[p2p.ID][]int)
hvs.addRound(0)
hvs.round = 0
@ -101,8 +102,8 @@ func (hvs *HeightVoteSet) addRound(round int) {
}
// Duplicate votes return added=false, err=nil.
// By convention, peerKey is "" if origin is self.
func (hvs *HeightVoteSet) AddVote(vote *types.Vote, peerKey string) (added bool, err error) {
// By convention, peerID is "" if origin is self.
func (hvs *HeightVoteSet) AddVote(vote *types.Vote, peerID p2p.ID) (added bool, err error) {
hvs.mtx.Lock()
defer hvs.mtx.Unlock()
if !types.IsVoteTypeValid(vote.Type) {
@ -110,10 +111,10 @@ func (hvs *HeightVoteSet) AddVote(vote *types.Vote, peerKey string) (added bool,
}
voteSet := hvs.getVoteSet(vote.Round, vote.Type)
if voteSet == nil {
if rndz := hvs.peerCatchupRounds[peerKey]; len(rndz) < 2 {
if rndz := hvs.peerCatchupRounds[peerID]; len(rndz) < 2 {
hvs.addRound(vote.Round)
voteSet = hvs.getVoteSet(vote.Round, vote.Type)
hvs.peerCatchupRounds[peerKey] = append(rndz, vote.Round)
hvs.peerCatchupRounds[peerID] = append(rndz, vote.Round)
} else {
// Peer has sent a vote that does not match our round,
// for more than one round. Bad peer!
@ -206,7 +207,7 @@ func (hvs *HeightVoteSet) StringIndented(indent string) string {
// NOTE: if there are too many peers, or too much peer churn,
// this can cause memory issues.
// TODO: implement ability to remove peers too
func (hvs *HeightVoteSet) SetPeerMaj23(round int, type_ byte, peerID string, blockID types.BlockID) {
func (hvs *HeightVoteSet) SetPeerMaj23(round int, type_ byte, peerID p2p.ID, blockID types.BlockID) {
hvs.mtx.Lock()
defer hvs.mtx.Unlock()
if !types.IsVoteTypeValid(type_) {


+ 1
- 1
consensus/wal.go View File

@ -121,7 +121,7 @@ func (wal *baseWAL) Save(msg WALMessage) {
if wal.light {
// in light mode we only write new steps, timeouts, and our own votes (no proposals, block parts)
if mi, ok := msg.(msgInfo); ok {
if mi.PeerKey != "" {
if mi.PeerID != "" {
return
}
}


+ 13
- 10
node/node.go View File

@ -96,7 +96,6 @@ type Node struct {
privValidator types.PrivValidator // local node's validator key
// network
privKey crypto.PrivKeyEd25519 // local node's p2p key
sw *p2p.Switch // p2p connections
addrBook *p2p.AddrBook // known peers
trustMetricStore *trust.TrustMetricStore // trust metrics for all peers
@ -170,9 +169,6 @@ func NewNode(config *cfg.Config,
// reload the state (it may have been updated by the handshake)
state = sm.LoadState(stateDB)
// Generate node PrivKey
privKey := crypto.GenPrivKeyEd25519()
// Decide whether to fast-sync or not
// We don't fast-sync when the only validator is us.
fastSync := config.FastSync
@ -276,7 +272,7 @@ func NewNode(config *cfg.Config,
}
return nil
})
sw.SetPubKeyFilter(func(pubkey crypto.PubKeyEd25519) error {
sw.SetPubKeyFilter(func(pubkey crypto.PubKey) error {
resQuery, err := proxyApp.Query().QuerySync(abci.RequestQuery{Path: cmn.Fmt("/p2p/filter/pubkey/%X", pubkey.Bytes())})
if err != nil {
return err
@ -329,7 +325,6 @@ func NewNode(config *cfg.Config,
genesisDoc: genDoc,
privValidator: privValidator,
privKey: privKey,
sw: sw,
addrBook: addrBook,
trustMetricStore: trustMetricStore,
@ -372,9 +367,17 @@ func (n *Node) OnStart() error {
l := p2p.NewDefaultListener(protocol, address, n.config.P2P.SkipUPNP, n.Logger.With("module", "p2p"))
n.sw.AddListener(l)
// Generate node PrivKey
// TODO: pass in like priv_val
nodeKey, err := p2p.LoadOrGenNodeKey(n.config.NodeKeyFile())
if err != nil {
return err
}
n.Logger.Info("P2P Node ID", "ID", nodeKey.ID(), "file", n.config.NodeKeyFile())
// Start the switch
n.sw.SetNodeInfo(n.makeNodeInfo())
n.sw.SetNodePrivKey(n.privKey)
n.sw.SetNodeInfo(n.makeNodeInfo(nodeKey.PubKey()))
n.sw.SetNodeKey(nodeKey)
err = n.sw.Start()
if err != nil {
return err
@ -534,13 +537,13 @@ func (n *Node) ProxyApp() proxy.AppConns {
return n.proxyApp
}
func (n *Node) makeNodeInfo() *p2p.NodeInfo {
func (n *Node) makeNodeInfo(pubKey crypto.PubKey) *p2p.NodeInfo {
txIndexerStatus := "on"
if _, ok := n.txIndexer.(*null.TxIndex); ok {
txIndexerStatus = "off"
}
nodeInfo := &p2p.NodeInfo{
PubKey: n.privKey.PubKey().Unwrap().(crypto.PubKeyEd25519),
PubKey: pubKey,
Moniker: n.config.Moniker,
Network: n.genesisDoc.ChainID,
Version: version.Version,


+ 0
- 1
p2p/README.md View File

@ -11,4 +11,3 @@ See:
- [docs/node] for details about different types of nodes and how they should work
- [docs/pex] for details on peer discovery and exchange
- [docs/config] for details on some config options

+ 19
- 15
p2p/addrbook.go View File

@ -89,7 +89,7 @@ type AddrBook struct {
mtx sync.Mutex
rand *rand.Rand
ourAddrs map[string]*NetAddress
addrLookup map[string]*knownAddress // new & old
addrLookup map[ID]*knownAddress // new & old
bucketsOld []map[string]*knownAddress
bucketsNew []map[string]*knownAddress
nOld int
@ -104,7 +104,7 @@ func NewAddrBook(filePath string, routabilityStrict bool) *AddrBook {
am := &AddrBook{
rand: rand.New(rand.NewSource(time.Now().UnixNano())),
ourAddrs: make(map[string]*NetAddress),
addrLookup: make(map[string]*knownAddress),
addrLookup: make(map[ID]*knownAddress),
filePath: filePath,
routabilityStrict: routabilityStrict,
}
@ -244,11 +244,11 @@ func (a *AddrBook) PickAddress(newBias int) *NetAddress {
}
// MarkGood marks the peer as good and moves it into an "old" bucket.
// XXX: we never call this!
// TODO: call this from somewhere
func (a *AddrBook) MarkGood(addr *NetAddress) {
a.mtx.Lock()
defer a.mtx.Unlock()
ka := a.addrLookup[addr.String()]
ka := a.addrLookup[addr.ID]
if ka == nil {
return
}
@ -262,7 +262,7 @@ func (a *AddrBook) MarkGood(addr *NetAddress) {
func (a *AddrBook) MarkAttempt(addr *NetAddress) {
a.mtx.Lock()
defer a.mtx.Unlock()
ka := a.addrLookup[addr.String()]
ka := a.addrLookup[addr.ID]
if ka == nil {
return
}
@ -279,11 +279,11 @@ func (a *AddrBook) MarkBad(addr *NetAddress) {
func (a *AddrBook) RemoveAddress(addr *NetAddress) {
a.mtx.Lock()
defer a.mtx.Unlock()
ka := a.addrLookup[addr.String()]
ka := a.addrLookup[addr.ID]
if ka == nil {
return
}
a.Logger.Info("Remove address from book", "addr", addr)
a.Logger.Info("Remove address from book", "addr", ka.Addr, "ID", ka.ID)
a.removeFromAllBuckets(ka)
}
@ -300,8 +300,8 @@ func (a *AddrBook) GetSelection() []*NetAddress {
allAddr := make([]*NetAddress, a.size())
i := 0
for _, v := range a.addrLookup {
allAddr[i] = v.Addr
for _, ka := range a.addrLookup {
allAddr[i] = ka.Addr
i++
}
@ -388,7 +388,7 @@ func (a *AddrBook) loadFromFile(filePath string) bool {
bucket := a.getBucket(ka.BucketType, bucketIndex)
bucket[ka.Addr.String()] = ka
}
a.addrLookup[ka.Addr.String()] = ka
a.addrLookup[ka.ID()] = ka
if ka.BucketType == bucketTypeNew {
a.nNew++
} else {
@ -466,7 +466,7 @@ func (a *AddrBook) addToNewBucket(ka *knownAddress, bucketIdx int) bool {
}
// Ensure in addrLookup
a.addrLookup[addrStr] = ka
a.addrLookup[ka.ID()] = ka
return true
}
@ -503,7 +503,7 @@ func (a *AddrBook) addToOldBucket(ka *knownAddress, bucketIdx int) bool {
}
// Ensure in addrLookup
a.addrLookup[addrStr] = ka
a.addrLookup[ka.ID()] = ka
return true
}
@ -521,7 +521,7 @@ func (a *AddrBook) removeFromBucket(ka *knownAddress, bucketType byte, bucketIdx
} else {
a.nOld--
}
delete(a.addrLookup, ka.Addr.String())
delete(a.addrLookup, ka.ID())
}
}
@ -536,7 +536,7 @@ func (a *AddrBook) removeFromAllBuckets(ka *knownAddress) {
} else {
a.nOld--
}
delete(a.addrLookup, ka.Addr.String())
delete(a.addrLookup, ka.ID())
}
func (a *AddrBook) pickOldest(bucketType byte, bucketIdx int) *knownAddress {
@ -559,7 +559,7 @@ func (a *AddrBook) addAddress(addr, src *NetAddress) error {
return fmt.Errorf("Cannot add ourselves with address %v", addr)
}
ka := a.addrLookup[addr.String()]
ka := a.addrLookup[addr.ID]
if ka != nil {
// Already old.
@ -768,6 +768,10 @@ func newKnownAddress(addr *NetAddress, src *NetAddress) *knownAddress {
}
}
func (ka *knownAddress) ID() ID {
return ka.Addr.ID
}
func (ka *knownAddress) isOld() bool {
return ka.BucketType == bucketTypeOld
}


+ 6
- 2
p2p/addrbook_test.go View File

@ -1,12 +1,14 @@
package p2p
import (
"encoding/hex"
"fmt"
"io/ioutil"
"math/rand"
"testing"
"github.com/stretchr/testify/assert"
cmn "github.com/tendermint/tmlibs/common"
"github.com/tendermint/tmlibs/log"
)
@ -102,7 +104,7 @@ func TestAddrBookLookup(t *testing.T) {
src := addrSrc.src
book.AddAddress(addr, src)
ka := book.addrLookup[addr.String()]
ka := book.addrLookup[addr.ID]
assert.NotNil(t, ka, "Expected to find KnownAddress %v but wasn't there.", addr)
if !(ka.Addr.Equals(addr) && ka.Src.Equals(src)) {
@ -187,7 +189,9 @@ func randIPv4Address(t *testing.T) *NetAddress {
rand.Intn(255),
)
port := rand.Intn(65535-1) + 1
addr, err := NewNetAddressString(fmt.Sprintf("%v:%v", ip, port))
id := ID(hex.EncodeToString(cmn.RandBytes(IDByteLength)))
idAddr := IDAddressString(id, fmt.Sprintf("%v:%v", ip, port))
addr, err := NewNetAddressString(idAddr)
assert.Nil(t, err, "error generating rand network address")
if addr.Routable() {
return addr


+ 0
- 6
p2p/connection.go View File

@ -88,9 +88,6 @@ type MConnection struct {
flushTimer *cmn.ThrottleTimer // flush writes as necessary but throttled.
pingTimer *cmn.RepeatTimer // send pings periodically
chStatsTimer *cmn.RepeatTimer // update channel stats periodically
LocalAddress *NetAddress
RemoteAddress *NetAddress
}
// MConnConfig is a MConnection configuration.
@ -140,9 +137,6 @@ func NewMConnectionWithConfig(conn net.Conn, chDescs []*ChannelDescriptor, onRec
onReceive: onReceive,
onError: onError,
config: config,
LocalAddress: NewNetAddress(conn.LocalAddr()),
RemoteAddress: NewNetAddress(conn.RemoteAddr()),
}
// Create channels


+ 113
- 0
p2p/key.go View File

@ -0,0 +1,113 @@
package p2p
import (
"bytes"
"encoding/hex"
"encoding/json"
"fmt"
"io/ioutil"
crypto "github.com/tendermint/go-crypto"
cmn "github.com/tendermint/tmlibs/common"
)
// ID is a hex-encoded crypto.Address
type ID string
// IDByteLength is the length of a crypto.Address. Currently only 20.
// TODO: support other length addresses ?
const IDByteLength = 20
//------------------------------------------------------------------------------
// Persistent peer ID
// TODO: encrypt on disk
// NodeKey is the persistent peer key.
// It contains the nodes private key for authentication.
type NodeKey struct {
PrivKey crypto.PrivKey `json:"priv_key"` // our priv key
}
// ID returns the peer's canonical ID - the hash of its public key.
func (nodeKey *NodeKey) ID() ID {
return PubKeyToID(nodeKey.PubKey())
}
// PubKey returns the peer's PubKey
func (nodeKey *NodeKey) PubKey() crypto.PubKey {
return nodeKey.PrivKey.PubKey()
}
// PubKeyToID returns the ID corresponding to the given PubKey.
// It's the hex-encoding of the pubKey.Address().
func PubKeyToID(pubKey crypto.PubKey) ID {
return ID(hex.EncodeToString(pubKey.Address()))
}
// LoadOrGenNodeKey attempts to load the NodeKey from the given filePath.
// If the file does not exist, it generates and saves a new NodeKey.
func LoadOrGenNodeKey(filePath string) (*NodeKey, error) {
if cmn.FileExists(filePath) {
nodeKey, err := loadNodeKey(filePath)
if err != nil {
return nil, err
}
return nodeKey, nil
} else {
return genNodeKey(filePath)
}
}
func loadNodeKey(filePath string) (*NodeKey, error) {
jsonBytes, err := ioutil.ReadFile(filePath)
if err != nil {
return nil, err
}
nodeKey := new(NodeKey)
err = json.Unmarshal(jsonBytes, nodeKey)
if err != nil {
return nil, fmt.Errorf("Error reading NodeKey from %v: %v\n", filePath, err)
}
return nodeKey, nil
}
func genNodeKey(filePath string) (*NodeKey, error) {
privKey := crypto.GenPrivKeyEd25519().Wrap()
nodeKey := &NodeKey{
PrivKey: privKey,
}
jsonBytes, err := json.Marshal(nodeKey)
if err != nil {
return nil, err
}
err = ioutil.WriteFile(filePath, jsonBytes, 0600)
if err != nil {
return nil, err
}
return nodeKey, nil
}
//------------------------------------------------------------------------------
// MakePoWTarget returns the big-endian encoding of 2^(targetBits - difficulty) - 1.
// It can be used as a Proof of Work target.
// NOTE: targetBits must be a multiple of 8 and difficulty must be less than targetBits.
func MakePoWTarget(difficulty, targetBits uint) []byte {
if targetBits%8 != 0 {
panic(fmt.Sprintf("targetBits (%d) not a multiple of 8", targetBits))
}
if difficulty >= targetBits {
panic(fmt.Sprintf("difficulty (%d) >= targetBits (%d)", difficulty, targetBits))
}
targetBytes := targetBits / 8
zeroPrefixLen := (int(difficulty) / 8)
prefix := bytes.Repeat([]byte{0}, zeroPrefixLen)
mod := (difficulty % 8)
if mod > 0 {
nonZeroPrefix := byte(1<<(8-mod) - 1)
prefix = append(prefix, nonZeroPrefix)
}
tailLen := int(targetBytes) - len(prefix)
return append(prefix, bytes.Repeat([]byte{0xFF}, tailLen)...)
}

+ 50
- 0
p2p/key_test.go View File

@ -0,0 +1,50 @@
package p2p
import (
"bytes"
"os"
"path/filepath"
"testing"
"github.com/stretchr/testify/assert"
cmn "github.com/tendermint/tmlibs/common"
)
func TestLoadOrGenNodeKey(t *testing.T) {
filePath := filepath.Join(os.TempDir(), cmn.RandStr(12)+"_peer_id.json")
nodeKey, err := LoadOrGenNodeKey(filePath)
assert.Nil(t, err)
nodeKey2, err := LoadOrGenNodeKey(filePath)
assert.Nil(t, err)
assert.Equal(t, nodeKey, nodeKey2)
}
//----------------------------------------------------------
func padBytes(bz []byte, targetBytes int) []byte {
return append(bz, bytes.Repeat([]byte{0xFF}, targetBytes-len(bz))...)
}
func TestPoWTarget(t *testing.T) {
targetBytes := 20
cases := []struct {
difficulty uint
target []byte
}{
{0, padBytes([]byte{}, targetBytes)},
{1, padBytes([]byte{127}, targetBytes)},
{8, padBytes([]byte{0}, targetBytes)},
{9, padBytes([]byte{0, 127}, targetBytes)},
{10, padBytes([]byte{0, 63}, targetBytes)},
{16, padBytes([]byte{0, 0}, targetBytes)},
{17, padBytes([]byte{0, 0, 127}, targetBytes)},
}
for _, c := range cases {
assert.Equal(t, MakePoWTarget(c.difficulty, 20*8), c.target)
}
}

+ 52
- 27
p2p/netaddress.go View File

@ -5,6 +5,7 @@
package p2p
import (
"encoding/hex"
"flag"
"fmt"
"net"
@ -12,41 +13,69 @@ import (
"strings"
"time"
"github.com/pkg/errors"
cmn "github.com/tendermint/tmlibs/common"
)
// NetAddress defines information about a peer on the network
// including its IP address, and port.
// including its ID, IP address, and port.
type NetAddress struct {
ID ID
IP net.IP
Port uint16
str string
}
// IDAddressString returns id@hostPort.
func IDAddressString(id ID, hostPort string) string {
return fmt.Sprintf("%s@%s", id, hostPort)
}
// NewNetAddress returns a new NetAddress using the provided TCP
// address. When testing, other net.Addr (except TCP) will result in
// using 0.0.0.0:0. When normal run, other net.Addr (except TCP) will
// panic.
// TODO: socks proxies?
func NewNetAddress(addr net.Addr) *NetAddress {
func NewNetAddress(id ID, addr net.Addr) *NetAddress {
tcpAddr, ok := addr.(*net.TCPAddr)
if !ok {
if flag.Lookup("test.v") == nil { // normal run
cmn.PanicSanity(cmn.Fmt("Only TCPAddrs are supported. Got: %v", addr))
} else { // in testing
return NewNetAddressIPPort(net.IP("0.0.0.0"), 0)
netAddr := NewNetAddressIPPort(net.IP("0.0.0.0"), 0)
netAddr.ID = id
return netAddr
}
}
ip := tcpAddr.IP
port := uint16(tcpAddr.Port)
return NewNetAddressIPPort(ip, port)
netAddr := NewNetAddressIPPort(ip, port)
netAddr.ID = id
return netAddr
}
// NewNetAddressString returns a new NetAddress using the provided
// address in the form of "IP:Port". Also resolves the host if host
// is not an IP.
// address in the form of "ID@IP:Port", where the ID is optional.
// Also resolves the host if host is not an IP.
func NewNetAddressString(addr string) (*NetAddress, error) {
host, portStr, err := net.SplitHostPort(removeProtocolIfDefined(addr))
addr = removeProtocolIfDefined(addr)
var id ID
spl := strings.Split(addr, "@")
if len(spl) == 2 {
idStr := spl[0]
idBytes, err := hex.DecodeString(idStr)
if err != nil {
return nil, errors.Wrap(err, fmt.Sprintf("Address (%s) contains invalid ID", addr))
}
if len(idBytes) != IDByteLength {
return nil, fmt.Errorf("Address (%s) contains ID of invalid length (%d). Should be %d hex-encoded bytes",
addr, len(idBytes), IDByteLength)
}
id, addr = ID(idStr), spl[1]
}
host, portStr, err := net.SplitHostPort(addr)
if err != nil {
return nil, err
}
@ -68,6 +97,7 @@ func NewNetAddressString(addr string) (*NetAddress, error) {
}
na := NewNetAddressIPPort(ip, uint16(port))
na.ID = id
return na, nil
}
@ -93,10 +123,6 @@ func NewNetAddressIPPort(ip net.IP, port uint16) *NetAddress {
na := &NetAddress{
IP: ip,
Port: port,
str: net.JoinHostPort(
ip.String(),
strconv.FormatUint(uint64(port), 10),
),
}
return na
}
@ -110,29 +136,28 @@ func (na *NetAddress) Equals(other interface{}) bool {
return false
}
func (na *NetAddress) Less(other interface{}) bool {
if o, ok := other.(*NetAddress); ok {
return na.String() < o.String()
}
cmn.PanicSanity("Cannot compare unequal types")
return false
}
// String representation.
// String representation: <ID>@<IP>:<PORT>
func (na *NetAddress) String() string {
if na.str == "" {
na.str = net.JoinHostPort(
na.IP.String(),
strconv.FormatUint(uint64(na.Port), 10),
)
addrStr := na.DialString()
if na.ID != "" {
addrStr = IDAddressString(na.ID, addrStr)
}
na.str = addrStr
}
return na.str
}
func (na *NetAddress) DialString() string {
return net.JoinHostPort(
na.IP.String(),
strconv.FormatUint(uint64(na.Port), 10),
)
}
// Dial calls net.Dial on the address.
func (na *NetAddress) Dial() (net.Conn, error) {
conn, err := net.Dial("tcp", na.String())
conn, err := net.Dial("tcp", na.DialString())
if err != nil {
return nil, err
}
@ -141,7 +166,7 @@ func (na *NetAddress) Dial() (net.Conn, error) {
// DialTimeout calls net.DialTimeout on the address.
func (na *NetAddress) DialTimeout(timeout time.Duration) (net.Conn, error) {
conn, err := net.DialTimeout("tcp", na.String(), timeout)
conn, err := net.DialTimeout("tcp", na.DialString(), timeout)
if err != nil {
return nil, err
}


+ 19
- 2
p2p/netaddress_test.go View File

@ -13,12 +13,12 @@ func TestNewNetAddress(t *testing.T) {
tcpAddr, err := net.ResolveTCPAddr("tcp", "127.0.0.1:8080")
require.Nil(err)
addr := NewNetAddress(tcpAddr)
addr := NewNetAddress("", tcpAddr)
assert.Equal("127.0.0.1:8080", addr.String())
assert.NotPanics(func() {
NewNetAddress(&net.UDPAddr{IP: net.ParseIP("127.0.0.1"), Port: 8000})
NewNetAddress("", &net.UDPAddr{IP: net.ParseIP("127.0.0.1"), Port: 8000})
}, "Calling NewNetAddress with UDPAddr should not panic in testing")
}
@ -38,6 +38,23 @@ func TestNewNetAddressString(t *testing.T) {
{"notahost:8080", "", false},
{"8082", "", false},
{"127.0.0:8080000", "", false},
{"deadbeef@127.0.0.1:8080", "", false},
{"this-isnot-hex@127.0.0.1:8080", "", false},
{"xxxxbeefdeadbeefdeadbeefdeadbeefdeadbeef@127.0.0.1:8080", "", false},
{"deadbeefdeadbeefdeadbeefdeadbeefdeadbeef@127.0.0.1:8080", "deadbeefdeadbeefdeadbeefdeadbeefdeadbeef@127.0.0.1:8080", true},
{"tcp://deadbeef@127.0.0.1:8080", "", false},
{"tcp://this-isnot-hex@127.0.0.1:8080", "", false},
{"tcp://xxxxbeefdeadbeefdeadbeefdeadbeefdeadbeef@127.0.0.1:8080", "", false},
{"tcp://deadbeefdeadbeefdeadbeefdeadbeefdeadbeef@127.0.0.1:8080", "deadbeefdeadbeefdeadbeefdeadbeefdeadbeef@127.0.0.1:8080", true},
{"tcp://@127.0.0.1:8080", "", false},
{"tcp://@", "", false},
{"", "", false},
{"@", "", false},
{" @", "", false},
{" @ ", "", false},
}
for _, tc := range testCases {


+ 17
- 18
p2p/peer.go View File

@ -1,6 +1,7 @@
package p2p
import (
"encoding/hex"
"fmt"
"net"
"time"
@ -17,7 +18,7 @@ import (
type Peer interface {
cmn.Service
Key() string
ID() ID
IsOutbound() bool
IsPersistent() bool
NodeInfo() *NodeInfo
@ -77,7 +78,7 @@ func DefaultPeerConfig() *PeerConfig {
}
func newOutboundPeer(addr *NetAddress, reactorsByCh map[byte]Reactor, chDescs []*ChannelDescriptor,
onPeerError func(Peer, interface{}), ourNodePrivKey crypto.PrivKeyEd25519, config *PeerConfig) (*peer, error) {
onPeerError func(Peer, interface{}), ourNodePrivKey crypto.PrivKey, config *PeerConfig) (*peer, error) {
conn, err := dial(addr, config)
if err != nil {
@ -91,17 +92,18 @@ func newOutboundPeer(addr *NetAddress, reactorsByCh map[byte]Reactor, chDescs []
}
return nil, err
}
return peer, nil
}
func newInboundPeer(conn net.Conn, reactorsByCh map[byte]Reactor, chDescs []*ChannelDescriptor,
onPeerError func(Peer, interface{}), ourNodePrivKey crypto.PrivKeyEd25519, config *PeerConfig) (*peer, error) {
onPeerError func(Peer, interface{}), ourNodePrivKey crypto.PrivKey, config *PeerConfig) (*peer, error) {
return newPeerFromConnAndConfig(conn, false, reactorsByCh, chDescs, onPeerError, ourNodePrivKey, config)
}
func newPeerFromConnAndConfig(rawConn net.Conn, outbound bool, reactorsByCh map[byte]Reactor, chDescs []*ChannelDescriptor,
onPeerError func(Peer, interface{}), ourNodePrivKey crypto.PrivKeyEd25519, config *PeerConfig) (*peer, error) {
onPeerError func(Peer, interface{}), ourNodePrivKey crypto.PrivKey, config *PeerConfig) (*peer, error) {
conn := rawConn
@ -204,8 +206,6 @@ func (p *peer) HandshakeTimeout(ourNodeInfo *NodeInfo, timeout time.Duration) er
return errors.Wrap(err, "Error removing deadline")
}
peerNodeInfo.RemoteAddr = p.Addr().String()
p.nodeInfo = peerNodeInfo
return nil
}
@ -216,14 +216,13 @@ func (p *peer) Addr() net.Addr {
}
// PubKey returns peer's public key.
func (p *peer) PubKey() crypto.PubKeyEd25519 {
if p.config.AuthEnc {
func (p *peer) PubKey() crypto.PubKey {
if p.NodeInfo() != nil {
return p.nodeInfo.PubKey
} else if p.config.AuthEnc {
return p.conn.(*SecretConnection).RemotePubKey()
}
if p.NodeInfo() == nil {
panic("Attempt to get peer's PubKey before calling Handshake")
}
return p.PubKey()
panic("Attempt to get peer's PubKey before calling Handshake")
}
// OnStart implements BaseService.
@ -282,15 +281,15 @@ func (p *peer) CanSend(chID byte) bool {
// String representation.
func (p *peer) String() string {
if p.outbound {
return fmt.Sprintf("Peer{%v %v out}", p.mconn, p.Key())
return fmt.Sprintf("Peer{%v %v out}", p.mconn, p.ID())
}
return fmt.Sprintf("Peer{%v %v in}", p.mconn, p.Key())
return fmt.Sprintf("Peer{%v %v in}", p.mconn, p.ID())
}
// Equals reports whenever 2 peers are actually represent the same node.
func (p *peer) Equals(other Peer) bool {
return p.Key() == other.Key()
return p.ID() == other.ID()
}
// Get the data for a given key.
@ -303,9 +302,9 @@ func (p *peer) Set(key string, data interface{}) {
p.Data.Set(key, data)
}
// Key returns the peer's id key.
func (p *peer) Key() string {
return p.nodeInfo.ListenAddr // XXX: should probably be PubKey.KeyString()
// ID returns the peer's ID - the hex encoded hash of its pubkey.
func (p *peer) ID() ID {
return ID(hex.EncodeToString(p.PubKey().Address()))
}
// NodeInfo returns a copy of the peer's NodeInfo.


+ 12
- 12
p2p/peer_set.go View File

@ -6,8 +6,8 @@ import (
// IPeerSet has a (immutable) subset of the methods of PeerSet.
type IPeerSet interface {
Has(key string) bool
Get(key string) Peer
Has(key ID) bool
Get(key ID) Peer
List() []Peer
Size() int
}
@ -18,7 +18,7 @@ type IPeerSet interface {
// Iteration over the peers is super fast and thread-safe.
type PeerSet struct {
mtx sync.Mutex
lookup map[string]*peerSetItem
lookup map[ID]*peerSetItem
list []Peer
}
@ -30,7 +30,7 @@ type peerSetItem struct {
// NewPeerSet creates a new peerSet with a list of initial capacity of 256 items.
func NewPeerSet() *PeerSet {
return &PeerSet{
lookup: make(map[string]*peerSetItem),
lookup: make(map[ID]*peerSetItem),
list: make([]Peer, 0, 256),
}
}
@ -40,7 +40,7 @@ func NewPeerSet() *PeerSet {
func (ps *PeerSet) Add(peer Peer) error {
ps.mtx.Lock()
defer ps.mtx.Unlock()
if ps.lookup[peer.Key()] != nil {
if ps.lookup[peer.ID()] != nil {
return ErrSwitchDuplicatePeer
}
@ -48,13 +48,13 @@ func (ps *PeerSet) Add(peer Peer) error {
// Appending is safe even with other goroutines
// iterating over the ps.list slice.
ps.list = append(ps.list, peer)
ps.lookup[peer.Key()] = &peerSetItem{peer, index}
ps.lookup[peer.ID()] = &peerSetItem{peer, index}
return nil
}
// Has returns true iff the PeerSet contains
// the peer referred to by this peerKey.
func (ps *PeerSet) Has(peerKey string) bool {
func (ps *PeerSet) Has(peerKey ID) bool {
ps.mtx.Lock()
_, ok := ps.lookup[peerKey]
ps.mtx.Unlock()
@ -62,7 +62,7 @@ func (ps *PeerSet) Has(peerKey string) bool {
}
// Get looks up a peer by the provided peerKey.
func (ps *PeerSet) Get(peerKey string) Peer {
func (ps *PeerSet) Get(peerKey ID) Peer {
ps.mtx.Lock()
defer ps.mtx.Unlock()
item, ok := ps.lookup[peerKey]
@ -77,7 +77,7 @@ func (ps *PeerSet) Get(peerKey string) Peer {
func (ps *PeerSet) Remove(peer Peer) {
ps.mtx.Lock()
defer ps.mtx.Unlock()
item := ps.lookup[peer.Key()]
item := ps.lookup[peer.ID()]
if item == nil {
return
}
@ -90,18 +90,18 @@ func (ps *PeerSet) Remove(peer Peer) {
// If it's the last peer, that's an easy special case.
if index == len(ps.list)-1 {
ps.list = newList
delete(ps.lookup, peer.Key())
delete(ps.lookup, peer.ID())
return
}
// Replace the popped item with the last item in the old list.
lastPeer := ps.list[len(ps.list)-1]
lastPeerKey := lastPeer.Key()
lastPeerKey := lastPeer.ID()
lastPeerItem := ps.lookup[lastPeerKey]
newList[index] = lastPeer
lastPeerItem.index = index
ps.list = newList
delete(ps.lookup, peer.Key())
delete(ps.lookup, peer.ID())
}
// Size returns the number of unique items in the peerSet.


+ 7
- 6
p2p/peer_set_test.go View File

@ -7,6 +7,7 @@ import (
"github.com/stretchr/testify/assert"
crypto "github.com/tendermint/go-crypto"
cmn "github.com/tendermint/tmlibs/common"
)
@ -14,8 +15,8 @@ import (
func randPeer() *peer {
return &peer{
nodeInfo: &NodeInfo{
RemoteAddr: cmn.Fmt("%v.%v.%v.%v:46656", rand.Int()%256, rand.Int()%256, rand.Int()%256, rand.Int()%256),
ListenAddr: cmn.Fmt("%v.%v.%v.%v:46656", rand.Int()%256, rand.Int()%256, rand.Int()%256, rand.Int()%256),
PubKey: crypto.GenPrivKeyEd25519().Wrap().PubKey(),
},
}
}
@ -39,7 +40,7 @@ func TestPeerSetAddRemoveOne(t *testing.T) {
peerSet.Remove(peerAtFront)
wantSize := n - i - 1
for j := 0; j < 2; j++ {
assert.Equal(t, false, peerSet.Has(peerAtFront.Key()), "#%d Run #%d: failed to remove peer", i, j)
assert.Equal(t, false, peerSet.Has(peerAtFront.ID()), "#%d Run #%d: failed to remove peer", i, j)
assert.Equal(t, wantSize, peerSet.Size(), "#%d Run #%d: failed to remove peer and decrement size", i, j)
// Test the route of removing the now non-existent element
peerSet.Remove(peerAtFront)
@ -58,7 +59,7 @@ func TestPeerSetAddRemoveOne(t *testing.T) {
for i := n - 1; i >= 0; i-- {
peerAtEnd := peerList[i]
peerSet.Remove(peerAtEnd)
assert.Equal(t, false, peerSet.Has(peerAtEnd.Key()), "#%d: failed to remove item at end", i)
assert.Equal(t, false, peerSet.Has(peerAtEnd.ID()), "#%d: failed to remove item at end", i)
assert.Equal(t, i, peerSet.Size(), "#%d: differing sizes after peerSet.Remove(atEndPeer)", i)
}
}
@ -82,7 +83,7 @@ func TestPeerSetAddRemoveMany(t *testing.T) {
for i, peer := range peers {
peerSet.Remove(peer)
if peerSet.Has(peer.Key()) {
if peerSet.Has(peer.ID()) {
t.Errorf("Failed to remove peer")
}
if peerSet.Size() != len(peers)-i-1 {
@ -129,7 +130,7 @@ func TestPeerSetGet(t *testing.T) {
t.Parallel()
peerSet := NewPeerSet()
peer := randPeer()
assert.Nil(t, peerSet.Get(peer.Key()), "expecting a nil lookup, before .Add")
assert.Nil(t, peerSet.Get(peer.ID()), "expecting a nil lookup, before .Add")
if err := peerSet.Add(peer); err != nil {
t.Fatalf("Failed to add new peer: %v", err)
@ -142,7 +143,7 @@ func TestPeerSetGet(t *testing.T) {
wg.Add(1)
go func(i int) {
defer wg.Done()
got, want := peerSet.Get(peer.Key()), peer
got, want := peerSet.Get(peer.ID()), peer
assert.Equal(t, got, want, "#%d: got=%v want=%v", i, got, want)
}(i)
}


+ 14
- 13
p2p/peer_test.go View File

@ -16,7 +16,7 @@ func TestPeerBasic(t *testing.T) {
assert, require := assert.New(t), require.New(t)
// simulate remote peer
rp := &remotePeer{PrivKey: crypto.GenPrivKeyEd25519(), Config: DefaultPeerConfig()}
rp := &remotePeer{PrivKey: crypto.GenPrivKeyEd25519().Wrap(), Config: DefaultPeerConfig()}
rp.Start()
defer rp.Stop()
@ -43,7 +43,7 @@ func TestPeerWithoutAuthEnc(t *testing.T) {
config.AuthEnc = false
// simulate remote peer
rp := &remotePeer{PrivKey: crypto.GenPrivKeyEd25519(), Config: config}
rp := &remotePeer{PrivKey: crypto.GenPrivKeyEd25519().Wrap(), Config: config}
rp.Start()
defer rp.Stop()
@ -64,7 +64,7 @@ func TestPeerSend(t *testing.T) {
config.AuthEnc = false
// simulate remote peer
rp := &remotePeer{PrivKey: crypto.GenPrivKeyEd25519(), Config: config}
rp := &remotePeer{PrivKey: crypto.GenPrivKeyEd25519().Wrap(), Config: config}
rp.Start()
defer rp.Stop()
@ -85,13 +85,13 @@ func createOutboundPeerAndPerformHandshake(addr *NetAddress, config *PeerConfig)
{ID: 0x01, Priority: 1},
}
reactorsByCh := map[byte]Reactor{0x01: NewTestReactor(chDescs, true)}
pk := crypto.GenPrivKeyEd25519()
pk := crypto.GenPrivKeyEd25519().Wrap()
p, err := newOutboundPeer(addr, reactorsByCh, chDescs, func(p Peer, r interface{}) {}, pk, config)
if err != nil {
return nil, err
}
err = p.HandshakeTimeout(&NodeInfo{
PubKey: pk.PubKey().Unwrap().(crypto.PubKeyEd25519),
PubKey: pk.PubKey(),
Moniker: "host_peer",
Network: "testing",
Version: "123.123.123",
@ -103,7 +103,7 @@ func createOutboundPeerAndPerformHandshake(addr *NetAddress, config *PeerConfig)
}
type remotePeer struct {
PrivKey crypto.PrivKeyEd25519
PrivKey crypto.PrivKey
Config *PeerConfig
addr *NetAddress
quit chan struct{}
@ -113,8 +113,8 @@ func (p *remotePeer) Addr() *NetAddress {
return p.addr
}
func (p *remotePeer) PubKey() crypto.PubKeyEd25519 {
return p.PrivKey.PubKey().Unwrap().(crypto.PubKeyEd25519)
func (p *remotePeer) PubKey() crypto.PubKey {
return p.PrivKey.PubKey()
}
func (p *remotePeer) Start() {
@ -122,7 +122,7 @@ func (p *remotePeer) Start() {
if e != nil {
golog.Fatalf("net.Listen tcp :0: %+v", e)
}
p.addr = NewNetAddress(l.Addr())
p.addr = NewNetAddress("", l.Addr())
p.quit = make(chan struct{})
go p.accept(l)
}
@ -142,10 +142,11 @@ func (p *remotePeer) accept(l net.Listener) {
golog.Fatalf("Failed to create a peer: %+v", err)
}
err = peer.HandshakeTimeout(&NodeInfo{
PubKey: p.PrivKey.PubKey().Unwrap().(crypto.PubKeyEd25519),
Moniker: "remote_peer",
Network: "testing",
Version: "123.123.123",
PubKey: p.PrivKey.PubKey(),
Moniker: "remote_peer",
Network: "testing",
Version: "123.123.123",
ListenAddr: l.Addr().String(),
}, 1*time.Second)
if err != nil {
golog.Fatalf("Failed to perform handshake: %+v", err)


+ 23
- 34
p2p/pex_reactor.go View File

@ -109,18 +109,14 @@ func (r *PEXReactor) GetChannels() []*ChannelDescriptor {
func (r *PEXReactor) AddPeer(p Peer) {
if p.IsOutbound() {
// For outbound peers, the address is already in the books.
// Either it was added in DialPersistentPeers or when we
// Either it was added in DialPeersAsync or when we
// received the peer's address in r.Receive
if r.book.NeedMoreAddrs() {
r.RequestPEX(p)
}
} else { // For inbound connections, the peer is its own source
addr, err := NewNetAddressString(p.NodeInfo().ListenAddr)
if err != nil {
// peer gave us a bad ListenAddr. TODO: punish
r.Logger.Error("Error in AddPeer: invalid peer address", "addr", p.NodeInfo().ListenAddr, "err", err)
return
}
} else {
// For inbound connections, the peer is its own source
addr := p.NodeInfo().NetAddress()
r.book.AddAddress(addr, addr)
}
}
@ -133,17 +129,11 @@ func (r *PEXReactor) RemovePeer(p Peer, reason interface{}) {
// Receive implements Reactor by handling incoming PEX messages.
func (r *PEXReactor) Receive(chID byte, src Peer, msgBytes []byte) {
srcAddrStr := src.NodeInfo().RemoteAddr
srcAddr, err := NewNetAddressString(srcAddrStr)
if err != nil {
// this should never happen. TODO: cancel conn
r.Logger.Error("Error in Receive: invalid peer address", "addr", srcAddrStr, "err", err)
return
}
srcAddr := src.NodeInfo().NetAddress()
r.IncrementMsgCountForPeer(srcAddrStr)
if r.ReachedMaxMsgCountForPeer(srcAddrStr) {
r.Logger.Error("Maximum number of messages reached for peer", "peer", srcAddrStr)
r.IncrementMsgCountForPeer(srcAddr.ID)
if r.ReachedMaxMsgCountForPeer(srcAddr.ID) {
r.Logger.Error("Maximum number of messages reached for peer", "peer", srcAddr)
// TODO remove src from peers?
return
}
@ -163,9 +153,9 @@ func (r *PEXReactor) Receive(chID byte, src Peer, msgBytes []byte) {
case *pexAddrsMessage:
// We received some peer addresses from src.
// TODO: (We don't want to get spammed with bad peers)
for _, addr := range msg.Addrs {
if addr != nil {
r.book.AddAddress(addr, srcAddr)
for _, netAddr := range msg.Addrs {
if netAddr != nil {
r.book.AddAddress(netAddr, srcAddr)
}
}
default:
@ -179,8 +169,8 @@ func (r *PEXReactor) RequestPEX(p Peer) {
}
// SendAddrs sends addrs to the peer.
func (r *PEXReactor) SendAddrs(p Peer, addrs []*NetAddress) {
p.Send(PexChannel, struct{ PexMessage }{&pexAddrsMessage{Addrs: addrs}})
func (r *PEXReactor) SendAddrs(p Peer, netAddrs []*NetAddress) {
p.Send(PexChannel, struct{ PexMessage }{&pexAddrsMessage{Addrs: netAddrs}})
}
// SetEnsurePeersPeriod sets period to ensure peers connected.
@ -196,19 +186,19 @@ func (r *PEXReactor) SetMaxMsgCountByPeer(v uint16) {
// ReachedMaxMsgCountForPeer returns true if we received too many
// messages from peer with address `addr`.
// NOTE: assumes the value in the CMap is non-nil
func (r *PEXReactor) ReachedMaxMsgCountForPeer(addr string) bool {
return r.msgCountByPeer.Get(addr).(uint16) >= r.maxMsgCountByPeer
func (r *PEXReactor) ReachedMaxMsgCountForPeer(peerID ID) bool {
return r.msgCountByPeer.Get(string(peerID)).(uint16) >= r.maxMsgCountByPeer
}
// Increment or initialize the msg count for the peer in the CMap
func (r *PEXReactor) IncrementMsgCountForPeer(addr string) {
func (r *PEXReactor) IncrementMsgCountForPeer(peerID ID) {
var count uint16
countI := r.msgCountByPeer.Get(addr)
countI := r.msgCountByPeer.Get(string(peerID))
if countI != nil {
count = countI.(uint16)
}
count++
r.msgCountByPeer.Set(addr, count)
r.msgCountByPeer.Set(string(peerID), count)
}
// Ensures that sufficient peers are connected. (continuous)
@ -259,7 +249,7 @@ func (r *PEXReactor) ensurePeers() {
// NOTE: range here is [10, 90]. Too high ?
newBias := cmn.MinInt(numOutPeers, 8)*10 + 10
toDial := make(map[string]*NetAddress)
toDial := make(map[ID]*NetAddress)
// Try maxAttempts times to pick numToDial addresses to dial
maxAttempts := numToDial * 3
for i := 0; i < maxAttempts && len(toDial) < numToDial; i++ {
@ -267,18 +257,17 @@ func (r *PEXReactor) ensurePeers() {
if try == nil {
continue
}
if _, selected := toDial[try.IP.String()]; selected {
if _, selected := toDial[try.ID]; selected {
continue
}
if dialling := r.Switch.IsDialing(try); dialling {
if dialling := r.Switch.IsDialing(try.ID); dialling {
continue
}
// XXX: Should probably use pubkey as peer key ...
if connected := r.Switch.Peers().Has(try.String()); connected {
if connected := r.Switch.Peers().Has(try.ID); connected {
continue
}
r.Logger.Info("Will dial address", "addr", try)
toDial[try.IP.String()] = try
toDial[try.ID] = try
}
// Dial picked addresses


+ 5
- 4
p2p/pex_reactor_test.go View File

@ -10,6 +10,7 @@ import (
"github.com/stretchr/testify/assert"
"github.com/stretchr/testify/require"
crypto "github.com/tendermint/go-crypto"
wire "github.com/tendermint/go-wire"
cmn "github.com/tendermint/tmlibs/common"
"github.com/tendermint/tmlibs/log"
@ -183,7 +184,7 @@ func TestPEXReactorAbuseFromPeer(t *testing.T) {
r.Receive(PexChannel, peer, msg)
}
assert.True(r.ReachedMaxMsgCountForPeer(peer.NodeInfo().ListenAddr))
assert.True(r.ReachedMaxMsgCountForPeer(peer.NodeInfo().ID()))
}
func TestPEXReactorUsesSeedsIfNeeded(t *testing.T) {
@ -242,11 +243,11 @@ func createRandomPeer(outbound bool) *peer {
addr, netAddr := createRoutableAddr()
p := &peer{
nodeInfo: &NodeInfo{
ListenAddr: addr,
RemoteAddr: netAddr.String(),
ListenAddr: netAddr.String(),
PubKey: crypto.GenPrivKeyEd25519().Wrap().PubKey(),
},
outbound: outbound,
mconn: &MConnection{RemoteAddress: netAddr},
mconn: &MConnection{},
}
p.SetLogger(log.TestingLogger().With("peer", addr))
return p


+ 8
- 8
p2p/secret_connection.go View File

@ -38,7 +38,7 @@ type SecretConnection struct {
recvBuffer []byte
recvNonce *[24]byte
sendNonce *[24]byte
remPubKey crypto.PubKeyEd25519
remPubKey crypto.PubKey
shrSecret *[32]byte // shared secret
}
@ -46,9 +46,9 @@ type SecretConnection struct {
// Returns nil if error in handshake.
// Caller should call conn.Close()
// See docs/sts-final.pdf for more information.
func MakeSecretConnection(conn io.ReadWriteCloser, locPrivKey crypto.PrivKeyEd25519) (*SecretConnection, error) {
func MakeSecretConnection(conn io.ReadWriteCloser, locPrivKey crypto.PrivKey) (*SecretConnection, error) {
locPubKey := locPrivKey.PubKey().Unwrap().(crypto.PubKeyEd25519)
locPubKey := locPrivKey.PubKey()
// Generate ephemeral keys for perfect forward secrecy.
locEphPub, locEphPriv := genEphKeys()
@ -100,12 +100,12 @@ func MakeSecretConnection(conn io.ReadWriteCloser, locPrivKey crypto.PrivKeyEd25
}
// We've authorized.
sc.remPubKey = remPubKey.Unwrap().(crypto.PubKeyEd25519)
sc.remPubKey = remPubKey
return sc, nil
}
// Returns authenticated remote pubkey
func (sc *SecretConnection) RemotePubKey() crypto.PubKeyEd25519 {
func (sc *SecretConnection) RemotePubKey() crypto.PubKey {
return sc.remPubKey
}
@ -258,8 +258,8 @@ func genChallenge(loPubKey, hiPubKey *[32]byte) (challenge *[32]byte) {
return hash32(append(loPubKey[:], hiPubKey[:]...))
}
func signChallenge(challenge *[32]byte, locPrivKey crypto.PrivKeyEd25519) (signature crypto.SignatureEd25519) {
signature = locPrivKey.Sign(challenge[:]).Unwrap().(crypto.SignatureEd25519)
func signChallenge(challenge *[32]byte, locPrivKey crypto.PrivKey) (signature crypto.Signature) {
signature = locPrivKey.Sign(challenge[:])
return
}
@ -268,7 +268,7 @@ type authSigMessage struct {
Sig crypto.Signature
}
func shareAuthSignature(sc *SecretConnection, pubKey crypto.PubKeyEd25519, signature crypto.SignatureEd25519) (*authSigMessage, error) {
func shareAuthSignature(sc *SecretConnection, pubKey crypto.PubKey, signature crypto.Signature) (*authSigMessage, error) {
var recvMsg authSigMessage
var err1, err2 error


+ 7
- 8
p2p/secret_connection_test.go View File

@ -1,7 +1,6 @@
package p2p
import (
"bytes"
"io"
"testing"
@ -32,10 +31,10 @@ func makeDummyConnPair() (fooConn, barConn dummyConn) {
func makeSecretConnPair(tb testing.TB) (fooSecConn, barSecConn *SecretConnection) {
fooConn, barConn := makeDummyConnPair()
fooPrvKey := crypto.GenPrivKeyEd25519()
fooPubKey := fooPrvKey.PubKey().Unwrap().(crypto.PubKeyEd25519)
barPrvKey := crypto.GenPrivKeyEd25519()
barPubKey := barPrvKey.PubKey().Unwrap().(crypto.PubKeyEd25519)
fooPrvKey := crypto.GenPrivKeyEd25519().Wrap()
fooPubKey := fooPrvKey.PubKey()
barPrvKey := crypto.GenPrivKeyEd25519().Wrap()
barPubKey := barPrvKey.PubKey()
cmn.Parallel(
func() {
@ -46,7 +45,7 @@ func makeSecretConnPair(tb testing.TB) (fooSecConn, barSecConn *SecretConnection
return
}
remotePubBytes := fooSecConn.RemotePubKey()
if !bytes.Equal(remotePubBytes[:], barPubKey[:]) {
if !remotePubBytes.Equals(barPubKey) {
tb.Errorf("Unexpected fooSecConn.RemotePubKey. Expected %v, got %v",
barPubKey, fooSecConn.RemotePubKey())
}
@ -59,7 +58,7 @@ func makeSecretConnPair(tb testing.TB) (fooSecConn, barSecConn *SecretConnection
return
}
remotePubBytes := barSecConn.RemotePubKey()
if !bytes.Equal(remotePubBytes[:], fooPubKey[:]) {
if !remotePubBytes.Equals(fooPubKey) {
tb.Errorf("Unexpected barSecConn.RemotePubKey. Expected %v, got %v",
fooPubKey, barSecConn.RemotePubKey())
}
@ -93,7 +92,7 @@ func TestSecretConnectionReadWrite(t *testing.T) {
genNodeRunner := func(nodeConn dummyConn, nodeWrites []string, nodeReads *[]string) func() {
return func() {
// Node handskae
nodePrvKey := crypto.GenPrivKeyEd25519()
nodePrvKey := crypto.GenPrivKeyEd25519().Wrap()
nodeSecretConn, err := MakeSecretConnection(nodeConn, nodePrvKey)
if err != nil {
t.Errorf("Failed to establish SecretConnection for node: %v", err)


+ 46
- 34
p2p/switch.go View File

@ -12,6 +12,7 @@ import (
crypto "github.com/tendermint/go-crypto"
cfg "github.com/tendermint/tendermint/config"
cmn "github.com/tendermint/tmlibs/common"
"github.com/tendermint/tmlibs/log"
)
const (
@ -81,17 +82,18 @@ type Switch struct {
reactorsByCh map[byte]Reactor
peers *PeerSet
dialing *cmn.CMap
nodeInfo *NodeInfo // our node info
nodePrivKey crypto.PrivKeyEd25519 // our node privkey
nodeInfo *NodeInfo // our node info
nodeKey *NodeKey // our node privkey
filterConnByAddr func(net.Addr) error
filterConnByPubKey func(crypto.PubKeyEd25519) error
filterConnByPubKey func(crypto.PubKey) error
rng *rand.Rand // seed for randomizing dial times and orders
}
var (
ErrSwitchDuplicatePeer = errors.New("Duplicate peer")
ErrSwitchConnectToSelf = errors.New("Connect to self")
)
func NewSwitch(config *cfg.P2PConfig) *Switch {
@ -181,13 +183,13 @@ func (sw *Switch) NodeInfo() *NodeInfo {
return sw.nodeInfo
}
// SetNodePrivKey sets the switch's private key for authenticated encryption.
// SetNodeKey sets the switch's private key for authenticated encryption.
// NOTE: Overwrites sw.nodeInfo.PubKey.
// NOTE: Not goroutine safe.
func (sw *Switch) SetNodePrivKey(nodePrivKey crypto.PrivKeyEd25519) {
sw.nodePrivKey = nodePrivKey
func (sw *Switch) SetNodeKey(nodeKey *NodeKey) {
sw.nodeKey = nodeKey
if sw.nodeInfo != nil {
sw.nodeInfo.PubKey = nodePrivKey.PubKey().Unwrap().(crypto.PubKeyEd25519)
sw.nodeInfo.PubKey = nodeKey.PubKey()
}
}
@ -231,11 +233,15 @@ func (sw *Switch) OnStop() {
// NOTE: This performs a blocking handshake before the peer is added.
// NOTE: If error is returned, caller is responsible for calling peer.CloseConn()
func (sw *Switch) addPeer(peer *peer) error {
// Avoid self
if sw.nodeKey.ID() == peer.ID() {
return ErrSwitchConnectToSelf
}
// Filter peer against white list
if err := sw.FilterConnByAddr(peer.Addr()); err != nil {
return err
}
if err := sw.FilterConnByPubKey(peer.PubKey()); err != nil {
return err
}
@ -244,9 +250,10 @@ func (sw *Switch) addPeer(peer *peer) error {
return err
}
// Avoid self
if sw.nodeInfo.PubKey.Equals(peer.PubKey().Wrap()) {
return errors.New("Ignoring connection from self")
// Avoid duplicate
if sw.peers.Has(peer.ID()) {
return ErrSwitchDuplicatePeer
}
// Check version, chain id
@ -254,12 +261,6 @@ func (sw *Switch) addPeer(peer *peer) error {
return err
}
// Check for duplicate peer
if sw.peers.Has(peer.Key()) {
return ErrSwitchDuplicatePeer
}
// Start peer
if sw.IsRunning() {
sw.startInitPeer(peer)
@ -285,7 +286,7 @@ func (sw *Switch) FilterConnByAddr(addr net.Addr) error {
}
// FilterConnByPubKey returns an error if connecting to the given public key is forbidden.
func (sw *Switch) FilterConnByPubKey(pubkey crypto.PubKeyEd25519) error {
func (sw *Switch) FilterConnByPubKey(pubkey crypto.PubKey) error {
if sw.filterConnByPubKey != nil {
return sw.filterConnByPubKey(pubkey)
}
@ -299,7 +300,7 @@ func (sw *Switch) SetAddrFilter(f func(net.Addr) error) {
}
// SetPubKeyFilter sets the function for filtering connections by public key.
func (sw *Switch) SetPubKeyFilter(f func(crypto.PubKeyEd25519) error) {
func (sw *Switch) SetPubKeyFilter(f func(crypto.PubKey) error) {
sw.filterConnByPubKey = f
}
@ -318,6 +319,7 @@ func (sw *Switch) startInitPeer(peer *peer) {
// DialPeersAsync dials a list of peers asynchronously in random order (optionally, making them persistent).
func (sw *Switch) DialPeersAsync(addrBook *AddrBook, peers []string, persistent bool) error {
netAddrs, errs := NewNetAddressStrings(peers)
// TODO: IDs
for _, err := range errs {
sw.Logger.Error("Error in peer's address", "err", err)
}
@ -362,16 +364,24 @@ func (sw *Switch) randomSleep(interval time.Duration) {
// DialPeerWithAddress dials the given peer and runs sw.addPeer if it connects successfully.
// If `persistent == true`, the switch will always try to reconnect to this peer if the connection ever fails.
func (sw *Switch) DialPeerWithAddress(addr *NetAddress, persistent bool) (Peer, error) {
sw.dialing.Set(addr.IP.String(), addr)
defer sw.dialing.Delete(addr.IP.String())
sw.dialing.Set(string(addr.ID), addr)
defer sw.dialing.Delete(string(addr.ID))
sw.Logger.Info("Dialing peer", "address", addr)
peer, err := newOutboundPeer(addr, sw.reactorsByCh, sw.chDescs, sw.StopPeerForError, sw.nodePrivKey, sw.peerConfig)
peer, err := newOutboundPeer(addr, sw.reactorsByCh, sw.chDescs, sw.StopPeerForError, sw.nodeKey.PrivKey, sw.peerConfig)
if err != nil {
sw.Logger.Error("Failed to dial peer", "address", addr, "err", err)
return nil, err
}
peer.SetLogger(sw.Logger.With("peer", addr))
// authenticate peer
if addr.ID == "" {
peer.Logger.Info("Dialed peer with unknown ID - unable to authenticate", "addr", addr)
} else if addr.ID != peer.ID() {
return nil, fmt.Errorf("Failed to authenticate peer %v. Connected to peer with ID %s", addr, peer.ID())
}
if persistent {
peer.makePersistent()
}
@ -385,9 +395,9 @@ func (sw *Switch) DialPeerWithAddress(addr *NetAddress, persistent bool) (Peer,
return peer, nil
}
// IsDialing returns true if the switch is currently dialing the given address.
func (sw *Switch) IsDialing(addr *NetAddress) bool {
return sw.dialing.Has(addr.IP.String())
// IsDialing returns true if the switch is currently dialing the given ID.
func (sw *Switch) IsDialing(id ID) bool {
return sw.dialing.Has(string(id))
}
// Broadcast runs a go routine for each attempted send, which will block
@ -443,7 +453,7 @@ func (sw *Switch) StopPeerForError(peer Peer, reason interface{}) {
// If no success after all that, it stops trying, and leaves it
// to the PEX/Addrbook to find the peer again
func (sw *Switch) reconnectToPeer(peer Peer) {
addr, _ := NewNetAddressString(peer.NodeInfo().RemoteAddr)
netAddr := peer.NodeInfo().NetAddress()
start := time.Now()
sw.Logger.Info("Reconnecting to peer", "peer", peer)
for i := 0; i < reconnectAttempts; i++ {
@ -451,7 +461,7 @@ func (sw *Switch) reconnectToPeer(peer Peer) {
return
}
peer, err := sw.DialPeerWithAddress(addr, true)
peer, err := sw.DialPeerWithAddress(netAddr, true)
if err != nil {
sw.Logger.Info("Error reconnecting to peer. Trying again", "tries", i, "err", err, "peer", peer)
// sleep a set amount
@ -473,7 +483,7 @@ func (sw *Switch) reconnectToPeer(peer Peer) {
// sleep an exponentially increasing amount
sleepIntervalSeconds := math.Pow(reconnectBackOffBaseSeconds, float64(i))
sw.randomSleep(time.Duration(sleepIntervalSeconds) * time.Second)
peer, err := sw.DialPeerWithAddress(addr, true)
peer, err := sw.DialPeerWithAddress(netAddr, true)
if err != nil {
sw.Logger.Info("Error reconnecting to peer. Trying again", "tries", i, "err", err, "peer", peer)
continue
@ -594,24 +604,26 @@ func StartSwitches(switches []*Switch) error {
}
func makeSwitch(cfg *cfg.P2PConfig, i int, network, version string, initSwitch func(int, *Switch) *Switch) *Switch {
privKey := crypto.GenPrivKeyEd25519()
// new switch, add reactors
// TODO: let the config be passed in?
nodeKey := &NodeKey{
PrivKey: crypto.GenPrivKeyEd25519().Wrap(),
}
s := initSwitch(i, NewSwitch(cfg))
s.SetNodeInfo(&NodeInfo{
PubKey: privKey.PubKey().Unwrap().(crypto.PubKeyEd25519),
PubKey: nodeKey.PubKey(),
Moniker: cmn.Fmt("switch%d", i),
Network: network,
Version: version,
RemoteAddr: cmn.Fmt("%v:%v", network, rand.Intn(64512)+1023),
ListenAddr: cmn.Fmt("%v:%v", network, rand.Intn(64512)+1023),
})
s.SetNodePrivKey(privKey)
s.SetNodeKey(nodeKey)
s.SetLogger(log.TestingLogger())
return s
}
func (sw *Switch) addPeerWithConnection(conn net.Conn) error {
peer, err := newInboundPeer(conn, sw.reactorsByCh, sw.chDescs, sw.StopPeerForError, sw.nodePrivKey, sw.peerConfig)
peer, err := newInboundPeer(conn, sw.reactorsByCh, sw.chDescs, sw.StopPeerForError, sw.nodeKey.PrivKey, sw.peerConfig)
if err != nil {
if err := conn.Close(); err != nil {
sw.Logger.Error("Error closing connection", "err", err)
@ -628,7 +640,7 @@ func (sw *Switch) addPeerWithConnection(conn net.Conn) error {
}
func (sw *Switch) addPeerWithConnectionAndConfig(conn net.Conn, config *PeerConfig) error {
peer, err := newInboundPeer(conn, sw.reactorsByCh, sw.chDescs, sw.StopPeerForError, sw.nodePrivKey, config)
peer, err := newInboundPeer(conn, sw.reactorsByCh, sw.chDescs, sw.StopPeerForError, sw.nodeKey.PrivKey, config)
if err != nil {
if err := conn.Close(); err != nil {
sw.Logger.Error("Error closing connection", "err", err)


+ 7
- 7
p2p/switch_test.go View File

@ -28,7 +28,7 @@ func init() {
}
type PeerMessage struct {
PeerKey string
PeerID ID
Bytes []byte
Counter int
}
@ -77,7 +77,7 @@ func (tr *TestReactor) Receive(chID byte, peer Peer, msgBytes []byte) {
tr.mtx.Lock()
defer tr.mtx.Unlock()
//fmt.Printf("Received: %X, %X\n", chID, msgBytes)
tr.msgsReceived[chID] = append(tr.msgsReceived[chID], PeerMessage{peer.Key(), msgBytes, tr.msgsCounter})
tr.msgsReceived[chID] = append(tr.msgsReceived[chID], PeerMessage{peer.ID(), msgBytes, tr.msgsCounter})
tr.msgsCounter++
}
}
@ -200,7 +200,7 @@ func TestConnPubKeyFilter(t *testing.T) {
c1, c2 := netPipe()
// set pubkey filter
s1.SetPubKeyFilter(func(pubkey crypto.PubKeyEd25519) error {
s1.SetPubKeyFilter(func(pubkey crypto.PubKey) error {
if bytes.Equal(pubkey.Bytes(), s2.nodeInfo.PubKey.Bytes()) {
return fmt.Errorf("Error: pipe is blacklisted")
}
@ -232,11 +232,11 @@ func TestSwitchStopsNonPersistentPeerOnError(t *testing.T) {
defer sw.Stop()
// simulate remote peer
rp := &remotePeer{PrivKey: crypto.GenPrivKeyEd25519(), Config: DefaultPeerConfig()}
rp := &remotePeer{PrivKey: crypto.GenPrivKeyEd25519().Wrap(), Config: DefaultPeerConfig()}
rp.Start()
defer rp.Stop()
peer, err := newOutboundPeer(rp.Addr(), sw.reactorsByCh, sw.chDescs, sw.StopPeerForError, sw.nodePrivKey, DefaultPeerConfig())
peer, err := newOutboundPeer(rp.Addr(), sw.reactorsByCh, sw.chDescs, sw.StopPeerForError, sw.nodeKey.PrivKey, DefaultPeerConfig())
require.Nil(err)
err = sw.addPeer(peer)
require.Nil(err)
@ -259,11 +259,11 @@ func TestSwitchReconnectsToPersistentPeer(t *testing.T) {
defer sw.Stop()
// simulate remote peer
rp := &remotePeer{PrivKey: crypto.GenPrivKeyEd25519(), Config: DefaultPeerConfig()}
rp := &remotePeer{PrivKey: crypto.GenPrivKeyEd25519().Wrap(), Config: DefaultPeerConfig()}
rp.Start()
defer rp.Stop()
peer, err := newOutboundPeer(rp.Addr(), sw.reactorsByCh, sw.chDescs, sw.StopPeerForError, sw.nodePrivKey, DefaultPeerConfig())
peer, err := newOutboundPeer(rp.Addr(), sw.reactorsByCh, sw.chDescs, sw.StopPeerForError, sw.nodeKey.PrivKey, DefaultPeerConfig())
peer.makePersistent()
require.Nil(err)
err = sw.addPeer(peer)


+ 23
- 8
p2p/types.go View File

@ -11,14 +11,15 @@ import (
const maxNodeInfoSize = 10240 // 10Kb
// NodeInfo is the basic node information exchanged
// between two peers during the Tendermint P2P handshake
type NodeInfo struct {
PubKey crypto.PubKeyEd25519 `json:"pub_key"`
Moniker string `json:"moniker"`
Network string `json:"network"`
RemoteAddr string `json:"remote_addr"`
ListenAddr string `json:"listen_addr"`
Version string `json:"version"` // major.minor.revision
Other []string `json:"other"` // other application specific data
PubKey crypto.PubKey `json:"pub_key"` // authenticated pubkey
Moniker string `json:"moniker"` // arbitrary moniker
Network string `json:"network"` // network/chain ID
ListenAddr string `json:"listen_addr"` // accepting incoming
Version string `json:"version"` // major.minor.revision
Other []string `json:"other"` // other application specific data
}
// CONTRACT: two nodes are compatible if the major/minor versions match and network match
@ -54,6 +55,20 @@ func (info *NodeInfo) CompatibleWith(other *NodeInfo) error {
return nil
}
func (info *NodeInfo) ID() ID {
return PubKeyToID(info.PubKey)
}
func (info *NodeInfo) NetAddress() *NetAddress {
id := PubKeyToID(info.PubKey)
addr := info.ListenAddr
netAddr, err := NewNetAddressString(IDAddressString(id, addr))
if err != nil {
panic(err) // everything should be well formed by now
}
return netAddr
}
func (info *NodeInfo) ListenHost() string {
host, _, _ := net.SplitHostPort(info.ListenAddr) // nolint: errcheck, gas
return host
@ -69,7 +84,7 @@ func (info *NodeInfo) ListenPort() int {
}
func (info NodeInfo) String() string {
return fmt.Sprintf("NodeInfo{pk: %v, moniker: %v, network: %v [remote %v, listen %v], version: %v (%v)}", info.PubKey, info.Moniker, info.Network, info.RemoteAddr, info.ListenAddr, info.Version, info.Other)
return fmt.Sprintf("NodeInfo{pk: %v, moniker: %v, network: %v [listen %v], version: %v (%v)}", info.PubKey, info.Moniker, info.Network, info.ListenAddr, info.Version, info.Other)
}
func splitVersion(version string) (string, string, string, error) {


+ 3
- 2
rpc/core/consensus.go View File

@ -3,6 +3,7 @@ package core
import (
cm "github.com/tendermint/tendermint/consensus"
cstypes "github.com/tendermint/tendermint/consensus/types"
p2p "github.com/tendermint/tendermint/p2p"
ctypes "github.com/tendermint/tendermint/rpc/core/types"
sm "github.com/tendermint/tendermint/state"
"github.com/tendermint/tendermint/types"
@ -82,11 +83,11 @@ func Validators(heightPtr *int64) (*ctypes.ResultValidators, error) {
// }
// ```
func DumpConsensusState() (*ctypes.ResultDumpConsensusState, error) {
peerRoundStates := make(map[string]*cstypes.PeerRoundState)
peerRoundStates := make(map[p2p.ID]*cstypes.PeerRoundState)
for _, peer := range p2pSwitch.Peers().List() {
peerState := peer.Get(types.PeerStateKey).(*cm.PeerState)
peerRoundState := peerState.GetRoundState()
peerRoundStates[peer.Key()] = peerRoundState
peerRoundStates[peer.ID()] = peerRoundState
}
return &ctypes.ResultDumpConsensusState{consensusState.GetRoundState(), peerRoundStates}, nil
}

+ 1
- 1
rpc/core/types/responses.go View File

@ -103,7 +103,7 @@ type ResultValidators struct {
type ResultDumpConsensusState struct {
RoundState *cstypes.RoundState `json:"round_state"`
PeerRoundStates map[string]*cstypes.PeerRoundState `json:"peer_round_states"`
PeerRoundStates map[p2p.ID]*cstypes.PeerRoundState `json:"peer_round_states"`
}
type ResultBroadcastTx struct {


+ 4
- 3
types/vote_set.go View File

@ -8,6 +8,7 @@ import (
"github.com/pkg/errors"
"github.com/tendermint/tendermint/p2p"
cmn "github.com/tendermint/tmlibs/common"
)
@ -58,7 +59,7 @@ type VoteSet struct {
sum int64 // Sum of voting power for seen votes, discounting conflicts
maj23 *BlockID // First 2/3 majority seen
votesByBlock map[string]*blockVotes // string(blockHash|blockParts) -> blockVotes
peerMaj23s map[string]BlockID // Maj23 for each peer
peerMaj23s map[p2p.ID]BlockID // Maj23 for each peer
}
// Constructs a new VoteSet struct used to accumulate votes for given height/round.
@ -77,7 +78,7 @@ func NewVoteSet(chainID string, height int64, round int, type_ byte, valSet *Val
sum: 0,
maj23: nil,
votesByBlock: make(map[string]*blockVotes, valSet.Size()),
peerMaj23s: make(map[string]BlockID),
peerMaj23s: make(map[p2p.ID]BlockID),
}
}
@ -290,7 +291,7 @@ func (voteSet *VoteSet) addVerifiedVote(vote *Vote, blockKey string, votingPower
// this can cause memory issues.
// TODO: implement ability to remove peers too
// NOTE: VoteSet must not be nil
func (voteSet *VoteSet) SetPeerMaj23(peerID string, blockID BlockID) {
func (voteSet *VoteSet) SetPeerMaj23(peerID p2p.ID, blockID BlockID) {
if voteSet == nil {
cmn.PanicSanity("SetPeerMaj23() on nil VoteSet")
}


Loading…
Cancel
Save