Spec-compliant implementation of Eth1 monitoring; Eth1-enabled local sim

BEWARE! This commit will trigger a stack overflow during local sim
This commit is contained in:
Zahary Karadjov 2020-03-24 13:13:07 +02:00 committed by zah
parent 39a893ea90
commit 740b76d152
21 changed files with 791 additions and 226 deletions

5
.gitmodules vendored
View File

@ -148,3 +148,8 @@
url = https://github.com/status-im/nim-protobuf-serialization.git
ignore = dirty
branch = master
[submodule "vendor/nim-rocksdb"]
path = vendor/nim-rocksdb
url = https://github.com/status-im/nim-rocksdb.git
ignore = dirty
branch = master

View File

@ -31,6 +31,7 @@ requires "nim >= 0.19.0",
"nimcrypto",
"serialization",
"stew",
"testutils",
"prompt",
"web3",
"yaml"
@ -45,6 +46,12 @@ proc buildBinary(name: string, srcDir = "./", params = "", cmdParams = "", lang
extra_params &= " " & paramStr(i)
exec "nim " & lang & " --out:./build/" & name & " -r " & extra_params & " " & srcDir & name & ".nim" & " " & cmdParams
task moduleTests, "Run all module tests":
buildBinary "beacon_node", "beacon_chain/",
"-d:chronicles_log_level=TRACE " &
"-d:const_preset=minimal " &
"-d:testutils_test_build"
### tasks
task test, "Run all tests":
# We're enabling the TRACE log level so we're sure that those rarely used

View File

@ -168,9 +168,11 @@ proc init*(T: type BeaconNode, conf: BeaconNodeConf): Future[BeaconNode] {.async
# Didn't work, try creating a genesis state using main chain monitor
# TODO Could move this to a separate "GenesisMonitor" process or task
# that would do only this - see
if conf.depositWeb3Url.len != 0:
if conf.web3Url.len > 0 and conf.depositContractAddress.len > 0:
mainchainMonitor = MainchainMonitor.init(
conf.depositWeb3Url, conf.depositContractAddress, Eth2Digest())
web3Provider(conf.web3Url),
conf.depositContractAddress,
Eth2Digest())
mainchainMonitor.start()
else:
error "No initial state, need genesis state or deposit contract address"
@ -198,9 +200,12 @@ proc init*(T: type BeaconNode, conf: BeaconNodeConf): Future[BeaconNode] {.async
let
blockPool = BlockPool.init(db)
if mainchainMonitor.isNil and conf.depositWeb3Url.len != 0:
if mainchainMonitor.isNil and
conf.web3Url.len > 0 and
conf.depositContractAddress.len > 0:
mainchainMonitor = MainchainMonitor.init(
conf.depositWeb3Url, conf.depositContractAddress,
web3Provider(conf.web3Url),
conf.depositContractAddress,
blockPool.headState.data.data.eth1_data.block_hash)
# TODO if we don't have any validators attached, we don't need a mainchain
# monitor
@ -399,12 +404,10 @@ proc proposeBlock(node: BeaconNode,
node.blockPool.tmpState, head.atSlot(slot)):
let (eth1data, deposits) =
if node.mainchainMonitor.isNil:
(get_eth1data_stub(
state.eth1_deposit_index, slot.compute_epoch_at_slot()),
newSeq[Deposit]())
(get_eth1data_stub(state.eth1_deposit_index, slot.compute_epoch_at_slot()),
newSeq[Deposit]())
else:
(node.mainchainMonitor.eth1Data,
node.mainchainMonitor.getPendingDeposits())
node.mainchainMonitor.getBlockProposalData(state)
let message = makeBeaconBlock(
state,
@ -1301,8 +1304,7 @@ when hasPrompt:
render statusBar
# p.showPrompt
except Exception as e: # render raises Exception
if e is Defect: raise (ref Defect)(e)
discard # Status bar not critical
logLoggingFailure(cstring(msg), e)
proc statusBarUpdatesPollingLoop() {.async.} =
while true:
@ -1314,7 +1316,7 @@ when hasPrompt:
# var t: Thread[ptr Prompt]
# createThread(t, processPromptCommands, addr p)
when isMainModule:
programMain:
let
banner = clientId & "\p" & copyrights & "\p\p" & nimBanner
config = BeaconNodeConf.load(version = banner, copyrightBanner = banner)
@ -1324,8 +1326,8 @@ when isMainModule:
proc (logLevel: LogLevel, msg: LogOutputStr) {.gcsafe, raises: [Defect].} =
try:
stdout.write(msg)
except IOError:
discard # nothing to do..
except IOError as err:
logLoggingFailure(cstring(msg), err)
randomize()
@ -1358,8 +1360,8 @@ when isMainModule:
let
startTime = uint64(times.toUnix(times.getTime()) + config.genesisOffset)
outGenesis = config.outputGenesis.string
eth1Hash = if config.depositWeb3Url.len == 0: eth1BlockHash
else: waitFor getLatestEth1BlockHash(config.depositWeb3Url)
eth1Hash = if config.web3Url.len == 0: eth1BlockHash
else: waitFor getLatestEth1BlockHash(config.web3Url)
var
initialState = initialize_beacon_state_from_eth1(
eth1Hash, startTime, deposits, {skipBlsValidation, skipMerkleValidation})
@ -1446,16 +1448,26 @@ when isMainModule:
config.totalRandomDeposits, config.depositsDir, true,
firstIdx = config.totalQuickstartDeposits)
if config.depositWeb3Url.len > 0 and config.depositContractAddress.len > 0:
if config.web3Url.len > 0 and config.depositContractAddress.len > 0:
if config.minDelay > config.maxDelay:
echo "The minimum delay should not be larger than the maximum delay"
quit 1
var delayGenerator: DelayGenerator
if config.maxDelay > 0.0:
delayGenerator = proc (): chronos.Duration {.gcsafe.} =
chronos.milliseconds (rand(config.minDelay..config.maxDelay)*1000).int
info "Sending deposits",
web3 = config.depositWeb3Url,
web3 = config.web3Url,
depositContract = config.depositContractAddress
waitFor sendDeposits(
quickstartDeposits & randomDeposits,
config.depositWeb3Url,
config.web3Url,
config.depositContractAddress,
config.depositPrivateKey)
config.depositPrivateKey,
delayGenerator)
of query:
case config.queryCmd

View File

@ -52,7 +52,7 @@ type
abbr: "d"
name: "data-dir" }: OutDir
depositWeb3Url* {.
web3Url* {.
defaultValue: ""
desc: "URL of the Web3 server to observe Eth1."
name: "web3-url" }: string
@ -252,6 +252,16 @@ type
desc: "Private key of the controlling (sending) account",
name: "deposit-private-key" }: string
minDelay* {.
defaultValue: 0.0
desc: "Minimum possible delay between making two deposits (in seconds)"
name: "min-delay" }: float
maxDelay* {.
defaultValue: 0.0
desc: "Maximum possible delay between making two deposits (in seconds)"
name: "max-delay" }: float
of query:
case queryCmd* {.
defaultValue: nimQuery

View File

@ -14,7 +14,7 @@ type
sendEth
CliConfig = object
depositWeb3Url* {.
web3Url* {.
desc: "URL of the Web3 server to observe Eth1"
name: "web3-url" }: string
@ -65,7 +65,7 @@ proc sendEth(web3: Web3, to: string, valueEth: int): Future[TxHash] =
proc main() {.async.} =
let cfg = CliConfig.load()
let web3 = await newWeb3(cfg.depositWeb3Url)
let web3 = await newWeb3(cfg.web3Url)
if cfg.privateKey.len != 0:
web3.privateKey = PrivateKey.fromHex(cfg.privateKey)[]
else:

View File

@ -9,7 +9,7 @@ import
json_serialization, json_serialization/std/[net, options],
chronos, chronicles, metrics,
# TODO: create simpler to use libp2p modules that use re-exports
libp2p/[switch, standard_setup, peerinfo, peer, connection,
libp2p/[switch, standard_setup, peerinfo, peer, connection, errors,
multiaddress, multicodec, crypto/crypto, crypto/secp,
protocols/identify, protocols/protocol],
libp2p/protocols/secure/[secure, secio],
@ -39,6 +39,10 @@ type
Bytes = seq[byte]
# TODO: This is here only to eradicate a compiler
# warning about unused import (rpc/messages).
GossipMsg = messages.Message
# TODO Is this really needed?
Eth2Node* = ref object of RootObj
switch*: Switch
@ -47,6 +51,7 @@ type
peerPool*: PeerPool[Peer, PeerID]
protocolStates*: seq[RootRef]
libp2pTransportLoops*: seq[Future[void]]
discoveryLoop: Future[void]
metadata*: Eth2Metadata
EthereumNode = Eth2Node # needed for the definitions in p2p_backends_helpers
@ -511,7 +516,7 @@ proc performProtocolHandshakes*(peer: Peer) {.async.} =
if protocol.handshake != nil:
subProtocolsHandshakes.add((protocol.handshake)(peer, nil))
await all(subProtocolsHandshakes)
await allFuturesThrowing(subProtocolsHandshakes)
template initializeConnection*(peer: Peer): auto =
performProtocolHandshakes(peer)
@ -755,7 +760,8 @@ proc start*(node: Eth2Node) {.async.} =
node.discovery.open()
node.discovery.start()
node.libp2pTransportLoops = await node.switch.start()
traceAsyncErrors node.runDiscoveryLoop()
node.discoveryLoop = node.runDiscoveryLoop()
traceAsyncErrors node.discoveryLoop
proc init*(T: type Peer, network: Eth2Node, info: PeerInfo): Peer =
new result
@ -1032,9 +1038,11 @@ proc subscribe*[MsgType](node: Eth2Node,
msgValidator SSZ.decode(gossipBytes, MsgType)
# Validate messages as soon as subscribed
let incomingMsgValidator = proc(topic: string, message: messages.Message):
Future[bool] {.async, gcsafe.} =
let incomingMsgValidator = proc(topic: string,
message: GossipMsg): Future[bool]
{.async, gcsafe.} =
return execMsgValidator(message.data, topic)
node.switch.addValidator(topic, incomingMsgValidator)
let incomingMsgHandler = proc(topic: string,

View File

@ -1,105 +1,430 @@
import
deques, tables, hashes, options,
chronos, web3, json, chronicles,
spec/[datatypes, digest, crypto, beaconstate, helpers]
contract(DepositContract):
proc deposit(pubkey: Bytes48,
withdrawalCredentials: Bytes32,
signature: Bytes96,
deposit_data_root: FixedBytes[32])
proc get_deposit_root(): FixedBytes[32]
proc get_deposit_count(): Bytes8
proc DepositEvent(pubkey: Bytes48,
withdrawalCredentials: Bytes32,
amount: Bytes8,
signature: Bytes96,
index: Bytes8) {.event.}
# TODO
# The raises list of this module are still not usable due to general
# Exceptions being reported from Chronos's asyncfutures2.
type
Eth1BlockNumber* = uint64
Eth1BlockTimestamp* = uint64
Eth1Block* = ref object
number*: Eth1BlockNumber
timestamp*: Eth1BlockTimestamp
deposits*: seq[Deposit]
voteData*: Eth1Data
Eth1Chain* = object
blocks: Deque[Eth1Block]
blocksByHash: Table[BlockHash, Eth1Block]
MainchainMonitor* = ref object
web3Url: string
startBlock: BlockHash
depositContractAddress: Address
dataProviderFactory*: DataProviderFactory
genesisState: ref BeaconState
genesisStateFut: Future[void]
pendingDeposits: seq[Deposit]
depositCount: uint64
curBlock: uint64
depositQueue: AsyncQueue[QueueElement]
eth1Block: BlockHash
eth1Data*: Eth1Data
eth1Chain: Eth1Chain
depositQueue: AsyncQueue[DepositQueueElem]
runFut: Future[void]
QueueElement = (BlockHash, DepositData)
Web3EventType = enum
NewEvent
RemovedEvent
proc init*(
T: type MainchainMonitor,
web3Url, depositContractAddress: string,
startBlock: Eth2Digest): T =
T(
web3Url: web3Url,
depositContractAddress: Address.fromHex(depositContractAddress),
depositQueue: newAsyncQueue[QueueElement](),
eth1Block: BlockHash(startBlock.data),
)
DepositQueueElem = (BlockHash, Web3EventType)
contract(DepositContract):
proc deposit(pubkey: Bytes48, withdrawalCredentials: Bytes32, signature: Bytes96, deposit_data_root: FixedBytes[32])
proc get_deposit_root(): FixedBytes[32]
proc get_deposit_count(): Bytes8
proc DepositEvent(pubkey: Bytes48, withdrawalCredentials: Bytes32, amount: Bytes8, signature: Bytes96, index: Bytes8) {.event.}
DataProvider* = object of RootObj
DataProviderRef* = ref DataProvider
DataProviderFactory* = object
desc: string
new: proc(depositContractAddress: Address): Future[DataProviderRef] {.
gcsafe
# raises: [Defect]
.}
Web3DataProvider* = object of DataProvider
url: string
web3: Web3
ns: Sender[DepositContract]
subscription: Subscription
Web3DataProviderRef* = ref Web3DataProvider
ReorgDepthLimitExceeded = object of CatchableError
CorruptDataProvider = object of CatchableError
DisconnectHandler* = proc () {.gcsafe, raises: [Defect].}
DepositEventHandler* = proc (
pubkey: Bytes48,
withdrawalCredentials: Bytes32,
amount: Bytes8,
signature: Bytes96, merkleTreeIndex: Bytes8, j: JsonNode) {.gcsafe.}
const
reorgDepthLimit = 1000
# https://github.com/ethereum/eth2.0-specs/blob/v0.11.1/specs/phase0/validator.md#get_eth1_data
func compute_time_at_slot(state: BeaconState, slot: Slot): uint64 =
return state.genesis_time + slot * SECONDS_PER_SLOT
# https://github.com/ethereum/eth2.0-specs/blob/v0.11.1/specs/phase0/validator.md#get_eth1_data
func voting_period_start_time*(state: BeaconState): uint64 =
let eth1_voting_period_start_slot = state.slot - state.slot mod SLOTS_PER_ETH1_VOTING_PERIOD.uint64
return compute_time_at_slot(state, eth1_voting_period_start_slot)
# https://github.com/ethereum/eth2.0-specs/blob/v0.11.1/specs/phase0/validator.md#get_eth1_data
func is_candidate_block(blk: Eth1Block, period_start: uint64): bool =
(blk.timestamp <= period_start - SECONDS_PER_ETH1_BLOCK * ETH1_FOLLOW_DISTANCE) and
(blk.timestamp >= period_start - SECONDS_PER_ETH1_BLOCK * ETH1_FOLLOW_DISTANCE * 2)
func asEth2Digest(x: BlockHash): Eth2Digest =
Eth2Digest(data: array[32, byte](x))
template asBlockHash(x: Eth2Digest): BlockHash =
BlockHash(x.data)
func getDepositsInRange(eth1Chain: Eth1Chain,
sinceBlock, latestBlock: Eth1BlockNumber): seq[Deposit] =
## Returns all deposits that happened AFTER the block `sinceBlock` (not inclusive).
## The deposits in `latestBlock` will be included.
if latestBlock <= sinceBlock: return
let firstBlockInCache = eth1Chain.blocks[0].number
# This function should be used with indices obtained with `eth1Chain.findBlock`.
# This guarantess that both of these indices will be valid:
doAssert sinceBlock >= firstBlockInCache and
int(latestBlock - firstBlockInCache) < eth1Chain.blocks.len
let
sinceBlockIdx = sinceBlock - firstBlockInCache
latestBlockIdx = latestBlock - firstBlockInCache
for i in (sinceBlockIdx + 1) ..< latestBlockIdx:
result.add eth1Chain.blocks[i].deposits
template findBlock*(eth1Chain: Eth1Chain, hash: BlockHash): Eth1Block =
eth1Chain.blocksByHash.getOrDefault(hash, nil)
template findBlock*(eth1Chain: Eth1Chain, eth1Data: Eth1Data): Eth1Block =
getOrDefault(eth1Chain.blocksByHash, asBlockHash(eth1Data.block_hash), nil)
proc findParent*(eth1Chain: Eth1Chain, blk: BlockObject): Eth1Block =
result = eth1Chain.findBlock(blk.parentHash)
# a distinct type is stipped here:
let blockNumber = Eth1BlockNumber(blk.number)
if result != nil and result.number != blockNumber - 1:
debug "Found inconsistent numbering of Eth1 blocks. Ignoring block.",
blockHash = blk.hash.toHex, blockNumber,
parentHash = blk.parentHash.toHex, parentNumber = result.number
result = nil
when false:
func getCacheIdx(eth1Chain: Eth1Chain, blockNumber: Eth1BlockNumber): int =
if eth1Chain.blocks.len == 0:
return -1
let idx = blockNumber - eth1Chain.blocks[0].number
if idx < 0 or idx >= eth1Chain.blocks.len:
return -1
idx
func `{}`*(eth1Chain: Eth1Chain, blockNumber: Eth1BlockNumber): Eth1Block =
## Finds a block in our cache that corresponds to a particular Eth block
## number. May return `nil` if we don't have such a block in the cache.
let idx = eth1Chain.getCacheIdx(blockNumber)
if idx != -1: eth1Chain.blocks[idx] else: nil
func latestCandidateBlock(eth1Chain: Eth1Chain, periodStart: uint64): Eth1Block =
for i in countdown(eth1Chain.blocks.len - 1, 0):
let blk = eth1Chain.blocks[i]
if is_candidate_block(blk, periodStart):
return blk
func trimHeight(eth1Chain: var Eth1Chain, blockNumber: Eth1BlockNumber) =
## Removes all blocks above certain `blockNumber`
if eth1Chain.blocks.len == 0:
return
let newLen = max(0, int(blockNumber - eth1Chain.blocks[0].number + 1))
for i in newLen ..< eth1Chain.blocks.len:
let removed = eth1Chain.blocks.popLast
eth1Chain.blocksByHash.del removed.voteData.block_hash.asBlockHash
template purgeChain*(eth1Chain: var Eth1Chain, blk: Eth1Block) =
## This is used when we discover that a previously considered block
## is no longer part of the selected chain (due to a reorg). We can
## then remove from our chain together with all blocks that follow it.
trimHeight(eth1Chain, blk.number - 1)
func purgeChain*(eth1Chain: var Eth1Chain, blockHash: BlockHash) =
let blk = eth1Chain.findBlock(blockHash)
if blk != nil: eth1Chain.purgeChain(blk)
template purgeDescendants*(eth1CHain: Eth1Chain, blk: Eth1Block) =
trimHeight(eth1Chain, blk.number)
func addBlock*(eth1Chain: var Eth1Chain, newBlock: Eth1Block) =
if eth1Chain.blocks.len > 0:
doAssert eth1Chain.blocks.peekLast.number + 1 == newBlock.number
eth1Chain.blocks.addLast newBlock
eth1Chain.blocksByHash[newBlock.voteData.block_hash.asBlockHash] = newBlock
func totalDeposits*(eth1Chain: Eth1Chain): int =
for blk in eth1Chain.blocks:
result += blk.deposits.len
func allDeposits*(eth1Chain: Eth1Chain): seq[Deposit] =
for blk in eth1Chain.blocks:
result.add blk.deposits
template hash*(x: Eth1Block): Hash =
hash(x.voteData.block_hash.data)
template notImplemented =
doAssert false, "Method not implemented"
method getBlockByHash*(p: DataProviderRef, hash: BlockHash): Future[BlockObject] {.
base
gcsafe
locks: 0
# raises: [Defect]
.} =
discard
# notImplemented
method onDisconnect*(p: DataProviderRef, handler: DisconnectHandler) {.
base
gcsafe
locks: 0
# raises: []
.} =
notImplemented
method onDepositEvent*(p: DataProviderRef,
startBlock: Eth1BlockNumber,
handler: DepositEventHandler): Future[void] {.
base
gcsafe
locks: 0
# raises: []
.} =
notImplemented
method close*(p: DataProviderRef): Future[void] {.
base
gcsafe
locks: 0
# raises: [Defect]
.} =
notImplemented
method fetchDepositData*(p: DataProviderRef,
web3Block: BlockObject): Future[Eth1Block] {.
base
gcsafe
locks: 0
# raises: [Defect, CatchableError]
.} =
notImplemented
# https://github.com/ethereum/eth2.0-specs/blob/v0.11.1/specs/phase0/validator.md#get_eth1_data
func getBlockProposalData*(eth1Chain: Eth1Chain,
state: BeaconState): (Eth1Data, seq[Deposit]) =
template voteForNoChange() =
return (state.eth1_data, newSeq[Deposit]())
let prevBlock = eth1Chain.findBlock(state.eth1_data)
if prevBlock == nil:
# The Eth1 block currently referenced in the BeaconState is unknown to us.
# This situation is not specifically covered in the honest validator spec,
# but there is a similar condition where none of the eth1_data_votes is
# present in our worldview. The suggestion there is to vote for "no change"
# and we'll do the same here:
voteForNoChange()
let periodStart = voting_period_start_time(state)
var otherVotesCountTable = initCountTable[Eth1Block]()
for vote in state.eth1_data_votes:
let eth1Block = eth1Chain.findBlock(vote)
if eth1Block != nil and is_candidate_block(eth1Block, periodStart):
otherVotesCountTable.inc eth1Block
var ourVote: Eth1Block
if otherVotesCountTable.len > 0:
ourVote = otherVotesCountTable.largest.key
else:
ourVote = eth1Chain.latestCandidateBlock(periodStart)
if ourVote == nil:
voteForNoChange()
(ourVote.voteData, eth1Chain.getDepositsInRange(prevBlock.number, ourVote.number))
template getBlockProposalData*(m: MainchainMonitor, state: BeaconState): untyped =
getBlockProposalData(m.eth1Chain, state)
proc init*(T: type MainchainMonitor,
dataProviderFactory: DataProviderFactory,
depositContractAddress: string,
startBlock: Eth2Digest): T =
T(depositContractAddress: Address.fromHex(depositContractAddress),
depositQueue: newAsyncQueue[DepositQueueElem](),
startBlock: BlockHash(startBlock.data),
dataProviderFactory: dataProviderFactory)
const MIN_GENESIS_TIME = 0
proc updateEth1Data(m: MainchainMonitor, count: uint64, root: FixedBytes[32]) =
m.eth1Data.deposit_count = count
m.eth1Data.deposit_root.data = array[32, byte](root)
m.eth1Data.block_hash.data = array[32, byte](m.eth1Block)
proc readJsonDeposits(json: JsonNode): seq[Deposit] =
if json.kind != JArray:
raise newException(CatchableError,
"Web3 provider didn't return a list of deposit events")
proc processDeposits(m: MainchainMonitor, web3: Web3) {.async.} =
for logEvent in json:
var logData = strip0xPrefix(json["data"].getStr)
var
pubkey: Bytes48
withdrawalCredentials: Bytes32
amount: Bytes8
signature: Bytes96
index: Bytes8
var offset = 0
offset = decode(logData, offset, pubkey)
offset = decode(logData, offset, withdrawalCredentials)
offset = decode(logData, offset, amount)
offset = decode(logData, offset, signature)
offset = decode(logData, offset, index)
result.add Deposit(
# proof: TODO
data: DepositData(
pubkey: ValidatorPubKey.init(array[48, byte](pubkey)),
withdrawal_credentials: Eth2Digest(data: array[32, byte](withdrawalCredentials)),
amount: bytes_to_int(array[8, byte](amount)),
signature: ValidatorSig.init(array[96, byte](signature))))
proc checkForGenesisEvent(m: MainchainMonitor) =
if not m.genesisState.isNil:
return
let lastBlock = m.eth1Chain.blocks.peekLast
const totalDepositsNeeded = max(SLOTS_PER_EPOCH,
MIN_GENESIS_ACTIVE_VALIDATOR_COUNT)
if lastBlock.timestamp.uint64 >= MIN_GENESIS_TIME.uint64 and
m.eth1Chain.totalDeposits >= totalDepositsNeeded:
# This block is a genesis candidate
let startTime = lastBlock.timestamp.uint64
var s = initialize_beacon_state_from_eth1(lastBlock.voteData.block_hash,
startTime, m.eth1Chain.allDeposits, {})
if is_valid_genesis_state(s):
# https://github.com/ethereum/eth2.0-pm/tree/6e41fcf383ebeb5125938850d8e9b4e9888389b4/interop/mocked_start#create-genesis-state
s.genesis_time = startTime
m.genesisState.new()
m.genesisState[] = s
if not m.genesisStateFut.isNil:
m.genesisStateFut.complete()
m.genesisStateFut = nil
proc processDeposits(m: MainchainMonitor, dataProvider: DataProviderRef) {.
async
# raises: [Defect]
.} =
# ATTENTION!
# Please note that this code is using a queue to guarantee the
# strict serial order of processing of deposits. If we had the
# same code embedded in the deposit contracts events handler,
# it could easily re-order the steps due to the intruptable
# interleaved execution of async code.
while true:
let (blkHash, data) = await m.depositQueue.popFirst()
var blk: BlockObject
var depositCount: uint64
var depositRoot: FixedBytes[32]
try:
blk = await web3.provider.eth_getBlockByHash(blkHash, false)
let (blockHash, eventType) = await m.depositQueue.popFirst()
let ns = web3.contractSender(DepositContract, m.depositContractAddress)
if eventType == RemovedEvent:
m.eth1Chain.purgeChain(blockHash)
continue
# TODO: use m.eth1Block for web3 calls
let cnt = await ns.get_deposit_count().call()
depositRoot = await ns.get_deposit_root().call()
depositCount = bytes_to_int(array[8, byte](cnt))
let cachedBlock = m.eth1Chain.findBlock(blockHash)
if cachedBlock == nil:
try:
let
web3Block = await dataProvider.getBlockByHash(blockHash)
eth1Block = await dataProvider.fetchDepositData(web3Block)
except:
# Connection problem? Put the unprocessed deposit back to queue
m.depositQueue.addFirstNoWait((blkHash, data))
raise
if m.eth1Chain.blocks.len > 0:
var cachedParent = m.eth1Chain.findParent(web3Block)
if cachedParent == nil:
# We are missing the parent block.
# This shouldn't be happening if the deposits events are reported in
# proper order, but nevertheless let's try to repair our chain:
var chainOfParents = newSeq[Eth1Block]()
var parentHash = web3Block.parentHash
var expectedParentBlockNumber = web3Block.number.uint64 - 1
warn "Eth1 parent block missing. Attempting to request from the network",
parentHash = parentHash.toHex
debug "Got deposit from eth1", pubKey = data.pubKey
while true:
if chainOfParents.len > reorgDepthLimit:
error "Detected Eth1 re-org exceeded the maximum depth limit",
headBlockHash = web3Block.hash.toHex,
ourHeadHash = m.eth1Chain.blocks.peekLast.voteData.block_hash
raise newException(ReorgDepthLimitExceeded, "Reorg depth limit exceeded")
let dep = datatypes.Deposit(data: data)
m.pendingDeposits.add(dep)
inc m.depositCount
m.eth1Block = blkHash
let parentWeb3Block = await dataProvider.getBlockByHash(parentHash)
if parentWeb3Block.number.uint64 != expectedParentBlockNumber:
error "Eth1 data provider supplied invalid parent block",
parentBlockNumber = parentWeb3Block.number.uint64,
expectedParentBlockNumber, parentHash = parentHash.toHex
raise newException(CorruptDataProvider,
"Parent block with incorrect number")
if m.pendingDeposits.len >= SLOTS_PER_EPOCH and
m.pendingDeposits.len >= MIN_GENESIS_ACTIVE_VALIDATOR_COUNT and
blk.timestamp.uint64 >= MIN_GENESIS_TIME.uint64:
# This block is a genesis candidate
var h: Eth2Digest
h.data = array[32, byte](blkHash)
let startTime = blk.timestamp.uint64
var s = initialize_beacon_state_from_eth1(
h, startTime, m.pendingDeposits, {})
chainOfParents.add(await dataProvider.fetchDepositData(parentWeb3Block))
let localParent = m.eth1Chain.findParent(parentWeb3Block)
if localParent != nil:
m.eth1Chain.purgeDescendants(localParent)
for i in countdown(chainOfParents.len - 1, 0):
m.eth1Chain.addBlock chainOfParents[i]
cachedParent = m.eth1Chain.blocks.peekLast
break
if is_valid_genesis_state(s):
# https://github.com/ethereum/eth2.0-pm/tree/6e41fcf383ebeb5125938850d8e9b4e9888389b4/interop/mocked_start#create-genesis-state
s.genesis_time = startTime
dec expectedParentBlockNumber
parentHash = parentWeb3Block.parentHash
m.pendingDeposits.setLen(0)
m.genesisState.new()
m.genesisState[] = s
if not m.genesisStateFut.isNil:
m.genesisStateFut.complete()
m.genesisStateFut = nil
# TODO: Set curBlock to blk number
m.eth1Chain.purgeDescendants(cachedParent)
# TODO: This should be progressing in more independent way.
# The Eth1 cross-link can advance even when there are no new deposits.
m.updateEth1Data(depositCount, depositRoot)
m.eth1Chain.addBlock eth1Block
m.checkForGenesisEvent()
except CatchableError:
# Connection problem? Put the unprocessed deposit back to queue.
# Raising the exception here will lead to a restart of the whole monitor.
m.depositQueue.addFirstNoWait((blockHash, eventType))
raise
proc isRunning*(m: MainchainMonitor): bool =
not m.runFut.isNil
@ -114,76 +439,141 @@ proc getGenesis*(m: MainchainMonitor): Future[BeaconState] {.async.} =
doAssert(not m.genesisState.isNil)
return m.genesisState[]
proc getBlockNumber(web3: Web3, hash: BlockHash): Future[Quantity] {.async.} =
method getBlockByHash*(p: Web3DataProviderRef, hash: BlockHash): Future[BlockObject] =
discard
# p.web3.provider.eth_getBlockByHash(hash, false)
method close*(p: Web3DataProviderRef): Future[void] {.async, locks: 0.} =
if p.subscription != nil:
await p.subscription.unsubscribe()
await p.web3.close()
method fetchDepositData*(p: Web3DataProviderRef,
web3Block: BlockObject): Future[Eth1Block] {.async, locks: 0.} =
let
blockHash = web3Block.hash
depositRoot = await p.ns.get_deposit_root.call(blockNumber = web3Block.number.uint64)
rawCount = await p.ns.get_deposit_count.call(blockNumber = web3Block.number.uint64)
depositCount = bytes_to_int(array[8, byte](rawCount))
depositsJson = await p.ns.getJsonLogs(DepositEvent, blockHash = some(blockHash))
deposits = readJsonDeposits(depositsJson)
return Eth1Block(
number: Eth1BlockNumber(web3Block.number),
timestamp: Eth1BlockTimestamp(web3Block.timestamp),
deposits: deposits,
voteData: Eth1Data(deposit_root: depositRoot.asEth2Digest,
deposit_count: depositCount,
block_hash: blockHash.asEth2Digest))
method onDisconnect*(p: Web3DataProviderRef, handler: DisconnectHandler) {.
gcsafe
locks: 0
# raises: []
.} =
p.web3.onDisconnect = handler
method onDepositEvent*(p: Web3DataProviderRef,
startBlock: Eth1BlockNumber,
handler: DepositEventHandler): Future[void] {.
async
gcsafe
locks: 0
# raises: []
.} =
if p.subscription != nil:
await p.subscription.unsubscribe()
p.subscription = await p.ns.subscribe(
DepositEvent, %*{"fromBlock": startBlock}, handler)
proc getBlockNumber(p: DataProviderRef, hash: BlockHash): Future[Quantity] {.async.} =
debug "Querying block number", hash = $hash
try:
let blk = await web3.provider.eth_getBlockByHash(hash, false)
let blk = await p.getBlockByHash(hash)
return blk.number
except CatchableError as exc:
# TODO this doesn't make too much sense really, but what would be a
# reasonable behavior? no idea - the whole algorithm needs to be
# rewritten to match the spec.
notice "Failed to get block number from hash, using current block instead",
notice "Failed to get Eth1 block number from hash",
hash = $hash, err = exc.msg
return await web3.provider.eth_blockNumber()
raise
proc new*(T: type Web3DataProvider,
web3Url: string,
depositContractAddress: Address): Future[ref Web3DataProvider] {.
async
# raises: [Defect]
.} =
try:
type R = ref T
let
web3 = await newWeb3(web3Url)
ns = web3.contractSender(DepositContract, depositContractAddress)
return R(url: web3Url, web3: web3, ns: ns)
except CatchableError:
return nil
func web3Provider*(web3Url: string): DataProviderFactory =
proc factory(depositContractAddress: Address): Future[DataProviderRef] {.async.} =
result = await Web3DataProvider.new(web3Url, depositContractAddress)
DataProviderFactory(desc: "web3(" & web3Url & ")", new: factory)
proc run(m: MainchainMonitor, delayBeforeStart: Duration) {.async.} =
if delayBeforeStart != ZeroDuration:
await sleepAsync(delayBeforeStart)
let web3 = await newWeb3(m.web3Url)
defer: await web3.close()
let dataProvider = await m.dataProviderFactory.new(m.depositContractAddress)
if dataProvider == nil:
error "Failed to initialize Eth1 data provider",
provider = m.dataProviderFactory.desc
raise newException(CatchableError, "Failed to initialize Eth1 data provider")
defer: await close(dataProvider)
let processFut = m.processDeposits(web3)
let processFut = m.processDeposits(dataProvider)
defer: await processFut
web3.onDisconnect = proc() =
error "Web3 server disconnected", ulr = m.web3Url
dataProvider.onDisconnect do:
error "Eth1 data provider disconnected",
provider = m.dataProviderFactory.desc
processFut.cancel()
# TODO this needs to implement follow distance and the rest of the honest
# validator spec..
let startBlkNum = await web3.getBlockNumber(m.eth1Block)
let startBlkNum = await dataProvider.getBlockNumber(m.startBlock)
notice "Monitoring eth1 deposits",
fromBlock = startBlkNum.uint64,
contract = $m.depositContractAddress,
url = m.web3Url
url = m.dataProviderFactory.desc
let ns = web3.contractSender(DepositContract, m.depositContractAddress)
let s = await ns.subscribe(DepositEvent, %*{"fromBlock": startBlkNum}) do(
await dataProvider.onDepositEvent(Eth1BlockNumber(startBlkNum)) do (
pubkey: Bytes48,
withdrawalCredentials: Bytes32,
amount: Bytes8,
signature: Bytes96, merkleTreeIndex: Bytes8, j: JsonNode):
try:
let blkHash = BlockHash.fromHex(j["blockHash"].getStr())
let amount = bytes_to_int(array[8, byte](amount))
let
blockHash = BlockHash.fromHex(j["blockHash"].getStr())
eventType = if j.hasKey("removed"): RemovedEvent
else: NewEvent
m.depositQueue.addLastNoWait((blockHash, eventType))
m.depositQueue.addLastNoWait((blkHash,
DepositData(pubkey: ValidatorPubKey.init(array[48, byte](pubkey)),
withdrawal_credentials: Eth2Digest(data: array[32, byte](withdrawalCredentials)),
amount: amount,
signature: ValidatorSig.init(array[96, byte](signature)))))
except CatchableError as exc:
warn "Received invalid deposit", err = exc.msg, j
try:
await processFut
finally:
await s.unsubscribe()
proc start(m: MainchainMonitor, delayBeforeStart: Duration) =
if m.runFut.isNil:
let runFut = m.run(delayBeforeStart)
m.runFut = runFut
runFut.addCallback() do(p: pointer):
if runFut.failed and runFut == m.runFut:
error "Mainchain monitor failure, restarting", err = runFut.error.msg
m.runFut = nil
m.start(5.seconds)
runFut.addCallback do (p: pointer):
if runFut.failed:
if runFut.error[] of CatchableError:
if runFut == m.runFut:
error "Mainchain monitor failure, restarting", err = runFut.error.msg
m.runFut = nil
m.start(5.seconds)
else:
fatal "Fatal exception reached", err = runFut.error.msg
quit 1
proc start*(m: MainchainMonitor) {.inline.} =
m.start(0.seconds)
@ -193,20 +583,9 @@ proc stop*(m: MainchainMonitor) =
m.runFut.cancel()
m.runFut = nil
proc getPendingDeposits*(m: MainchainMonitor): seq[Deposit] =
# This should be a simple accessor for the reference kept above
m.pendingDeposits
# TODO update after spec change removed Specials
# iterator getValidatorActions*(m: MainchainMonitor,
# fromBlock, toBlock: Eth2Digest): SpecialRecord =
# # It's probably better if this doesn't return a SpecialRecord, but
# # rather a more readable description of the change that can be packed
# # in a SpecialRecord by the client of the API.
# discard
proc getLatestEth1BlockHash*(url: string): Future[Eth2Digest] {.async.} =
let web3 = await newWeb3(url)
defer: await web3.close()
let blk = await web3.provider.eth_getBlockByNumber("latest", false)
result.data = array[32, byte](blk.hash)
await web3.close()
return Eth2Digest(data: array[32, byte](blk.hash))

View File

@ -80,6 +80,8 @@ const
# https://github.com/ethereum/eth2.0-specs/blob/v0.11.1/specs/phase0/p2p-interface.md#configuration
ATTESTATION_PROPAGATION_SLOT_RANGE* = 32
SLOTS_PER_ETH1_VOTING_PERIOD* = Slot(EPOCHS_PER_ETH1_VOTING_PERIOD * SLOTS_PER_EPOCH)
template maxSize*(n: int) {.pragma.}
type

View File

@ -133,8 +133,7 @@ proc process_randao(
# https://github.com/ethereum/eth2.0-specs/blob/v0.11.1/specs/phase0/beacon-chain.md#eth1-data
func process_eth1_data(state: var BeaconState, body: BeaconBlockBody) {.nbench.}=
state.eth1_data_votes.add body.eth1_data
if state.eth1_data_votes.count(body.eth1_data) * 2 >
EPOCHS_PER_ETH1_VOTING_PERIOD * SLOTS_PER_EPOCH:
if state.eth1_data_votes.count(body.eth1_data) * 2 > SLOTS_PER_ETH1_VOTING_PERIOD.int:
state.eth1_data = body.eth1_data
# https://github.com/ethereum/eth2.0-specs/blob/v0.11.1/specs/phase0/beacon-chain.md#is_slashable_validator

View File

@ -235,3 +235,4 @@ func is_proposer(
var cache = get_empty_per_epoch_cache()
let proposer_index = get_beacon_proposer_index(state, cache)
proposer_index.isSome and proposer_index.get == validator_index

View File

@ -7,6 +7,9 @@ import
contract(DepositContract):
proc deposit(pubkey: Bytes48, withdrawalCredentials: Bytes32, signature: Bytes96, deposit_data_root: FixedBytes[32])
type
DelayGenerator* = proc(): chronos.Duration {.closure, gcsafe.}
proc writeTextFile(filename: string, contents: string) =
writeFile(filename, contents)
# echo "Wrote ", filename
@ -62,15 +65,16 @@ proc generateDeposits*(totalValidators: int,
proc sendDeposits*(
deposits: seq[Deposit],
depositWeb3Url, depositContractAddress, privateKey: string) {.async.} =
web3Url, depositContractAddress, privateKey: string,
delayGenerator: DelayGenerator = nil) {.async.} =
var web3 = await newWeb3(depositWeb3Url)
var web3 = await newWeb3(web3Url)
if privateKey.len != 0:
web3.privateKey = PrivateKey.fromHex(privateKey).tryGet()
else:
let accounts = await web3.provider.eth_accounts()
if accounts.len == 0:
error "No account offered by the web3 provider", web3url = depositWeb3Url
error "No account offered by the web3 provider", web3Url
return
web3.defaultAccount = accounts[0]
@ -84,17 +88,20 @@ proc sendDeposits*(
Bytes96(dp.data.signature.toRaw()),
FixedBytes[32](hash_tree_root(dp.data).data)).send(value = 32.u256.ethToWei, gasPrice = 1)
if delayGenerator != nil:
await sleepAsync(delayGenerator())
when isMainModule:
import confutils
cli do (totalValidators: int = 125000,
outputDir: string = "validators",
randomKeys: bool = false,
depositWeb3Url: string = "",
web3Url: string = "",
depositContractAddress: string = ""):
let deposits = generateDeposits(totalValidators, outputDir, randomKeys)
if depositWeb3Url.len() > 0 and depositContractAddress.len() > 0:
if web3Url.len() > 0 and depositContractAddress.len() > 0:
echo "Sending deposits to eth1..."
waitFor sendDeposits(deposits, depositWeb3Url, depositContractAddress, "")
waitFor sendDeposits(deposits, web3Url, depositContractAddress, "")
echo "Done"

View File

@ -41,6 +41,8 @@ else:
# for heap-usage-by-instance-type metrics and object base-type strings
--define:nimTypeNames
switch("import", "testutils/moduletests")
# the default open files limit is too low on macOS (512), breaking the
# "--debugger:native" build. It can be increased with `ulimit -n 1024`.
if not defined(macosx):

View File

@ -30,7 +30,7 @@ type
exit: SignedVoluntaryExit
# This and AssertionError are raised to indicate programming bugs
# A wrapper to allow exception tracking to identify unexpected exceptions
FuzzCrashError = object of Exception
FuzzCrashError = object of CatchableError
# TODO: change ptr uint to ptr csize_t when available in newer Nim version.
proc copyState(state: BeaconState, output: ptr byte,

View File

@ -140,5 +140,6 @@ cli do (skipGoerliKey {.
--web3-url={web3Url}
{bootstrapFileOpt}
{logLevelOpt}
{depositContractOpt}
--state-snapshot="{testnetDir/genesisFile}" """ & depositContractOpt, "\n", " ")

View File

@ -21,6 +21,7 @@ import # Unit test
./test_kvstore,
./test_mocking,
./test_kvstore_sqlite3,
./test_mainchain_monitor,
./test_ssz,
./test_state_transition,
./test_sync_protocol,

View File

@ -34,9 +34,9 @@ cd "$GIT_ROOT"
DATA_DIR="${SIMULATION_DIR}/node-$NODE_ID"
PORT=$(( BASE_P2P_PORT + NODE_ID ))
NAT_FLAG="--nat:extip:127.0.0.1"
NAT_ARG="--nat:extip:127.0.0.1"
if [ "${NAT:-}" == "1" ]; then
NAT_FLAG="--nat:any"
NAT_ARG="--nat:any"
fi
mkdir -p "$DATA_DIR/validators"
@ -54,6 +54,11 @@ fi
rm -rf "$DATA_DIR/dump"
mkdir -p "$DATA_DIR/dump"
SNAPSHOT_ARG=""
if [ -f "${SNAPSHOT_FILE}" ]; then
SNAPSHOT_ARG="--state-snapshot=${SNAPSHOT_FILE}"
fi
# if you want tracing messages, add "--log-level=TRACE" below
cd "$DATA_DIR" && $BEACON_NODE_BIN \
--log-level=${LOG_LEVEL:-DEBUG} \
@ -62,9 +67,9 @@ cd "$DATA_DIR" && $BEACON_NODE_BIN \
--node-name=$NODE_ID \
--tcp-port=$PORT \
--udp-port=$PORT \
$NAT_FLAG \
--state-snapshot=$SNAPSHOT_FILE \
$DEPOSIT_WEB3_URL_ARG \
$SNAPSHOT_ARG \
$NAT_ARG \
$WEB3_ARG \
--deposit-contract=$DEPOSIT_CONTRACT_ADDRESS \
--rpc \
--rpc-address="127.0.0.1" \

View File

@ -32,6 +32,86 @@ else
EXE_SUFFIX=""
fi
# to allow overriding the program names
MULTITAIL="${MULTITAIL:-multitail}"
TMUX="${TMUX:-tmux}"
GANACHE="${GANACHE:-ganache-cli}"
PROMETHEUS="${PROMETHEUS:-prometheus}"
TMUX_SESSION_NAME="${TMUX_SESSION_NAME:-nbc-sim}"
WAIT_GENESIS="${WAIT_GENESIS:-no}"
# Using tmux or multitail is an opt-in
USE_MULTITAIL="${USE_MULTITAIL:-no}"
type "$MULTITAIL" &>/dev/null || { echo "${MULTITAIL}" is missing; USE_MULTITAIL="no"; }
USE_TMUX="${USE_TMUX:-no}"
type "$TMUX" &>/dev/null || { echo "${TMUX}" is missing; USE_TMUX="no"; }
USE_GANACHE="${USE_GANACHE:-no}"
type "$GANACHE" &>/dev/null || { echo $GANACHE is missing; USE_GANACHE="no"; }
USE_PROMETHEUS="${LAUNCH_PROMETHEUS:-no}"
type "$PROMETHEUS" &>/dev/null || { echo $PROMETHEUS is missing; USE_PROMETHEUS="no"; }
# Prometheus config (continued inside the loop)
mkdir -p "${METRICS_DIR}"
cat > "${METRICS_DIR}/prometheus.yml" <<EOF
global:
scrape_interval: 1s
scrape_configs:
- job_name: "nimbus"
static_configs:
EOF
for i in $(seq $MASTER_NODE -1 $TOTAL_USER_NODES); do
# Prometheus config
cat >> "${METRICS_DIR}/prometheus.yml" <<EOF
- targets: ['127.0.0.1:$(( BASE_METRICS_PORT + i ))']
labels:
node: '$i'
EOF
done
COMMANDS=()
if [[ "$USE_TMUX" != "no" ]]; then
$TMUX new-session -s "${TMUX_SESSION_NAME}" -d
# maybe these should be moved to a user config file
$TMUX set-option -t "${TMUX_SESSION_NAME}" history-limit 999999
$TMUX set-option -t "${TMUX_SESSION_NAME}" remain-on-exit on
$TMUX set -t "${TMUX_SESSION_NAME}" mouse on
# We create a new window, so the above settings can take place
$TMUX new-window -d -t "${TMUX_SESSION_NAME}" -n "sim"
trap 'tmux kill-session -t "${TMUX_SESSION_NAME}"' SIGINT EXIT
fi
if [[ "$USE_GANACHE" != "no" ]]; then
if [[ "$USE_TMUX" != "no" ]]; then
$TMUX new-window -d -t $TMUX_SESSION_NAME -n "$GANACHE" "$GANACHE"
elif [[ "$USE_MULTITAIL" != "no" ]]; then
COMMANDS+=( " -cT ansi -t '$GANACHE'" )
else
$GANACHE &
fi
fi
if [[ "$USE_PROMETHEUS" != "no" ]]; then
if [[ "$USE_TMUX" != "no" ]]; then
$TMUX new-window -d -t $TMUX_SESSION_NAME -n "$PROMETHEUS" "cd '$METRICS_DIR' && $PROMETHEUS"
else
echo "$PROMETHEUS can be used currently only with USE_TMUX=1"
fi
fi
if [[ "$USE_TMUX" != "no" ]]; then
$TMUX select-window -t "${TMUX_SESSION_NAME}:sim"
fi
build_beacon_node () {
OUTPUT_BIN=$1; shift
PARAMS="$CUSTOM_NIMFLAGS $DEFS $@"
@ -45,29 +125,48 @@ if [ ! -f "${LAST_VALIDATOR}" ]; then
echo Building "${DEPLOY_DEPOSIT_CONTRACT_BIN}"
$MAKE NIMFLAGS="-o:\"$DEPLOY_DEPOSIT_CONTRACT_BIN\" $CUSTOM_NIMFLAGS $DEFS" deposit_contract
if [ "$DEPOSIT_WEB3_URL_ARG" != "" ]; then
DEPOSIT_CONTRACT_ADDRESS=$($DEPLOY_DEPOSIT_CONTRACT_BIN deploy $DEPOSIT_WEB3_URL_ARG)
if [ "$WEB3_ARG" != "" ]; then
echo Deploying the validator deposit contract...
DEPOSIT_CONTRACT_ADDRESS=$($DEPLOY_DEPOSIT_CONTRACT_BIN deploy $WEB3_ARG)
echo Contract deployed at $DEPOSIT_CONTRACT_ADDRESS
export DEPOSIT_CONTRACT_ADDRESS
fi
DELAY_ARGS=""
# Uncomment this line to slow down the initial deposits.
# This will spread them across multiple blocks which is
# a more realistic scenario.
DELAY_ARGS="--min-delay=1 --max-delay=5"
MAKE_DEPOSITS_WEB3_ARG=$WEB3_ARG
if [[ "$WAIT_GENESIS" == "no" ]]; then
MAKE_DEPOSITS_WEB3_ARG=""
fi
$BEACON_NODE_BIN makeDeposits \
--quickstart-deposits="${NUM_VALIDATORS}" \
--deposits-dir="$VALIDATORS_DIR" \
$DEPOSIT_WEB3_URL_ARG \
$MAKE_DEPOSITS_WEB3_ARG $DELAY_ARGS \
--deposit-contract="${DEPOSIT_CONTRACT_ADDRESS}"
echo "All deposits prepared"
fi
if [ ! -f "${SNAPSHOT_FILE}" ]; then
$BEACON_NODE_BIN \
--data-dir="${SIMULATION_DIR}/node-$MASTER_NODE" \
createTestnet \
--validators-dir="${VALIDATORS_DIR}" \
--total-validators="${NUM_VALIDATORS}" \
--output-genesis="${SNAPSHOT_FILE}" \
--output-bootstrap-file="${NETWORK_BOOTSTRAP_FILE}" \
--bootstrap-address=127.0.0.1 \
--bootstrap-port=$(( BASE_P2P_PORT + MASTER_NODE )) \
--genesis-offset=5 # Delay in seconds
if [[ "${WAIT_GENESIS}" == "no" ]]; then
echo Creating testnet genesis...
$BEACON_NODE_BIN \
--data-dir="${SIMULATION_DIR}/node-$MASTER_NODE" \
createTestnet \
--validators-dir="${VALIDATORS_DIR}" \
--total-validators="${NUM_VALIDATORS}" \
--output-genesis="${SNAPSHOT_FILE}" \
--output-bootstrap-file="${NETWORK_BOOTSTRAP_FILE}" \
--bootstrap-address=127.0.0.1 \
--bootstrap-port=$(( BASE_P2P_PORT + MASTER_NODE )) \
--genesis-offset=5 # Delay in seconds
fi
fi
rm -f beacon_node.log
@ -77,29 +176,6 @@ if [ -f "${MASTER_NODE_ADDRESS_FILE}" ]; then
rm "${MASTER_NODE_ADDRESS_FILE}"
fi
# to allow overriding the program names
MULTITAIL="${MULTITAIL:-multitail}"
TMUX="${TMUX:-tmux}"
TMUX_SESSION_NAME="${TMUX_SESSION_NAME:-nbc-network-sim}"
# Using tmux or multitail is an opt-in
USE_MULTITAIL="${USE_MULTITAIL:-no}"
type "$MULTITAIL" &>/dev/null || { echo "${MULTITAIL}" is missing; USE_MULTITAIL="no"; }
USE_TMUX="${USE_TMUX:-no}"
type "$TMUX" &>/dev/null || { echo "${TMUX}" is missing; USE_TMUX="no"; }
# Prometheus config (continued inside the loop)
mkdir -p "${METRICS_DIR}"
cat > "${METRICS_DIR}/prometheus.yml" <<EOF
global:
scrape_interval: 1s
scrape_configs:
- job_name: "nimbus"
static_configs:
EOF
PROCESS_DASHBOARD_BIN="build/process_dashboard${EXE_SUFFIX}"
if [[ ! -f "$PROCESS_DASHBOARD_BIN" ]]; then
@ -107,6 +183,7 @@ if [[ ! -f "$PROCESS_DASHBOARD_BIN" ]]; then
fi
# use the exported Grafana dashboard for a single node to create one for all nodes
echo Creating grafana dashboards...
"${PROCESS_DASHBOARD_BIN}" \
--nodes=${TOTAL_NODES} \
--in="${SIM_ROOT}/beacon-chain-sim-node0-Grafana-dashboard.json" \
@ -115,26 +192,20 @@ fi
# Kill child processes on Ctrl-C by sending SIGTERM to the whole process group,
# passing the negative PID of this shell instance to the "kill" command.
# Trap and ignore SIGTERM, so we don't kill this process along with its children.
if [ "$USE_MULTITAIL" = "no" ]; then
trap '' SIGTERM
if [[ "$USE_MULTITAIL" == "no" && "$USE_TMUX" == "no" ]]; then
trap 'pkill -P $$ beacon_node' SIGINT EXIT
fi
COMMANDS=()
if [[ "$USE_TMUX" != "no" ]]; then
$TMUX new-session -s "${TMUX_SESSION_NAME}" -d
# maybe these should be moved to a user config file
$TMUX set-option -t "${TMUX_SESSION_NAME}" history-limit 999999
$TMUX set-option -t "${TMUX_SESSION_NAME}" remain-on-exit on
$TMUX set -t "${TMUX_SESSION_NAME}" mouse on
fi
LAST_WAITING_NODE=0
for i in $(seq $MASTER_NODE -1 $TOTAL_USER_NODES); do
if [[ "$i" != "$MASTER_NODE" && "$USE_MULTITAIL" == "no" ]]; then
# Wait for the master node to write out its address file
while [ ! -f "${MASTER_NODE_ADDRESS_FILE}" ]; do
if (( LAST_WAITING_NODE != i )); then
echo Waiting for $MASTER_NODE_ADDRESS_FILE to appear...
LAST_WAITING_NODE=i
fi
sleep 0.1
done
fi
@ -142,6 +213,8 @@ for i in $(seq $MASTER_NODE -1 $TOTAL_USER_NODES); do
CMD="${SIM_ROOT}/run_node.sh ${i} --verify-finalization"
if [[ "$USE_TMUX" != "no" ]]; then
echo "Starting node $i..."
echo $TMUX split-window -t "${TMUX_SESSION_NAME}" "$CMD"
$TMUX split-window -t "${TMUX_SESSION_NAME}" "$CMD"
$TMUX select-layout -t "${TMUX_SESSION_NAME}" tiled
elif [[ "$USE_MULTITAIL" != "no" ]]; then
@ -155,16 +228,13 @@ for i in $(seq $MASTER_NODE -1 $TOTAL_USER_NODES); do
else
eval "${CMD}" &
fi
# Prometheus config
cat >> "${METRICS_DIR}/prometheus.yml" <<EOF
- targets: ['127.0.0.1:$(( BASE_METRICS_PORT + i ))']
labels:
node: '$i'
EOF
done
if [[ "$USE_TMUX" != "no" ]]; then
# kill the console window in the pane where the simulation is running
$TMUX kill-pane -t $TMUX_SESSION_NAME:sim.0
# kill the original console window
# (this one doesn't have the right history-limit)
$TMUX kill-pane -t $TMUX_SESSION_NAME:0.0
$TMUX select-layout -t "${TMUX_SESSION_NAME}" tiled
$TMUX attach-session -t "${TMUX_SESSION_NAME}" -d

View File

@ -37,8 +37,11 @@ MASTER_NODE_ADDRESS_FILE="${SIMULATION_DIR}/node-${MASTER_NODE}/beacon_node.addr
BASE_P2P_PORT=30000
BASE_RPC_PORT=7000
BASE_METRICS_PORT=8008
# Set DEPOSIT_WEB3_URL_ARG to empty to get genesis state from file, not using web3
# DEPOSIT_WEB3_URL_ARG=--web3-url=ws://localhost:8545
DEPOSIT_WEB3_URL_ARG=""
DEPOSIT_CONTRACT_ADDRESS="0x"
if [[ "$USE_GANACHE" == "yes" ]]; then
WEB3_ARG=--web3-url=ws://localhost:8545
else
WEB3_ARG=""
DEPOSIT_CONTRACT_ADDRESS="0x"
fi

View File

@ -0,0 +1,52 @@
import
unittest,
chronos, web3/ethtypes,
../beacon_chain/mainchain_monitor
type
MockDataProvider = ref object of DataProvider
method getBlockByHash*(p: MockDataProvider, hash: BlockHash): Future[BlockObject] {.
async
gcsafe
# raises: [Defect]
.} =
return BlockObject()
method onDisconnect*(p: MockDataProvider, handler: DisconnectHandler) {.
async
gcsafe
# raises: []
.} =
discard
method onDepositEvent*(p: MockDataProvider,
startBlock: Eth1BlockNumber,
handler: DepositEventHandler): Future[void] {.
async
gcsafe
# raises: []
.} =
discard
method close*(p: MockDataProvider): Future[void] {.
async
gcsafe
# raises: [Defect]
.} =
discard
method fetchDepositData*(p: MockDataProvider,
web3Block: BlockObject): Future[Eth1Block] {.
async
gcsafe
# raises: [Defect, CatchableError]
.} =
return Eth1Block()
suite "Eth1 Chain":
discard
suite "Mainchain monitor":
discard

1
vendor/nim-rocksdb vendored Submodule

@ -0,0 +1 @@
Subproject commit 08fec021c0f28f63d1221d40a655078b5b923d1b

2
vendor/nim-web3 vendored

@ -1 +1 @@
Subproject commit 969adf2f1ef42753ba26d5ab7eca01617c846792
Subproject commit 0ca608996289a2b2a4ea9bba715bb9f3e99de869