2019-01-24 20:15:10 +00:00
|
|
|
// Copyright (C) 2019 Storj Labs, Inc.
|
2018-05-16 19:47:59 +01:00
|
|
|
// See LICENSE for copying information.
|
|
|
|
|
|
|
|
package kademlia
|
|
|
|
|
|
|
|
import (
|
2018-12-22 04:51:42 +00:00
|
|
|
"context"
|
2018-08-13 09:39:45 +01:00
|
|
|
"encoding/binary"
|
2018-12-22 04:51:42 +00:00
|
|
|
"fmt"
|
2018-08-09 20:20:39 +01:00
|
|
|
"sync"
|
2018-05-16 19:47:59 +01:00
|
|
|
"time"
|
|
|
|
|
2019-02-16 03:23:35 +00:00
|
|
|
"github.com/gogo/protobuf/proto"
|
2018-08-09 20:20:39 +01:00
|
|
|
"github.com/zeebo/errs"
|
2018-12-22 00:48:53 +00:00
|
|
|
"go.uber.org/zap"
|
2018-06-05 12:48:19 +01:00
|
|
|
|
2019-04-22 10:07:50 +01:00
|
|
|
"storj.io/storj/pkg/overlay"
|
2018-09-18 05:39:06 +01:00
|
|
|
"storj.io/storj/pkg/pb"
|
2018-10-26 17:54:00 +01:00
|
|
|
"storj.io/storj/pkg/storj"
|
2018-08-09 20:20:39 +01:00
|
|
|
"storj.io/storj/storage"
|
2018-05-16 19:47:59 +01:00
|
|
|
)
|
|
|
|
|
2018-09-05 17:10:35 +01:00
|
|
|
const (
|
|
|
|
// KademliaBucket is the string representing the bucket used for the kademlia routing table k-bucket ids
|
|
|
|
KademliaBucket = "kbuckets"
|
|
|
|
// NodeBucket is the string representing the bucket used for the kademlia routing table node ids
|
|
|
|
NodeBucket = "nodes"
|
2019-07-01 22:20:19 +01:00
|
|
|
// AntechamberBucket is the string representing the bucket used for the kademlia antechamber nodes
|
|
|
|
AntechamberBucket = "antechamber"
|
2018-09-05 17:10:35 +01:00
|
|
|
)
|
|
|
|
|
2018-08-09 20:20:39 +01:00
|
|
|
// RoutingErr is the class for all errors pertaining to routing table operations
|
|
|
|
var RoutingErr = errs.Class("routing table error")
|
|
|
|
|
2018-11-29 18:39:27 +00:00
|
|
|
// Bucket IDs exist in the same address space as node IDs
|
2019-02-26 16:07:53 +00:00
|
|
|
type bucketID = storj.NodeID
|
2018-11-29 18:39:27 +00:00
|
|
|
|
2019-01-02 18:57:11 +00:00
|
|
|
var firstBucketID = bucketID{
|
|
|
|
0xFF, 0xFF, 0xFF, 0xFF,
|
|
|
|
0xFF, 0xFF, 0xFF, 0xFF,
|
|
|
|
0xFF, 0xFF, 0xFF, 0xFF,
|
|
|
|
0xFF, 0xFF, 0xFF, 0xFF,
|
|
|
|
|
|
|
|
0xFF, 0xFF, 0xFF, 0xFF,
|
|
|
|
0xFF, 0xFF, 0xFF, 0xFF,
|
|
|
|
0xFF, 0xFF, 0xFF, 0xFF,
|
|
|
|
0xFF, 0xFF, 0xFF, 0xFF,
|
|
|
|
}
|
|
|
|
|
|
|
|
var emptyBucketID = bucketID{}
|
|
|
|
|
2019-01-29 06:51:07 +00:00
|
|
|
// RoutingTableConfig configures the routing table
|
|
|
|
type RoutingTableConfig struct {
|
|
|
|
BucketSize int `help:"size of each Kademlia bucket" default:"20"`
|
|
|
|
ReplacementCacheSize int `help:"size of Kademlia replacement cache" default:"5"`
|
|
|
|
}
|
|
|
|
|
2018-08-09 20:20:39 +01:00
|
|
|
// RoutingTable implements the RoutingTable interface
|
|
|
|
type RoutingTable struct {
|
2018-12-22 00:48:53 +00:00
|
|
|
log *zap.Logger
|
2019-04-22 10:07:50 +01:00
|
|
|
self *overlay.NodeDossier
|
2018-08-17 20:11:46 +01:00
|
|
|
kadBucketDB storage.KeyValueStore
|
|
|
|
nodeBucketDB storage.KeyValueStore
|
2018-09-18 05:39:06 +01:00
|
|
|
transport *pb.NodeTransport
|
2018-08-17 20:11:46 +01:00
|
|
|
mutex *sync.Mutex
|
2019-02-22 18:39:29 +00:00
|
|
|
rcMutex *sync.Mutex
|
2018-11-29 18:39:27 +00:00
|
|
|
replacementCache map[bucketID][]*pb.Node
|
2018-08-17 20:11:46 +01:00
|
|
|
bucketSize int // max number of nodes stored in a kbucket = 20 (k)
|
|
|
|
rcBucketSize int // replacementCache bucket max length
|
2019-07-01 22:20:19 +01:00
|
|
|
antechamber storage.KeyValueStore
|
2018-06-05 12:48:19 +01:00
|
|
|
}
|
|
|
|
|
2018-08-09 20:20:39 +01:00
|
|
|
// NewRoutingTable returns a newly configured instance of a RoutingTable
|
2019-07-01 22:20:19 +01:00
|
|
|
func NewRoutingTable(logger *zap.Logger, localNode *overlay.NodeDossier, kdb, ndb, adb storage.KeyValueStore, config *RoutingTableConfig) (_ *RoutingTable, err error) {
|
2019-01-29 06:51:07 +00:00
|
|
|
if config == nil || config.BucketSize == 0 || config.ReplacementCacheSize == 0 {
|
|
|
|
// TODO: handle this more nicely
|
|
|
|
config = &RoutingTableConfig{
|
|
|
|
BucketSize: 20,
|
|
|
|
ReplacementCacheSize: 5,
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2018-08-09 20:20:39 +01:00
|
|
|
rt := &RoutingTable{
|
2018-12-22 00:48:53 +00:00
|
|
|
log: logger,
|
2018-11-20 16:54:52 +00:00
|
|
|
self: localNode,
|
|
|
|
kadBucketDB: kdb,
|
|
|
|
nodeBucketDB: ndb,
|
|
|
|
transport: &defaultTransport,
|
|
|
|
|
2018-08-17 20:11:46 +01:00
|
|
|
mutex: &sync.Mutex{},
|
2019-02-22 18:39:29 +00:00
|
|
|
rcMutex: &sync.Mutex{},
|
2018-11-29 18:39:27 +00:00
|
|
|
replacementCache: make(map[bucketID][]*pb.Node),
|
2018-11-20 16:54:52 +00:00
|
|
|
|
2019-01-29 06:51:07 +00:00
|
|
|
bucketSize: config.BucketSize,
|
|
|
|
rcBucketSize: config.ReplacementCacheSize,
|
2019-07-01 22:20:19 +01:00
|
|
|
antechamber: adb,
|
2018-08-09 20:20:39 +01:00
|
|
|
}
|
2019-06-13 15:51:50 +01:00
|
|
|
ok, err := rt.addNode(context.TODO(), &localNode.Node)
|
2018-08-27 18:28:16 +01:00
|
|
|
if !ok || err != nil {
|
2018-08-09 20:20:39 +01:00
|
|
|
return nil, RoutingErr.New("could not add localNode to routing table: %s", err)
|
|
|
|
}
|
|
|
|
return rt, nil
|
2018-05-16 19:47:59 +01:00
|
|
|
}
|
|
|
|
|
2019-02-22 18:39:29 +00:00
|
|
|
// Close closes without closing dependencies
|
2018-09-11 14:57:12 +01:00
|
|
|
func (rt *RoutingTable) Close() error {
|
2019-01-29 06:51:07 +00:00
|
|
|
return nil
|
2018-09-11 14:57:12 +01:00
|
|
|
}
|
|
|
|
|
2019-04-22 10:07:50 +01:00
|
|
|
// Local returns the local node
|
|
|
|
func (rt *RoutingTable) Local() overlay.NodeDossier {
|
2019-01-08 16:01:22 +00:00
|
|
|
rt.mutex.Lock()
|
|
|
|
defer rt.mutex.Unlock()
|
2019-04-22 10:07:50 +01:00
|
|
|
return *rt.self
|
|
|
|
}
|
|
|
|
|
|
|
|
// UpdateSelf updates the local node with the provided info
|
|
|
|
func (rt *RoutingTable) UpdateSelf(capacity *pb.NodeCapacity) {
|
|
|
|
rt.mutex.Lock()
|
|
|
|
defer rt.mutex.Unlock()
|
|
|
|
if capacity != nil {
|
|
|
|
rt.self.Capacity = *capacity
|
|
|
|
}
|
2018-05-16 19:47:59 +01:00
|
|
|
}
|
|
|
|
|
|
|
|
// K returns the currently configured maximum of nodes to store in a bucket
|
2018-08-09 20:20:39 +01:00
|
|
|
func (rt *RoutingTable) K() int {
|
|
|
|
return rt.bucketSize
|
2018-05-16 19:47:59 +01:00
|
|
|
}
|
|
|
|
|
2018-08-21 19:44:42 +01:00
|
|
|
// CacheSize returns the total current size of the replacement cache
|
2018-08-09 20:20:39 +01:00
|
|
|
func (rt *RoutingTable) CacheSize() int {
|
2018-08-21 19:44:42 +01:00
|
|
|
return rt.rcBucketSize
|
2018-05-16 19:47:59 +01:00
|
|
|
}
|
|
|
|
|
2018-12-18 18:04:46 +00:00
|
|
|
// GetNodes retrieves nodes within the same kbucket as the given node id
|
2018-08-09 20:20:39 +01:00
|
|
|
// Note: id doesn't need to be stored at time of search
|
2019-06-13 15:51:50 +01:00
|
|
|
func (rt *RoutingTable) GetNodes(ctx context.Context, id storj.NodeID) ([]*pb.Node, bool) {
|
2019-06-05 15:23:10 +01:00
|
|
|
defer mon.Task()(&ctx)(nil)
|
|
|
|
bID, err := rt.getKBucketID(ctx, id)
|
2018-06-05 12:48:19 +01:00
|
|
|
if err != nil {
|
2018-12-18 18:04:46 +00:00
|
|
|
return nil, false
|
2018-06-05 12:48:19 +01:00
|
|
|
}
|
2018-11-29 18:39:27 +00:00
|
|
|
if bID == (bucketID{}) {
|
2018-12-18 18:04:46 +00:00
|
|
|
return nil, false
|
2018-06-05 12:48:19 +01:00
|
|
|
}
|
2019-06-13 15:51:50 +01:00
|
|
|
unmarshaledNodes, err := rt.getUnmarshaledNodesFromBucket(ctx, bID)
|
2018-08-09 20:20:39 +01:00
|
|
|
if err != nil {
|
2018-12-18 18:04:46 +00:00
|
|
|
return nil, false
|
2018-08-09 20:20:39 +01:00
|
|
|
}
|
2018-12-18 18:04:46 +00:00
|
|
|
return unmarshaledNodes, true
|
2018-05-16 19:47:59 +01:00
|
|
|
}
|
|
|
|
|
2018-11-21 17:31:27 +00:00
|
|
|
// GetBucketIds returns a storage.Keys type of bucket ID's in the Kademlia instance
|
2019-06-13 15:51:50 +01:00
|
|
|
func (rt *RoutingTable) GetBucketIds(ctx context.Context) (_ storage.Keys, err error) {
|
2019-06-05 15:23:10 +01:00
|
|
|
defer mon.Task()(&ctx)(&err)
|
|
|
|
|
|
|
|
kbuckets, err := rt.kadBucketDB.List(ctx, nil, 0)
|
2018-11-21 17:31:27 +00:00
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
return kbuckets, nil
|
|
|
|
}
|
|
|
|
|
2019-02-28 19:55:27 +00:00
|
|
|
// DumpNodes iterates through all nodes in the nodeBucketDB and marshals them to &pb.Nodes, then returns them
|
2019-06-13 15:51:50 +01:00
|
|
|
func (rt *RoutingTable) DumpNodes(ctx context.Context) (_ []*pb.Node, err error) {
|
2019-06-05 15:23:10 +01:00
|
|
|
defer mon.Task()(&ctx)(&err)
|
|
|
|
|
2019-02-28 19:55:27 +00:00
|
|
|
var nodes []*pb.Node
|
2019-03-29 12:30:23 +00:00
|
|
|
var nodeErrors errs.Group
|
2019-02-28 19:55:27 +00:00
|
|
|
|
2019-06-05 15:23:10 +01:00
|
|
|
err = rt.iterateNodes(ctx, storj.NodeID{}, func(ctx context.Context, newID storj.NodeID, protoNode []byte) error {
|
2019-02-28 19:55:27 +00:00
|
|
|
newNode := pb.Node{}
|
|
|
|
err := proto.Unmarshal(protoNode, &newNode)
|
|
|
|
if err != nil {
|
2019-03-29 12:30:23 +00:00
|
|
|
nodeErrors.Add(err)
|
2019-02-28 19:55:27 +00:00
|
|
|
}
|
|
|
|
nodes = append(nodes, &newNode)
|
|
|
|
return nil
|
|
|
|
}, false)
|
|
|
|
|
|
|
|
if err != nil {
|
2019-03-29 12:30:23 +00:00
|
|
|
nodeErrors.Add(err)
|
2019-02-28 19:55:27 +00:00
|
|
|
}
|
|
|
|
|
2019-03-29 12:30:23 +00:00
|
|
|
return nodes, nodeErrors.Err()
|
2019-02-28 19:55:27 +00:00
|
|
|
}
|
|
|
|
|
2018-10-08 16:09:37 +01:00
|
|
|
// FindNear returns the node corresponding to the provided nodeID
|
2019-02-26 16:07:53 +00:00
|
|
|
// returns all Nodes (excluding self) closest via XOR to the provided nodeID up to the provided limit
|
2019-06-13 15:51:50 +01:00
|
|
|
func (rt *RoutingTable) FindNear(ctx context.Context, target storj.NodeID, limit int) (_ []*pb.Node, err error) {
|
2019-06-04 12:36:27 +01:00
|
|
|
defer mon.Task()(&ctx)(&err)
|
2019-02-16 03:23:35 +00:00
|
|
|
closestNodes := make([]*pb.Node, 0, limit+1)
|
2019-06-05 15:23:10 +01:00
|
|
|
err = rt.iterateNodes(ctx, storj.NodeID{}, func(ctx context.Context, newID storj.NodeID, protoNode []byte) error {
|
2019-02-16 03:23:35 +00:00
|
|
|
newPos := len(closestNodes)
|
|
|
|
for ; newPos > 0 && compareByXor(closestNodes[newPos-1].Id, newID, target) > 0; newPos-- {
|
|
|
|
}
|
|
|
|
if newPos != limit {
|
|
|
|
newNode := pb.Node{}
|
|
|
|
err := proto.Unmarshal(protoNode, &newNode)
|
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
2019-04-22 10:07:50 +01:00
|
|
|
closestNodes = append(closestNodes, &newNode)
|
|
|
|
if newPos != len(closestNodes) { //reorder
|
|
|
|
copy(closestNodes[newPos+1:], closestNodes[newPos:])
|
|
|
|
closestNodes[newPos] = &newNode
|
|
|
|
if len(closestNodes) > limit {
|
|
|
|
closestNodes = closestNodes[:limit]
|
2019-02-16 03:23:35 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return nil
|
2019-02-26 16:07:53 +00:00
|
|
|
}, true)
|
2019-02-16 03:23:35 +00:00
|
|
|
return closestNodes, Error.Wrap(err)
|
2018-05-16 19:47:59 +01:00
|
|
|
}
|
|
|
|
|
2018-08-27 18:28:16 +01:00
|
|
|
// ConnectionSuccess updates or adds a node to the routing table when
|
2018-08-21 19:44:42 +01:00
|
|
|
// a successful connection is made to the node on the network
|
2019-06-13 15:51:50 +01:00
|
|
|
func (rt *RoutingTable) ConnectionSuccess(ctx context.Context, node *pb.Node) (err error) {
|
2019-06-04 12:36:27 +01:00
|
|
|
defer mon.Task()(&ctx)(&err)
|
2018-11-20 16:54:52 +00:00
|
|
|
// valid to connect to node without ID but don't store connection
|
2018-11-29 18:39:27 +00:00
|
|
|
if node.Id == (storj.NodeID{}) {
|
2018-11-20 16:54:52 +00:00
|
|
|
return nil
|
|
|
|
}
|
2019-01-02 18:47:34 +00:00
|
|
|
|
2019-06-05 15:23:10 +01:00
|
|
|
v, err := rt.nodeBucketDB.Get(ctx, storage.Key(node.Id.Bytes()))
|
2018-08-21 19:44:42 +01:00
|
|
|
if err != nil && !storage.ErrKeyNotFound.Has(err) {
|
|
|
|
return RoutingErr.New("could not get node %s", err)
|
|
|
|
}
|
|
|
|
if v != nil {
|
2019-06-13 15:51:50 +01:00
|
|
|
err = rt.updateNode(ctx, node)
|
2018-08-21 19:44:42 +01:00
|
|
|
if err != nil {
|
|
|
|
return RoutingErr.New("could not update node %s", err)
|
|
|
|
}
|
|
|
|
return nil
|
|
|
|
}
|
2019-06-13 15:51:50 +01:00
|
|
|
_, err = rt.addNode(ctx, node)
|
2018-08-21 19:44:42 +01:00
|
|
|
if err != nil {
|
|
|
|
return RoutingErr.New("could not add node %s", err)
|
|
|
|
}
|
2019-02-22 18:39:29 +00:00
|
|
|
|
2018-08-21 19:44:42 +01:00
|
|
|
return nil
|
2018-05-16 19:47:59 +01:00
|
|
|
}
|
|
|
|
|
2018-08-21 19:44:42 +01:00
|
|
|
// ConnectionFailed removes a node from the routing table when
|
|
|
|
// a connection fails for the node on the network
|
2019-06-13 15:51:50 +01:00
|
|
|
func (rt *RoutingTable) ConnectionFailed(ctx context.Context, node *pb.Node) (err error) {
|
2019-06-04 12:36:27 +01:00
|
|
|
defer mon.Task()(&ctx)(&err)
|
2019-06-13 15:51:50 +01:00
|
|
|
err = rt.removeNode(ctx, node)
|
2018-08-21 19:44:42 +01:00
|
|
|
if err != nil {
|
|
|
|
return RoutingErr.New("could not remove node %s", err)
|
|
|
|
}
|
|
|
|
return nil
|
2018-05-16 19:47:59 +01:00
|
|
|
}
|
|
|
|
|
2018-12-20 21:45:06 +00:00
|
|
|
// SetBucketTimestamp records the time of the last node lookup for a bucket
|
2019-06-13 15:51:50 +01:00
|
|
|
func (rt *RoutingTable) SetBucketTimestamp(ctx context.Context, bIDBytes []byte, now time.Time) (err error) {
|
2019-06-05 15:23:10 +01:00
|
|
|
defer mon.Task()(&ctx)(&err)
|
2018-08-09 20:20:39 +01:00
|
|
|
rt.mutex.Lock()
|
|
|
|
defer rt.mutex.Unlock()
|
2019-06-05 15:23:10 +01:00
|
|
|
err = rt.createOrUpdateKBucket(ctx, keyToBucketID(bIDBytes), now)
|
2018-06-05 12:48:19 +01:00
|
|
|
if err != nil {
|
2018-08-09 20:20:39 +01:00
|
|
|
return NodeErr.New("could not update bucket timestamp %s", err)
|
2018-06-05 12:48:19 +01:00
|
|
|
}
|
2018-05-16 19:47:59 +01:00
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
2018-12-20 21:45:06 +00:00
|
|
|
// GetBucketTimestamp retrieves time of the last node lookup for a bucket
|
2019-06-13 15:51:50 +01:00
|
|
|
func (rt *RoutingTable) GetBucketTimestamp(ctx context.Context, bIDBytes []byte) (_ time.Time, err error) {
|
2019-06-05 15:23:10 +01:00
|
|
|
defer mon.Task()(&ctx)(&err)
|
|
|
|
|
|
|
|
t, err := rt.kadBucketDB.Get(ctx, bIDBytes)
|
2018-08-09 20:20:39 +01:00
|
|
|
if err != nil {
|
|
|
|
return time.Now(), RoutingErr.New("could not get bucket timestamp %s", err)
|
|
|
|
}
|
|
|
|
timestamp, _ := binary.Varint(t)
|
|
|
|
return time.Unix(0, timestamp).UTC(), nil
|
2018-05-16 19:47:59 +01:00
|
|
|
}
|
2018-10-16 16:22:31 +01:00
|
|
|
|
2019-06-05 15:23:10 +01:00
|
|
|
func (rt *RoutingTable) iterateNodes(ctx context.Context, start storj.NodeID, f func(context.Context, storj.NodeID, []byte) error, skipSelf bool) (err error) {
|
|
|
|
defer mon.Task()(&ctx)(&err)
|
|
|
|
return rt.nodeBucketDB.Iterate(ctx, storage.IterateOptions{First: storage.Key(start.Bytes()), Recurse: true},
|
|
|
|
func(ctx context.Context, it storage.Iterator) error {
|
2019-02-16 03:23:35 +00:00
|
|
|
var item storage.ListItem
|
2019-06-05 15:23:10 +01:00
|
|
|
for it.Next(ctx, &item) {
|
2019-02-16 03:23:35 +00:00
|
|
|
nodeID, err := storj.NodeIDFromBytes(item.Key)
|
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
2019-02-26 16:07:53 +00:00
|
|
|
if skipSelf && nodeID == rt.self.Id {
|
|
|
|
continue
|
2019-02-16 03:23:35 +00:00
|
|
|
}
|
2019-06-05 15:23:10 +01:00
|
|
|
err = f(ctx, nodeID, item.Value)
|
2019-02-16 03:23:35 +00:00
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return nil
|
|
|
|
},
|
|
|
|
)
|
2018-10-16 16:22:31 +01:00
|
|
|
}
|
2018-12-22 04:51:42 +00:00
|
|
|
|
|
|
|
// ConnFailure implements the Transport failure function
|
|
|
|
func (rt *RoutingTable) ConnFailure(ctx context.Context, node *pb.Node, err error) {
|
2019-06-13 15:51:50 +01:00
|
|
|
err2 := rt.ConnectionFailed(ctx, node)
|
2018-12-22 04:51:42 +00:00
|
|
|
if err2 != nil {
|
|
|
|
zap.L().Debug(fmt.Sprintf("error with ConnFailure hook %+v : %+v", err, err2))
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
// ConnSuccess implements the Transport success function
|
|
|
|
func (rt *RoutingTable) ConnSuccess(ctx context.Context, node *pb.Node) {
|
2019-06-13 15:51:50 +01:00
|
|
|
err := rt.ConnectionSuccess(ctx, node)
|
2018-12-22 04:51:42 +00:00
|
|
|
if err != nil {
|
|
|
|
zap.L().Debug("connection success error:", zap.Error(err))
|
|
|
|
}
|
|
|
|
}
|