rename bitswap and move to stores
This commit is contained in:
parent
ad63969f20
commit
f4f4ff9ade
49
dagger.nim
49
dagger.nim
|
@ -1,49 +0,0 @@
|
||||||
import pkg/chronos
|
|
||||||
import pkg/libp2p/peerinfo
|
|
||||||
import pkg/libp2p/multiaddress
|
|
||||||
import ./ipfs/p2p/switch
|
|
||||||
import ./ipfs/repo
|
|
||||||
import ./ipfs/chunking
|
|
||||||
import ./ipfs/bitswap
|
|
||||||
|
|
||||||
export peerinfo except IPFS
|
|
||||||
export multiaddress except IPFS
|
|
||||||
|
|
||||||
type
|
|
||||||
Ipfs* = ref object
|
|
||||||
repo: Repo
|
|
||||||
switch: Switch
|
|
||||||
bitswap: Bitswap
|
|
||||||
|
|
||||||
proc info*(ipfs: Ipfs): PeerInfo =
|
|
||||||
ipfs.switch.peerInfo
|
|
||||||
|
|
||||||
proc start*(_: type Ipfs, addresses: seq[MultiAddress]): Future[Ipfs] {.async.} =
|
|
||||||
let repo = Repo()
|
|
||||||
let switch = Switch.create()
|
|
||||||
let bitswap = Bitswap.start(switch, repo)
|
|
||||||
switch.peerInfo.addrs.add(addresses)
|
|
||||||
discard await switch.start()
|
|
||||||
result = Ipfs(repo: repo, switch: switch, bitswap: bitswap)
|
|
||||||
|
|
||||||
proc start*(_: type Ipfs, address: MultiAddress): Future[Ipfs] {.async.} =
|
|
||||||
result = await Ipfs.start(@[address])
|
|
||||||
|
|
||||||
proc start*(_: type Ipfs): Future[Ipfs] {.async.} =
|
|
||||||
result = await Ipfs.start(@[])
|
|
||||||
|
|
||||||
proc connect*(peer: Ipfs, info: PeerInfo) {.async.} =
|
|
||||||
await peer.bitswap.connect(info)
|
|
||||||
|
|
||||||
proc add*(peer: Ipfs, input: File): Future[Cid] {.async.} =
|
|
||||||
let obj = createObject(input)
|
|
||||||
peer.repo.store(obj)
|
|
||||||
result = obj.cid
|
|
||||||
|
|
||||||
proc get*(peer: Ipfs, identifier: Cid, output: File) {.async.} =
|
|
||||||
let obj = await peer.bitswap.retrieve(identifier)
|
|
||||||
if obj.isSome:
|
|
||||||
obj.get().writeToFile(output)
|
|
||||||
|
|
||||||
proc stop*(peer: Ipfs) {.async.} =
|
|
||||||
await peer.switch.stop()
|
|
|
@ -14,18 +14,18 @@ import pkg/chronos
|
||||||
import pkg/libp2p
|
import pkg/libp2p
|
||||||
import pkg/libp2p/errors
|
import pkg/libp2p/errors
|
||||||
|
|
||||||
import ./bitswap/protobuf/bitswap as pb
|
import ./blockexc/protobuf/blockexc as pb
|
||||||
import ./blocktype as bt
|
import ./blocktype as bt
|
||||||
import ./stores/blockstore
|
import ./stores/blockstore
|
||||||
import ./utils/asyncheapqueue
|
import ./utils/asyncheapqueue
|
||||||
|
|
||||||
import ./bitswap/network
|
import ./blockexc/network
|
||||||
import ./bitswap/engine
|
import ./blockexc/engine
|
||||||
|
|
||||||
export network, blockstore, asyncheapqueue, engine
|
export network, blockstore, asyncheapqueue, engine
|
||||||
|
|
||||||
logScope:
|
logScope:
|
||||||
topics = "dagger bitswap"
|
topics = "dagger blockexc"
|
||||||
|
|
||||||
const
|
const
|
||||||
DefaultTaskQueueSize = 100
|
DefaultTaskQueueSize = 100
|
||||||
|
@ -33,58 +33,58 @@ const
|
||||||
DefaultMaxRetries = 3
|
DefaultMaxRetries = 3
|
||||||
|
|
||||||
type
|
type
|
||||||
Bitswap* = ref object of BlockStore
|
BlockExc* = ref object of BlockStore
|
||||||
engine*: BitswapEngine # bitswap decision engine
|
engine*: BlockExcEngine # blockexc decision engine
|
||||||
taskQueue*: AsyncHeapQueue[BitswapPeerCtx] # peers we're currently processing tasks for
|
taskQueue*: AsyncHeapQueue[BlockExcPeerCtx] # peers we're currently processing tasks for
|
||||||
bitswapTasks: seq[Future[void]] # future to control bitswap task
|
blockexcTasks: seq[Future[void]] # future to control blockexc task
|
||||||
bitswapRunning: bool # indicates if the bitswap task is running
|
blockexcRunning: bool # indicates if the blockexc task is running
|
||||||
concurrentTasks: int # number of concurrent peers we're serving at any given time
|
concurrentTasks: int # number of concurrent peers we're serving at any given time
|
||||||
maxRetries: int # max number of tries for a failed block
|
maxRetries: int # max number of tries for a failed block
|
||||||
taskHandler: TaskHandler # handler provided by the engine called by the runner
|
taskHandler: TaskHandler # handler provided by the engine called by the runner
|
||||||
|
|
||||||
proc bitswapTaskRunner(b: Bitswap) {.async.} =
|
proc blockexcTaskRunner(b: BlockExc) {.async.} =
|
||||||
## process tasks
|
## process tasks
|
||||||
##
|
##
|
||||||
|
|
||||||
while b.bitswapRunning:
|
while b.blockexcRunning:
|
||||||
let peerCtx = await b.taskQueue.pop()
|
let peerCtx = await b.taskQueue.pop()
|
||||||
asyncSpawn b.taskHandler(peerCtx)
|
asyncSpawn b.taskHandler(peerCtx)
|
||||||
|
|
||||||
trace "Exiting bitswap task runner"
|
trace "Exiting blockexc task runner"
|
||||||
|
|
||||||
proc start*(b: Bitswap) {.async.} =
|
proc start*(b: BlockExc) {.async.} =
|
||||||
## Start the bitswap task
|
## Start the blockexc task
|
||||||
##
|
##
|
||||||
|
|
||||||
trace "bitswap start"
|
trace "blockexc start"
|
||||||
|
|
||||||
if b.bitswapTasks.len > 0:
|
if b.blockexcTasks.len > 0:
|
||||||
warn "Starting bitswap twice"
|
warn "Starting blockexc twice"
|
||||||
return
|
return
|
||||||
|
|
||||||
b.bitswapRunning = true
|
b.blockexcRunning = true
|
||||||
for i in 0..<b.concurrentTasks:
|
for i in 0..<b.concurrentTasks:
|
||||||
b.bitswapTasks.add(b.bitswapTaskRunner)
|
b.blockexcTasks.add(b.blockexcTaskRunner)
|
||||||
|
|
||||||
proc stop*(b: Bitswap) {.async.} =
|
proc stop*(b: BlockExc) {.async.} =
|
||||||
## Stop the bitswap bitswap
|
## Stop the blockexc blockexc
|
||||||
##
|
##
|
||||||
|
|
||||||
trace "Bitswap stop"
|
trace "BlockExc stop"
|
||||||
if b.bitswapTasks.len <= 0:
|
if b.blockexcTasks.len <= 0:
|
||||||
warn "Stopping bitswap without starting it"
|
warn "Stopping blockexc without starting it"
|
||||||
return
|
return
|
||||||
|
|
||||||
b.bitswapRunning = false
|
b.blockexcRunning = false
|
||||||
for t in b.bitswapTasks:
|
for t in b.blockexcTasks:
|
||||||
if not t.finished:
|
if not t.finished:
|
||||||
trace "Awaiting task to stop"
|
trace "Awaiting task to stop"
|
||||||
t.cancel()
|
t.cancel()
|
||||||
trace "Task stopped"
|
trace "Task stopped"
|
||||||
|
|
||||||
trace "Bitswap stopped"
|
trace "BlockExc stopped"
|
||||||
|
|
||||||
method getBlocks*(b: Bitswap, cid: seq[Cid]): Future[seq[bt.Block]] {.async.} =
|
method getBlocks*(b: BlockExc, cid: seq[Cid]): Future[seq[bt.Block]] {.async.} =
|
||||||
## Get a block from a remote peer
|
## Get a block from a remote peer
|
||||||
##
|
##
|
||||||
|
|
||||||
|
@ -95,42 +95,40 @@ method getBlocks*(b: Bitswap, cid: seq[Cid]): Future[seq[bt.Block]] {.async.} =
|
||||||
it.read
|
it.read
|
||||||
)
|
)
|
||||||
|
|
||||||
method putBlocks*(b: Bitswap, blocks: seq[bt.Block]) =
|
method putBlocks*(b: BlockExc, blocks: seq[bt.Block]) =
|
||||||
b.engine.resolveBlocks(blocks)
|
b.engine.resolveBlocks(blocks)
|
||||||
|
|
||||||
procCall BlockStore(b).putBlocks(blocks)
|
procCall BlockStore(b).putBlocks(blocks)
|
||||||
|
|
||||||
proc new*(
|
func new*(
|
||||||
T: type Bitswap,
|
T: type BlockExc,
|
||||||
localStore: BlockStore,
|
localStore: BlockStore,
|
||||||
wallet: WalletRef,
|
wallet: WalletRef,
|
||||||
network: BitswapNetwork,
|
network: BlockExcNetwork,
|
||||||
concurrentTasks = DefaultConcurrentTasks,
|
concurrentTasks = DefaultConcurrentTasks,
|
||||||
maxRetries = DefaultMaxRetries,
|
maxRetries = DefaultMaxRetries,
|
||||||
peersPerRequest = DefaultMaxPeersPerRequest): T =
|
peersPerRequest = DefaultMaxPeersPerRequest): T =
|
||||||
|
|
||||||
let engine = BitswapEngine.new(
|
let engine = BlockExcEngine.new(
|
||||||
localStore = localStore,
|
localStore = localStore,
|
||||||
wallet = wallet,
|
wallet = wallet,
|
||||||
peersPerRequest = peersPerRequest,
|
peersPerRequest = peersPerRequest,
|
||||||
request = network.request,
|
request = network.request,
|
||||||
)
|
)
|
||||||
|
|
||||||
let b = Bitswap(
|
let b = BlockExc(
|
||||||
engine: engine,
|
engine: engine,
|
||||||
taskQueue: newAsyncHeapQueue[BitswapPeerCtx](DefaultTaskQueueSize),
|
taskQueue: newAsyncHeapQueue[BlockExcPeerCtx](DefaultTaskQueueSize),
|
||||||
concurrentTasks: concurrentTasks,
|
concurrentTasks: concurrentTasks,
|
||||||
maxRetries: maxRetries,
|
maxRetries: maxRetries,
|
||||||
)
|
)
|
||||||
|
|
||||||
# attach engine's task handler
|
# attach engine's task handler
|
||||||
b.taskHandler = proc(task: BitswapPeerCtx):
|
b.taskHandler = proc(task: BlockExcPeerCtx): Future[void] {.gcsafe.} =
|
||||||
Future[void] {.gcsafe.} =
|
|
||||||
engine.taskHandler(task)
|
engine.taskHandler(task)
|
||||||
|
|
||||||
# attach task scheduler to engine
|
# attach task scheduler to engine
|
||||||
engine.scheduleTask = proc(task: BitswapPeerCtx):
|
engine.scheduleTask = proc(task: BlockExcPeerCtx): bool {.gcsafe} =
|
||||||
bool {.gcsafe} =
|
|
||||||
b.taskQueue.pushOrUpdateNoWait(task).isOk()
|
b.taskQueue.pushOrUpdateNoWait(task).isOk()
|
||||||
|
|
||||||
proc peerEventHandler(peerInfo: PeerInfo, event: PeerEvent) {.async.} =
|
proc peerEventHandler(peerInfo: PeerInfo, event: PeerEvent) {.async.} =
|
||||||
|
@ -167,7 +165,7 @@ proc new*(
|
||||||
proc paymentHandler(peer: PeerId, payment: SignedState) =
|
proc paymentHandler(peer: PeerId, payment: SignedState) =
|
||||||
engine.paymentHandler(peer, payment)
|
engine.paymentHandler(peer, payment)
|
||||||
|
|
||||||
network.handlers = BitswapHandlers(
|
network.handlers = BlockExcHandlers(
|
||||||
onWantList: blockWantListHandler,
|
onWantList: blockWantListHandler,
|
||||||
onBlocks: blocksHandler,
|
onBlocks: blocksHandler,
|
||||||
onPresence: blockPresenceHandler,
|
onPresence: blockPresenceHandler,
|
||||||
|
|
|
@ -54,7 +54,7 @@ method delBlocks*(s: MemoryStore, cids: seq[Cid]) =
|
||||||
|
|
||||||
procCall BlockStore(s).delBlocks(cids)
|
procCall BlockStore(s).delBlocks(cids)
|
||||||
|
|
||||||
proc new*(T: type MemoryStore, blocks: openArray[Block] = []): MemoryStore =
|
func new*(T: type MemoryStore, blocks: openArray[Block] = []): MemoryStore =
|
||||||
MemoryStore(
|
MemoryStore(
|
||||||
blocks: @blocks
|
blocks: @blocks
|
||||||
)
|
)
|
||||||
|
|
|
@ -15,11 +15,11 @@ import pkg/chronicles
|
||||||
import pkg/libp2p
|
import pkg/libp2p
|
||||||
import pkg/libp2p/errors
|
import pkg/libp2p/errors
|
||||||
|
|
||||||
import ./protobuf/bitswap as pb
|
import ./protobuf/blockexc
|
||||||
import ./protobuf/presence
|
import ./protobuf/presence
|
||||||
import ../blocktype as bt
|
import ../blocktype as bt
|
||||||
import ../stores/blockstore
|
import ../blockstore
|
||||||
import ../utils/asyncheapqueue
|
import ../../utils/asyncheapqueue
|
||||||
|
|
||||||
import ./network
|
import ./network
|
||||||
import ./pendingblocks
|
import ./pendingblocks
|
||||||
|
@ -29,24 +29,24 @@ import ./engine/payments
|
||||||
export peercontext
|
export peercontext
|
||||||
|
|
||||||
logScope:
|
logScope:
|
||||||
topics = "dagger bitswap engine"
|
topics = "dagger blockexc engine"
|
||||||
|
|
||||||
const
|
const
|
||||||
DefaultTimeout* = 5.seconds
|
DefaultTimeout* = 5.seconds
|
||||||
DefaultMaxPeersPerRequest* = 10
|
DefaultMaxPeersPerRequest* = 10
|
||||||
|
|
||||||
type
|
type
|
||||||
TaskHandler* = proc(task: BitswapPeerCtx): Future[void] {.gcsafe.}
|
TaskHandler* = proc(task: BlockExcPeerCtx): Future[void] {.gcsafe.}
|
||||||
TaskScheduler* = proc(task: BitswapPeerCtx): bool {.gcsafe.}
|
TaskScheduler* = proc(task: BlockExcPeerCtx): bool {.gcsafe.}
|
||||||
|
|
||||||
BitswapEngine* = ref object of RootObj
|
BlockExcEngine* = ref object of RootObj
|
||||||
localStore*: BlockStore # where we localStore blocks for this instance
|
localStore*: BlockStore # where we localStore blocks for this instance
|
||||||
peers*: seq[BitswapPeerCtx] # peers we're currently actively exchanging with
|
peers*: seq[BlockExcPeerCtx] # peers we're currently actively exchanging with
|
||||||
wantList*: seq[Cid] # local wants list
|
wantList*: seq[Cid] # local wants list
|
||||||
pendingBlocks*: PendingBlocksManager # blocks we're awaiting to be resolved
|
pendingBlocks*: PendingBlocksManager # blocks we're awaiting to be resolved
|
||||||
peersPerRequest: int # max number of peers to request from
|
peersPerRequest: int # max number of peers to request from
|
||||||
scheduleTask*: TaskScheduler # schedule a new task with the task runner
|
scheduleTask*: TaskScheduler # schedule a new task with the task runner
|
||||||
request*: BitswapRequest # bitswap network requests
|
request*: BlockExcRequest # block exchange network requests
|
||||||
wallet*: WalletRef # nitro wallet for micropayments
|
wallet*: WalletRef # nitro wallet for micropayments
|
||||||
pricing*: ?Pricing # optional bandwidth pricing
|
pricing*: ?Pricing # optional bandwidth pricing
|
||||||
|
|
||||||
|
@ -60,7 +60,7 @@ proc contains*(a: AsyncHeapQueue[Entry], b: Cid): bool =
|
||||||
|
|
||||||
a.anyIt( it.cid == b )
|
a.anyIt( it.cid == b )
|
||||||
|
|
||||||
proc getPeerCtx*(b: BitswapEngine, peerId: PeerID): BitswapPeerCtx =
|
proc getPeerCtx*(b: BlockExcEngine, peerId: PeerID): BlockExcPeerCtx =
|
||||||
## Get the peer's context
|
## Get the peer's context
|
||||||
##
|
##
|
||||||
|
|
||||||
|
@ -69,7 +69,7 @@ proc getPeerCtx*(b: BitswapEngine, peerId: PeerID): BitswapPeerCtx =
|
||||||
return peer[0]
|
return peer[0]
|
||||||
|
|
||||||
proc requestBlocks*(
|
proc requestBlocks*(
|
||||||
b: BitswapEngine,
|
b: BlockExcEngine,
|
||||||
cids: seq[Cid],
|
cids: seq[Cid],
|
||||||
timeout = DefaultTimeout): seq[Future[bt.Block]] =
|
timeout = DefaultTimeout): seq[Future[bt.Block]] =
|
||||||
## Request a block from remotes
|
## Request a block from remotes
|
||||||
|
@ -91,12 +91,11 @@ proc requestBlocks*(
|
||||||
blocks.add(
|
blocks.add(
|
||||||
b.pendingBlocks.addOrAwait(c).wait(timeout))
|
b.pendingBlocks.addOrAwait(c).wait(timeout))
|
||||||
|
|
||||||
|
|
||||||
var peers = b.peers
|
var peers = b.peers
|
||||||
|
|
||||||
# get the first peer with at least one (any)
|
# get the first peer with at least one (any)
|
||||||
# matching cid
|
# matching cid
|
||||||
var blockPeer: BitswapPeerCtx
|
var blockPeer: BlockExcPeerCtx
|
||||||
for i, p in peers:
|
for i, p in peers:
|
||||||
let has = cids.anyIt(
|
let has = cids.anyIt(
|
||||||
it in p.peerHave
|
it in p.peerHave
|
||||||
|
@ -125,7 +124,7 @@ proc requestBlocks*(
|
||||||
if peers.len == 0:
|
if peers.len == 0:
|
||||||
return blocks # no peers to send wants to
|
return blocks # no peers to send wants to
|
||||||
|
|
||||||
template sendWants(ctx: BitswapPeerCtx) =
|
template sendWants(ctx: BlockExcPeerCtx) =
|
||||||
# just send wants
|
# just send wants
|
||||||
b.request.sendWantList(
|
b.request.sendWantList(
|
||||||
ctx.id,
|
ctx.id,
|
||||||
|
@ -142,7 +141,7 @@ proc requestBlocks*(
|
||||||
return blocks
|
return blocks
|
||||||
|
|
||||||
proc blockPresenceHandler*(
|
proc blockPresenceHandler*(
|
||||||
b: BitswapEngine,
|
b: BlockExcEngine,
|
||||||
peer: PeerID,
|
peer: PeerID,
|
||||||
blocks: seq[BlockPresence]) =
|
blocks: seq[BlockPresence]) =
|
||||||
## Handle block presence
|
## Handle block presence
|
||||||
|
@ -156,7 +155,7 @@ proc blockPresenceHandler*(
|
||||||
if presence =? Presence.init(blk):
|
if presence =? Presence.init(blk):
|
||||||
peerCtx.updatePresence(presence)
|
peerCtx.updatePresence(presence)
|
||||||
|
|
||||||
proc scheduleTasks(b: BitswapEngine, blocks: seq[bt.Block]) =
|
proc scheduleTasks(b: BlockExcEngine, blocks: seq[bt.Block]) =
|
||||||
trace "Schedule a task for new blocks"
|
trace "Schedule a task for new blocks"
|
||||||
|
|
||||||
let cids = blocks.mapIt( it.cid )
|
let cids = blocks.mapIt( it.cid )
|
||||||
|
@ -170,7 +169,7 @@ proc scheduleTasks(b: BitswapEngine, blocks: seq[bt.Block]) =
|
||||||
trace "Unable to schedule task for peer", peer = p.id
|
trace "Unable to schedule task for peer", peer = p.id
|
||||||
break # do next peer
|
break # do next peer
|
||||||
|
|
||||||
proc resolveBlocks*(b: BitswapEngine, blocks: seq[bt.Block]) =
|
proc resolveBlocks*(b: BlockExcEngine, blocks: seq[bt.Block]) =
|
||||||
## Resolve pending blocks from the pending blocks manager
|
## Resolve pending blocks from the pending blocks manager
|
||||||
## and schedule any new task to be ran
|
## and schedule any new task to be ran
|
||||||
##
|
##
|
||||||
|
@ -179,8 +178,8 @@ proc resolveBlocks*(b: BitswapEngine, blocks: seq[bt.Block]) =
|
||||||
b.pendingBlocks.resolve(blocks)
|
b.pendingBlocks.resolve(blocks)
|
||||||
b.scheduleTasks(blocks)
|
b.scheduleTasks(blocks)
|
||||||
|
|
||||||
proc payForBlocks(engine: BitswapEngine,
|
proc payForBlocks(engine: BlockExcEngine,
|
||||||
peer: BitswapPeerCtx,
|
peer: BlockExcPeerCtx,
|
||||||
blocks: seq[bt.Block]) =
|
blocks: seq[bt.Block]) =
|
||||||
let sendPayment = engine.request.sendPayment
|
let sendPayment = engine.request.sendPayment
|
||||||
if sendPayment.isNil:
|
if sendPayment.isNil:
|
||||||
|
@ -191,7 +190,7 @@ proc payForBlocks(engine: BitswapEngine,
|
||||||
sendPayment(peer.id, payment)
|
sendPayment(peer.id, payment)
|
||||||
|
|
||||||
proc blocksHandler*(
|
proc blocksHandler*(
|
||||||
b: BitswapEngine,
|
b: BlockExcEngine,
|
||||||
peer: PeerID,
|
peer: PeerID,
|
||||||
blocks: seq[bt.Block]) =
|
blocks: seq[bt.Block]) =
|
||||||
## handle incoming blocks
|
## handle incoming blocks
|
||||||
|
@ -206,7 +205,7 @@ proc blocksHandler*(
|
||||||
b.payForBlocks(peerCtx, blocks)
|
b.payForBlocks(peerCtx, blocks)
|
||||||
|
|
||||||
proc wantListHandler*(
|
proc wantListHandler*(
|
||||||
b: BitswapEngine,
|
b: BlockExcEngine,
|
||||||
peer: PeerID,
|
peer: PeerID,
|
||||||
wantList: WantList) =
|
wantList: WantList) =
|
||||||
## Handle incoming want lists
|
## Handle incoming want lists
|
||||||
|
@ -250,14 +249,14 @@ proc wantListHandler*(
|
||||||
if not b.scheduleTask(peerCtx):
|
if not b.scheduleTask(peerCtx):
|
||||||
trace "Unable to schedule task for peer", peer
|
trace "Unable to schedule task for peer", peer
|
||||||
|
|
||||||
proc accountHandler*(engine: BitswapEngine, peer: PeerID, account: Account) =
|
proc accountHandler*(engine: BlockExcEngine, peer: PeerID, account: Account) =
|
||||||
let context = engine.getPeerCtx(peer)
|
let context = engine.getPeerCtx(peer)
|
||||||
if context.isNil:
|
if context.isNil:
|
||||||
return
|
return
|
||||||
|
|
||||||
context.account = account.some
|
context.account = account.some
|
||||||
|
|
||||||
proc paymentHandler*(engine: BitswapEngine, peer: PeerId, payment: SignedState) =
|
proc paymentHandler*(engine: BlockExcEngine, peer: PeerId, payment: SignedState) =
|
||||||
without context =? engine.getPeerCtx(peer).option and
|
without context =? engine.getPeerCtx(peer).option and
|
||||||
account =? context.account:
|
account =? context.account:
|
||||||
return
|
return
|
||||||
|
@ -268,14 +267,14 @@ proc paymentHandler*(engine: BitswapEngine, peer: PeerId, payment: SignedState)
|
||||||
else:
|
else:
|
||||||
context.paymentChannel = engine.wallet.acceptChannel(payment).option
|
context.paymentChannel = engine.wallet.acceptChannel(payment).option
|
||||||
|
|
||||||
proc setupPeer*(b: BitswapEngine, peer: PeerID) =
|
proc setupPeer*(b: BlockExcEngine, peer: PeerID) =
|
||||||
## Perform initial setup, such as want
|
## Perform initial setup, such as want
|
||||||
## list exchange
|
## list exchange
|
||||||
##
|
##
|
||||||
|
|
||||||
trace "Setting up new peer", peer
|
trace "Setting up new peer", peer
|
||||||
if peer notin b.peers:
|
if peer notin b.peers:
|
||||||
b.peers.add(BitswapPeerCtx(
|
b.peers.add(BlockExcPeerCtx(
|
||||||
id: peer
|
id: peer
|
||||||
))
|
))
|
||||||
|
|
||||||
|
@ -286,7 +285,7 @@ proc setupPeer*(b: BitswapEngine, peer: PeerID) =
|
||||||
if address =? b.pricing.?address:
|
if address =? b.pricing.?address:
|
||||||
b.request.sendAccount(peer, Account(address: address))
|
b.request.sendAccount(peer, Account(address: address))
|
||||||
|
|
||||||
proc dropPeer*(b: BitswapEngine, peer: PeerID) =
|
proc dropPeer*(b: BlockExcEngine, peer: PeerID) =
|
||||||
## Cleanup disconnected peer
|
## Cleanup disconnected peer
|
||||||
##
|
##
|
||||||
|
|
||||||
|
@ -295,7 +294,7 @@ proc dropPeer*(b: BitswapEngine, peer: PeerID) =
|
||||||
# drop the peer from the peers table
|
# drop the peer from the peers table
|
||||||
b.peers.keepItIf( it.id != peer )
|
b.peers.keepItIf( it.id != peer )
|
||||||
|
|
||||||
proc taskHandler*(b: BitswapEngine, task: BitswapPeerCtx) {.gcsafe, async.} =
|
proc taskHandler*(b: BlockExcEngine, task: BlockExcPeerCtx) {.gcsafe, async.} =
|
||||||
trace "Handling task for peer", peer = task.id
|
trace "Handling task for peer", peer = task.id
|
||||||
|
|
||||||
var wantsBlocks = newAsyncHeapQueue[Entry](queueType = QueueType.Max)
|
var wantsBlocks = newAsyncHeapQueue[Entry](queueType = QueueType.Max)
|
||||||
|
@ -334,19 +333,19 @@ proc taskHandler*(b: BitswapEngine, task: BitswapPeerCtx) {.gcsafe, async.} =
|
||||||
if wants.len > 0:
|
if wants.len > 0:
|
||||||
b.request.sendPresence(task.id, wants)
|
b.request.sendPresence(task.id, wants)
|
||||||
|
|
||||||
proc new*(
|
func new*(
|
||||||
T: type BitswapEngine,
|
T: type BlockExcEngine,
|
||||||
localStore: BlockStore,
|
localStore: BlockStore,
|
||||||
wallet: WalletRef,
|
wallet: WalletRef,
|
||||||
request: BitswapRequest = BitswapRequest(),
|
request: BlockExcRequest = BlockExcRequest(),
|
||||||
scheduleTask: TaskScheduler = nil,
|
scheduleTask: TaskScheduler = nil,
|
||||||
peersPerRequest = DefaultMaxPeersPerRequest): T =
|
peersPerRequest = DefaultMaxPeersPerRequest): T =
|
||||||
|
|
||||||
proc taskScheduler(task: BitswapPeerCtx): bool =
|
proc taskScheduler(task: BlockExcPeerCtx): bool =
|
||||||
if not isNil(scheduleTask):
|
if not isNil(scheduleTask):
|
||||||
return scheduleTask(task)
|
return scheduleTask(task)
|
||||||
|
|
||||||
let b = BitswapEngine(
|
let b = BlockExcEngine(
|
||||||
localStore: localStore,
|
localStore: localStore,
|
||||||
pendingBlocks: PendingBlocksManager.new(),
|
pendingBlocks: PendingBlocksManager.new(),
|
||||||
peersPerRequest: peersPerRequest,
|
peersPerRequest: peersPerRequest,
|
|
@ -18,7 +18,7 @@ func openLedgerChannel*(wallet: WalletRef,
|
||||||
asset: EthAddress): ?!ChannelId =
|
asset: EthAddress): ?!ChannelId =
|
||||||
wallet.openLedgerChannel(hub, ChainId, asset, AmountPerChannel)
|
wallet.openLedgerChannel(hub, ChainId, asset, AmountPerChannel)
|
||||||
|
|
||||||
func getOrOpenChannel(wallet: WalletRef, peer: BitswapPeerCtx): ?!ChannelId =
|
func getOrOpenChannel(wallet: WalletRef, peer: BlockExcPeerCtx): ?!ChannelId =
|
||||||
if channel =? peer.paymentChannel:
|
if channel =? peer.paymentChannel:
|
||||||
success channel
|
success channel
|
||||||
elif account =? peer.account:
|
elif account =? peer.account:
|
||||||
|
@ -29,7 +29,7 @@ func getOrOpenChannel(wallet: WalletRef, peer: BitswapPeerCtx): ?!ChannelId =
|
||||||
failure "no account set for peer"
|
failure "no account set for peer"
|
||||||
|
|
||||||
func pay*(wallet: WalletRef,
|
func pay*(wallet: WalletRef,
|
||||||
peer: BitswapPeerCtx,
|
peer: BlockExcPeerCtx,
|
||||||
amount: UInt256): ?!SignedState =
|
amount: UInt256): ?!SignedState =
|
||||||
if account =? peer.account:
|
if account =? peer.account:
|
||||||
let asset = Asset
|
let asset = Asset
|
|
@ -17,7 +17,7 @@ import pkg/questionable
|
||||||
import pkg/questionable/results
|
import pkg/questionable/results
|
||||||
|
|
||||||
import ../blocktype as bt
|
import ../blocktype as bt
|
||||||
import ./protobuf/bitswap as pb
|
import ./protobuf/blockexc
|
||||||
import ./protobuf/payments
|
import ./protobuf/payments
|
||||||
import ./networkpeer
|
import ./networkpeer
|
||||||
|
|
||||||
|
@ -25,9 +25,9 @@ export pb, networkpeer
|
||||||
export payments
|
export payments
|
||||||
|
|
||||||
logScope:
|
logScope:
|
||||||
topics = "dagger bitswap network"
|
topics = "dagger blockexc network"
|
||||||
|
|
||||||
const Codec* = "/ipfs/bitswap/1.2.0"
|
const Codec* = "/dagger/blockexc/1.0.0"
|
||||||
|
|
||||||
type
|
type
|
||||||
WantListHandler* = proc(peer: PeerID, wantList: WantList) {.gcsafe.}
|
WantListHandler* = proc(peer: PeerID, wantList: WantList) {.gcsafe.}
|
||||||
|
@ -36,7 +36,7 @@ type
|
||||||
AccountHandler* = proc(peer: PeerID, account: Account) {.gcsafe.}
|
AccountHandler* = proc(peer: PeerID, account: Account) {.gcsafe.}
|
||||||
PaymentHandler* = proc(peer: PeerID, payment: SignedState) {.gcsafe.}
|
PaymentHandler* = proc(peer: PeerID, payment: SignedState) {.gcsafe.}
|
||||||
|
|
||||||
BitswapHandlers* = object
|
BlockExcHandlers* = object
|
||||||
onWantList*: WantListHandler
|
onWantList*: WantListHandler
|
||||||
onBlocks*: BlocksHandler
|
onBlocks*: BlocksHandler
|
||||||
onPresence*: BlockPresenceHandler
|
onPresence*: BlockPresenceHandler
|
||||||
|
@ -57,22 +57,22 @@ type
|
||||||
AccountBroadcaster* = proc(peer: PeerID, account: Account) {.gcsafe.}
|
AccountBroadcaster* = proc(peer: PeerID, account: Account) {.gcsafe.}
|
||||||
PaymentBroadcaster* = proc(peer: PeerID, payment: SignedState) {.gcsafe.}
|
PaymentBroadcaster* = proc(peer: PeerID, payment: SignedState) {.gcsafe.}
|
||||||
|
|
||||||
BitswapRequest* = object
|
BlockExcRequest* = object
|
||||||
sendWantList*: WantListBroadcaster
|
sendWantList*: WantListBroadcaster
|
||||||
sendBlocks*: BlocksBroadcaster
|
sendBlocks*: BlocksBroadcaster
|
||||||
sendPresence*: PresenceBroadcaster
|
sendPresence*: PresenceBroadcaster
|
||||||
sendAccount*: AccountBroadcaster
|
sendAccount*: AccountBroadcaster
|
||||||
sendPayment*: PaymentBroadcaster
|
sendPayment*: PaymentBroadcaster
|
||||||
|
|
||||||
BitswapNetwork* = ref object of LPProtocol
|
BlockExcNetwork* = ref object of LPProtocol
|
||||||
peers*: Table[PeerID, NetworkPeer]
|
peers*: Table[PeerID, NetworkPeer]
|
||||||
switch*: Switch
|
switch*: Switch
|
||||||
handlers*: BitswapHandlers
|
handlers*: BlockExcHandlers
|
||||||
request*: BitswapRequest
|
request*: BlockExcRequest
|
||||||
getConn: ConnProvider
|
getConn: ConnProvider
|
||||||
|
|
||||||
proc handleWantList(
|
proc handleWantList(
|
||||||
b: BitswapNetwork,
|
b: BlockExcNetwork,
|
||||||
peer: NetworkPeer,
|
peer: NetworkPeer,
|
||||||
list: WantList) =
|
list: WantList) =
|
||||||
## Handle incoming want list
|
## Handle incoming want list
|
||||||
|
@ -104,7 +104,7 @@ proc makeWantList*(
|
||||||
WantList(entries: entries, full: full)
|
WantList(entries: entries, full: full)
|
||||||
|
|
||||||
proc broadcastWantList*(
|
proc broadcastWantList*(
|
||||||
b: BitswapNetwork,
|
b: BlockExcNetwork,
|
||||||
id: PeerID,
|
id: PeerID,
|
||||||
cids: seq[Cid],
|
cids: seq[Cid],
|
||||||
priority: int32 = 0,
|
priority: int32 = 0,
|
||||||
|
@ -130,7 +130,7 @@ proc broadcastWantList*(
|
||||||
asyncSpawn b.peers[id].send(Message(wantlist: wantList))
|
asyncSpawn b.peers[id].send(Message(wantlist: wantList))
|
||||||
|
|
||||||
proc handleBlocks(
|
proc handleBlocks(
|
||||||
b: BitswapNetwork,
|
b: BlockExcNetwork,
|
||||||
peer: NetworkPeer,
|
peer: NetworkPeer,
|
||||||
blocks: seq[auto]) =
|
blocks: seq[auto]) =
|
||||||
## Handle incoming blocks
|
## Handle incoming blocks
|
||||||
|
@ -159,7 +159,6 @@ template makeBlocks*(
|
||||||
seq[pb.Block] =
|
seq[pb.Block] =
|
||||||
var blks: seq[pb.Block]
|
var blks: seq[pb.Block]
|
||||||
for blk in blocks:
|
for blk in blocks:
|
||||||
# for now only send bitswap `1.1.0`
|
|
||||||
blks.add(pb.Block(
|
blks.add(pb.Block(
|
||||||
prefix: blk.cid.data.buffer,
|
prefix: blk.cid.data.buffer,
|
||||||
data: blk.data
|
data: blk.data
|
||||||
|
@ -168,7 +167,7 @@ template makeBlocks*(
|
||||||
blks
|
blks
|
||||||
|
|
||||||
proc broadcastBlocks*(
|
proc broadcastBlocks*(
|
||||||
b: BitswapNetwork,
|
b: BlockExcNetwork,
|
||||||
id: PeerID,
|
id: PeerID,
|
||||||
blocks: seq[bt.Block]) =
|
blocks: seq[bt.Block]) =
|
||||||
## Send blocks to remote
|
## Send blocks to remote
|
||||||
|
@ -181,7 +180,7 @@ proc broadcastBlocks*(
|
||||||
asyncSpawn b.peers[id].send(pb.Message(payload: makeBlocks(blocks)))
|
asyncSpawn b.peers[id].send(pb.Message(payload: makeBlocks(blocks)))
|
||||||
|
|
||||||
proc handleBlockPresence(
|
proc handleBlockPresence(
|
||||||
b: BitswapNetwork,
|
b: BlockExcNetwork,
|
||||||
peer: NetworkPeer,
|
peer: NetworkPeer,
|
||||||
presence: seq[BlockPresence]) =
|
presence: seq[BlockPresence]) =
|
||||||
## Handle block presence
|
## Handle block presence
|
||||||
|
@ -194,7 +193,7 @@ proc handleBlockPresence(
|
||||||
b.handlers.onPresence(peer.id, presence)
|
b.handlers.onPresence(peer.id, presence)
|
||||||
|
|
||||||
proc broadcastBlockPresence*(
|
proc broadcastBlockPresence*(
|
||||||
b: BitswapNetwork,
|
b: BlockExcNetwork,
|
||||||
id: PeerID,
|
id: PeerID,
|
||||||
presence: seq[BlockPresence]) =
|
presence: seq[BlockPresence]) =
|
||||||
## Send presence to remote
|
## Send presence to remote
|
||||||
|
@ -206,14 +205,14 @@ proc broadcastBlockPresence*(
|
||||||
trace "Sending presence to peer", peer = id
|
trace "Sending presence to peer", peer = id
|
||||||
asyncSpawn b.peers[id].send(Message(blockPresences: presence))
|
asyncSpawn b.peers[id].send(Message(blockPresences: presence))
|
||||||
|
|
||||||
proc handleAccount(network: BitswapNetwork,
|
proc handleAccount(network: BlockExcNetwork,
|
||||||
peer: NetworkPeer,
|
peer: NetworkPeer,
|
||||||
account: Account) =
|
account: Account) =
|
||||||
if network.handlers.onAccount.isNil:
|
if network.handlers.onAccount.isNil:
|
||||||
return
|
return
|
||||||
network.handlers.onAccount(peer.id, account)
|
network.handlers.onAccount(peer.id, account)
|
||||||
|
|
||||||
proc broadcastAccount*(network: BitswapNetwork,
|
proc broadcastAccount*(network: BlockExcNetwork,
|
||||||
id: PeerId,
|
id: PeerId,
|
||||||
account: Account) =
|
account: Account) =
|
||||||
if id notin network.peers:
|
if id notin network.peers:
|
||||||
|
@ -222,7 +221,7 @@ proc broadcastAccount*(network: BitswapNetwork,
|
||||||
let message = Message(account: AccountMessage.init(account))
|
let message = Message(account: AccountMessage.init(account))
|
||||||
asyncSpawn network.peers[id].send(message)
|
asyncSpawn network.peers[id].send(message)
|
||||||
|
|
||||||
proc broadcastPayment*(network: BitswapNetwork,
|
proc broadcastPayment*(network: BlockExcNetwork,
|
||||||
id: PeerId,
|
id: PeerId,
|
||||||
payment: SignedState) =
|
payment: SignedState) =
|
||||||
if id notin network.peers:
|
if id notin network.peers:
|
||||||
|
@ -231,14 +230,14 @@ proc broadcastPayment*(network: BitswapNetwork,
|
||||||
let message = Message(payment: StateChannelUpdate.init(payment))
|
let message = Message(payment: StateChannelUpdate.init(payment))
|
||||||
asyncSpawn network.peers[id].send(message)
|
asyncSpawn network.peers[id].send(message)
|
||||||
|
|
||||||
proc handlePayment(network: BitswapNetwork,
|
proc handlePayment(network: BlockExcNetwork,
|
||||||
peer: NetworkPeer,
|
peer: NetworkPeer,
|
||||||
payment: SignedState) =
|
payment: SignedState) =
|
||||||
if network.handlers.onPayment.isNil:
|
if network.handlers.onPayment.isNil:
|
||||||
return
|
return
|
||||||
network.handlers.onPayment(peer.id, payment)
|
network.handlers.onPayment(peer.id, payment)
|
||||||
|
|
||||||
proc rpcHandler(b: BitswapNetwork, peer: NetworkPeer, msg: Message) {.async.} =
|
proc rpcHandler(b: BlockExcNetwork, peer: NetworkPeer, msg: Message) {.async.} =
|
||||||
try:
|
try:
|
||||||
if msg.wantlist.entries.len > 0:
|
if msg.wantlist.entries.len > 0:
|
||||||
b.handleWantList(peer, msg.wantlist)
|
b.handleWantList(peer, msg.wantlist)
|
||||||
|
@ -259,10 +258,10 @@ proc rpcHandler(b: BitswapNetwork, peer: NetworkPeer, msg: Message) {.async.} =
|
||||||
b.handlePayment(peer, payment)
|
b.handlePayment(peer, payment)
|
||||||
|
|
||||||
except CatchableError as exc:
|
except CatchableError as exc:
|
||||||
trace "Exception in bitswap rpc handler", exc = exc.msg
|
trace "Exception in blockexc rpc handler", exc = exc.msg
|
||||||
|
|
||||||
proc getOrCreatePeer(b: BitswapNetwork, peer: PeerID): NetworkPeer =
|
proc getOrCreatePeer(b: BlockExcNetwork, peer: PeerID): NetworkPeer =
|
||||||
## Creates or retrieves a BitswapNetwork Peer
|
## Creates or retrieves a BlockExcNetwork Peer
|
||||||
##
|
##
|
||||||
|
|
||||||
if peer in b.peers:
|
if peer in b.peers:
|
||||||
|
@ -272,7 +271,7 @@ proc getOrCreatePeer(b: BitswapNetwork, peer: PeerID): NetworkPeer =
|
||||||
try:
|
try:
|
||||||
return await b.switch.dial(peer, Codec)
|
return await b.switch.dial(peer, Codec)
|
||||||
except CatchableError as exc:
|
except CatchableError as exc:
|
||||||
trace "unable to connect to bitswap peer", exc = exc.msg
|
trace "unable to connect to blockexc peer", exc = exc.msg
|
||||||
|
|
||||||
if not isNil(b.getConn):
|
if not isNil(b.getConn):
|
||||||
getConn = b.getConn
|
getConn = b.getConn
|
||||||
|
@ -281,27 +280,27 @@ proc getOrCreatePeer(b: BitswapNetwork, peer: PeerID): NetworkPeer =
|
||||||
b.rpcHandler(p, msg)
|
b.rpcHandler(p, msg)
|
||||||
|
|
||||||
# create new pubsub peer
|
# create new pubsub peer
|
||||||
let bitSwapPeer = NetworkPeer.new(peer, getConn, rpcHandler)
|
let blockExcPeer = NetworkPeer.new(peer, getConn, rpcHandler)
|
||||||
debug "created new bitswap peer", peer
|
debug "created new blockexc peer", peer
|
||||||
|
|
||||||
b.peers[peer] = bitSwapPeer
|
b.peers[peer] = blockExcPeer
|
||||||
|
|
||||||
return bitSwapPeer
|
return blockExcPeer
|
||||||
|
|
||||||
proc setupPeer*(b: BitswapNetwork, peer: PeerID) =
|
proc setupPeer*(b: BlockExcNetwork, peer: PeerID) =
|
||||||
## Perform initial setup, such as want
|
## Perform initial setup, such as want
|
||||||
## list exchange
|
## list exchange
|
||||||
##
|
##
|
||||||
|
|
||||||
discard b.getOrCreatePeer(peer)
|
discard b.getOrCreatePeer(peer)
|
||||||
|
|
||||||
proc dropPeer*(b: BitswapNetwork, peer: PeerID) =
|
proc dropPeer*(b: BlockExcNetwork, peer: PeerID) =
|
||||||
## Cleanup disconnected peer
|
## Cleanup disconnected peer
|
||||||
##
|
##
|
||||||
|
|
||||||
b.peers.del(peer)
|
b.peers.del(peer)
|
||||||
|
|
||||||
method init*(b: BitswapNetwork) =
|
method init*(b: BlockExcNetwork) =
|
||||||
## Perform protocol initialization
|
## Perform protocol initialization
|
||||||
##
|
##
|
||||||
|
|
||||||
|
@ -320,20 +319,20 @@ method init*(b: BitswapNetwork) =
|
||||||
|
|
||||||
proc handle(conn: Connection, proto: string) {.async, gcsafe, closure.} =
|
proc handle(conn: Connection, proto: string) {.async, gcsafe, closure.} =
|
||||||
let peerId = conn.peerInfo.peerId
|
let peerId = conn.peerInfo.peerId
|
||||||
let bitswapPeer = b.getOrCreatePeer(peerId)
|
let blockexcPeer = b.getOrCreatePeer(peerId)
|
||||||
await bitswapPeer.readLoop(conn) # attach read loop
|
await blockexcPeer.readLoop(conn) # attach read loop
|
||||||
|
|
||||||
b.handler = handle
|
b.handler = handle
|
||||||
b.codec = Codec
|
b.codec = Codec
|
||||||
|
|
||||||
proc new*(
|
func new*(
|
||||||
T: type BitswapNetwork,
|
T: type BlockExcNetwork,
|
||||||
switch: Switch,
|
switch: Switch,
|
||||||
connProvider: ConnProvider = nil): T =
|
connProvider: ConnProvider = nil): T =
|
||||||
## Create a new BitswapNetwork instance
|
## Create a new BlockExcNetwork instance
|
||||||
##
|
##
|
||||||
|
|
||||||
let b = BitswapNetwork(
|
let b = BlockExcNetwork(
|
||||||
switch: switch,
|
switch: switch,
|
||||||
getConn: connProvider)
|
getConn: connProvider)
|
||||||
|
|
||||||
|
@ -361,7 +360,7 @@ proc new*(
|
||||||
proc sendPayment(id: PeerID, payment: SignedState) =
|
proc sendPayment(id: PeerID, payment: SignedState) =
|
||||||
b.broadcastPayment(id, payment)
|
b.broadcastPayment(id, payment)
|
||||||
|
|
||||||
b.request = BitswapRequest(
|
b.request = BlockExcRequest(
|
||||||
sendWantList: sendWantList,
|
sendWantList: sendWantList,
|
||||||
sendBlocks: sendBlocks,
|
sendBlocks: sendBlocks,
|
||||||
sendPresence: sendPresence,
|
sendPresence: sendPresence,
|
|
@ -12,10 +12,10 @@ import pkg/chronicles
|
||||||
import pkg/protobuf_serialization
|
import pkg/protobuf_serialization
|
||||||
import pkg/libp2p
|
import pkg/libp2p
|
||||||
|
|
||||||
import ./protobuf/bitswap
|
import ./protobuf/blockexc
|
||||||
|
|
||||||
logScope:
|
logScope:
|
||||||
topics = "dagger bitswap networkpeer"
|
topics = "dagger blockexc networkpeer"
|
||||||
|
|
||||||
const MaxMessageSize = 8 * 1024 * 1024
|
const MaxMessageSize = 8 * 1024 * 1024
|
||||||
|
|
||||||
|
@ -43,7 +43,7 @@ proc readLoop*(b: NetworkPeer, conn: Connection) {.async.} =
|
||||||
trace "Got message for peer", peer = b.id, msg
|
trace "Got message for peer", peer = b.id, msg
|
||||||
await b.handler(b, msg)
|
await b.handler(b, msg)
|
||||||
except CatchableError as exc:
|
except CatchableError as exc:
|
||||||
trace "Exception in bitswap read loop", exc = exc.msg
|
trace "Exception in blockexc read loop", exc = exc.msg
|
||||||
finally:
|
finally:
|
||||||
await conn.close()
|
await conn.close()
|
||||||
|
|
||||||
|
@ -65,7 +65,7 @@ proc send*(b: NetworkPeer, msg: Message) {.async.} =
|
||||||
trace "Sending message to remote", peer = b.id, msg = $msg
|
trace "Sending message to remote", peer = b.id, msg = $msg
|
||||||
await conn.writeLp(Protobuf.encode(msg))
|
await conn.writeLp(Protobuf.encode(msg))
|
||||||
|
|
||||||
proc new*(
|
func new*(
|
||||||
T: type NetworkPeer,
|
T: type NetworkPeer,
|
||||||
peer: PeerId,
|
peer: PeerId,
|
||||||
connProvider: ConnProvider,
|
connProvider: ConnProvider,
|
|
@ -4,7 +4,7 @@ import pkg/libp2p
|
||||||
import pkg/chronos
|
import pkg/chronos
|
||||||
import pkg/nitro
|
import pkg/nitro
|
||||||
import pkg/questionable
|
import pkg/questionable
|
||||||
import ./protobuf/bitswap
|
import ./protobuf/blockexc
|
||||||
import ./protobuf/payments
|
import ./protobuf/payments
|
||||||
import ./protobuf/presence
|
import ./protobuf/presence
|
||||||
|
|
||||||
|
@ -12,7 +12,7 @@ export payments
|
||||||
export nitro
|
export nitro
|
||||||
|
|
||||||
type
|
type
|
||||||
BitswapPeerCtx* = ref object of RootObj
|
BlockExcPeerCtx* = ref object of RootObj
|
||||||
id*: PeerID
|
id*: PeerID
|
||||||
peerPrices*: Table[Cid, UInt256] # remote peer have list including price
|
peerPrices*: Table[Cid, UInt256] # remote peer have list including price
|
||||||
peerWants*: seq[Entry] # remote peers want lists
|
peerWants*: seq[Entry] # remote peers want lists
|
||||||
|
@ -21,16 +21,16 @@ type
|
||||||
account*: ?Account # ethereum account of this peer
|
account*: ?Account # ethereum account of this peer
|
||||||
paymentChannel*: ?ChannelId # payment channel id
|
paymentChannel*: ?ChannelId # payment channel id
|
||||||
|
|
||||||
proc peerHave*(context: BitswapPeerCtx): seq[Cid] =
|
proc peerHave*(context: BlockExcPeerCtx): seq[Cid] =
|
||||||
toSeq(context.peerPrices.keys)
|
toSeq(context.peerPrices.keys)
|
||||||
|
|
||||||
proc contains*(a: openArray[BitswapPeerCtx], b: PeerID): bool =
|
proc contains*(a: openArray[BlockExcPeerCtx], b: PeerID): bool =
|
||||||
## Convenience method to check for peer prepense
|
## Convenience method to check for peer prepense
|
||||||
##
|
##
|
||||||
|
|
||||||
a.anyIt( it.id == b )
|
a.anyIt( it.id == b )
|
||||||
|
|
||||||
func updatePresence*(context: BitswapPeerCtx, presence: Presence) =
|
func updatePresence*(context: BlockExcPeerCtx, presence: Presence) =
|
||||||
let cid = presence.cid
|
let cid = presence.cid
|
||||||
let price = presence.price
|
let price = presence.price
|
||||||
|
|
||||||
|
@ -39,7 +39,7 @@ func updatePresence*(context: BitswapPeerCtx, presence: Presence) =
|
||||||
elif cid in context.peerHave and not presence.have:
|
elif cid in context.peerHave and not presence.have:
|
||||||
context.peerPrices.del(cid)
|
context.peerPrices.del(cid)
|
||||||
|
|
||||||
func price*(context: BitswapPeerCtx, cids: seq[Cid]): UInt256 =
|
func price*(context: BlockExcPeerCtx, cids: seq[Cid]): UInt256 =
|
||||||
for cid in cids:
|
for cid in cids:
|
||||||
if price =? context.peerPrices.?[cid]:
|
if price =? context.peerPrices.?[cid]:
|
||||||
result += price
|
result += price
|
|
@ -16,7 +16,7 @@ import pkg/libp2p
|
||||||
import ../blocktype
|
import ../blocktype
|
||||||
|
|
||||||
logScope:
|
logScope:
|
||||||
topics = "dagger bitswap pendingblocks"
|
topics = "dagger blockexc pendingblocks"
|
||||||
|
|
||||||
type
|
type
|
||||||
PendingBlocksManager* = ref object of RootObj
|
PendingBlocksManager* = ref object of RootObj
|
||||||
|
@ -67,7 +67,7 @@ proc contains*(
|
||||||
p: PendingBlocksManager,
|
p: PendingBlocksManager,
|
||||||
cid: Cid): bool = p.pending(cid)
|
cid: Cid): bool = p.pending(cid)
|
||||||
|
|
||||||
proc new*(T: type PendingBlocksManager): T =
|
func new*(T: type PendingBlocksManager): T =
|
||||||
T(
|
T(
|
||||||
blocks: initTable[Cid, Future[Block]]()
|
blocks: initTable[Cid, Future[Block]]()
|
||||||
)
|
)
|
|
@ -1,6 +1,6 @@
|
||||||
syntax = "proto3";
|
syntax = "proto3";
|
||||||
|
|
||||||
package bitswap.message.pb;
|
package blockexc.message.pb;
|
||||||
|
|
||||||
message Message {
|
message Message {
|
||||||
|
|
||||||
|
@ -11,7 +11,7 @@ message Message {
|
||||||
}
|
}
|
||||||
|
|
||||||
message Entry {
|
message Entry {
|
||||||
bytes block = 1; // the block cid (cidV0 in bitswap 1.0.0, cidV1 in bitswap 1.1.0)
|
bytes block = 1; // the block cid
|
||||||
int32 priority = 2; // the priority (normalized). default to 1
|
int32 priority = 2; // the priority (normalized). default to 1
|
||||||
bool cancel = 3; // whether this revokes an entry
|
bool cancel = 3; // whether this revokes an entry
|
||||||
WantType wantType = 4; // Note: defaults to enum 0, ie Block
|
WantType wantType = 4; // Note: defaults to enum 0, ie Block
|
||||||
|
@ -47,10 +47,9 @@ message Message {
|
||||||
}
|
}
|
||||||
|
|
||||||
Wantlist wantlist = 1;
|
Wantlist wantlist = 1;
|
||||||
repeated bytes blocks = 2; // used to send Blocks in bitswap 1.0.0
|
repeated Block payload = 2;
|
||||||
repeated Block payload = 3; // used to send Blocks in bitswap 1.1.0
|
repeated BlockPresence blockPresences = 3;
|
||||||
repeated BlockPresence blockPresences = 4;
|
int32 pendingBytes = 4;
|
||||||
int32 pendingBytes = 5;
|
AccountMessage account = 5;
|
||||||
AccountMessage account = 6;
|
StateChannelUpdate payment = 6;
|
||||||
StateChannelUpdate payment = 7;
|
|
||||||
}
|
}
|
|
@ -4,7 +4,7 @@ import pkg/stint
|
||||||
import pkg/nitro
|
import pkg/nitro
|
||||||
import pkg/questionable
|
import pkg/questionable
|
||||||
import pkg/upraises
|
import pkg/upraises
|
||||||
import ./bitswap
|
import ./blockexc
|
||||||
|
|
||||||
export AccountMessage
|
export AccountMessage
|
||||||
export StateChannelUpdate
|
export StateChannelUpdate
|
|
@ -3,7 +3,7 @@ import pkg/stint
|
||||||
import pkg/questionable
|
import pkg/questionable
|
||||||
import pkg/questionable/results
|
import pkg/questionable/results
|
||||||
import pkg/upraises
|
import pkg/upraises
|
||||||
import ./bitswap
|
import ./blockexc
|
||||||
|
|
||||||
export questionable
|
export questionable
|
||||||
export stint
|
export stint
|
||||||
|
@ -12,7 +12,7 @@ export BlockPresenceType
|
||||||
upraises.push: {.upraises: [].}
|
upraises.push: {.upraises: [].}
|
||||||
|
|
||||||
type
|
type
|
||||||
PresenceMessage* = bitswap.BlockPresence
|
PresenceMessage* = blockexc.BlockPresence
|
||||||
Presence* = object
|
Presence* = object
|
||||||
cid*: Cid
|
cid*: Cid
|
||||||
have*: bool
|
have*: bool
|
2
nim.cfg
2
nim.cfg
|
@ -1 +1 @@
|
||||||
-d:"chronicles_enabled=off" # disable logging by default
|
-d:"chronicles_enabled=on" # disable logging by default
|
||||||
|
|
|
@ -1,20 +0,0 @@
|
||||||
import std/sequtils
|
|
||||||
|
|
||||||
import pkg/chronos
|
|
||||||
import pkg/asynctest
|
|
||||||
import pkg/stew/results
|
|
||||||
import pkg/dagger/chunker
|
|
||||||
import pkg/dagger/merkletree
|
|
||||||
import pkg/stew/byteutils
|
|
||||||
import pkg/dagger/p2p/rng
|
|
||||||
import pkg/dagger/blocktype as bt
|
|
||||||
|
|
||||||
suite "Data set":
|
|
||||||
|
|
||||||
test "Make from Blocks":
|
|
||||||
let
|
|
||||||
chunker = newRandomChunker(Rng.instance(), size = 256*3, chunkSize = 256)
|
|
||||||
blocks = chunker.mapIt( bt.Block.new(it) )
|
|
||||||
|
|
||||||
let merkle = MerkleTreeRef.fromBlocks(blocks)
|
|
||||||
|
|
Loading…
Reference in New Issue