Skip to content

Commit d1df279

Browse files
committed
serve libp2p protocol for light client sync
This extends the `--serve-light-client-data` launch option to serve locally collected light client data via libp2p. See ethereum/consensus-specs#2802 Not yet implemented: - Backfill of historic best `LightClientUpdate`. - Client side of the protocol. Known issues: - Heavy forking for more than a sync committee periods may result in incorrect best `LightClientUpdate` computation. To test, in `conf.nim` change `serveLightClientData`'s `defaultValue` to `true`, then run this command: ``` scripts/launch_local_testnet.sh --kill-old-processes --preset minimal \ --nodes 4 --disable-htop --stop-at-epoch 7 ``` The log files of the beacon nodes will be in the `local_testnet_data` directory. They are named `log0.txt` through `log3.txt`. The logs can be browsed for light client related messages.
1 parent 8b07541 commit d1df279

12 files changed

+253
-18
lines changed

beacon_chain/gossip_processing/eth2_processor.nim

+24
Original file line numberDiff line numberDiff line change
@@ -58,6 +58,10 @@ declareCounter beacon_sync_committee_contributions_received,
5858
"Number of valid sync committee contributions processed by this node"
5959
declareCounter beacon_sync_committee_contributions_dropped,
6060
"Number of invalid sync committee contributions dropped by this node", labels = ["reason"]
61+
declareCounter beacon_optimistic_light_client_updates_received,
62+
"Number of valid optimistic light client updates processed by this node"
63+
declareCounter beacon_optimistic_light_client_updates_dropped,
64+
"Number of invalid optimistic light client updates dropped by this node", labels = ["reason"]
6165

6266
const delayBuckets = [2.0, 4.0, 6.0, 8.0, 10.0, 12.0, 14.0, Inf]
6367

@@ -528,3 +532,23 @@ proc contributionValidator*(
528532
beacon_sync_committee_contributions_dropped.inc(1, [$v.error[0]])
529533

530534
err(v.error())
535+
536+
proc optimisticLightClientUpdateValidator*(
537+
self: var Eth2Processor, src: MsgSource,
538+
optimistic_update: OptimisticLightClientUpdate
539+
): Result[void, ValidationError] =
540+
logScope:
541+
optimisticUpdate = shortLog(optimistic_update)
542+
543+
debug "Optimistic light client update received"
544+
545+
let v = self.dag.validateOptimisticLightClientUpdate(optimistic_update)
546+
if v.isOk():
547+
trace "Optimistic light client update validated"
548+
549+
beacon_optimistic_light_client_updates_received.inc()
550+
else:
551+
debug "Dropping optimistic light client update", error = v.error
552+
beacon_optimistic_light_client_updates_dropped.inc(1, [$v.error[0]])
553+
554+
v

beacon_chain/gossip_processing/gossip_validation.nim

+17
Original file line numberDiff line numberDiff line change
@@ -984,3 +984,20 @@ proc validateContribution*(
984984
sig.get()
985985

986986
return ok((sig, participants))
987+
988+
# https://github.com/ethereum/consensus-specs/blob/v1.1.8/specs/altair/sync-protocol.md#optimistic_light_client_update
989+
proc validateOptimisticLightClientUpdate*(
990+
dag: ChainDAGRef, optimistic_update: OptimisticLightClientUpdate):
991+
Result[void, ValidationError] =
992+
template latest_local_update(): auto = dag.optimisticLightClientUpdate
993+
994+
if optimistic_update != latest_local_update:
995+
# [IGNORE] The optimistic update is not attesting to the latest block's
996+
# parent block.
997+
if optimistic_update.attested_header != latest_local_update.attested_header:
998+
return errIgnore("OptimisticLightClientUpdate: different attested block")
999+
1000+
# [REJECT] The optimistic update does not match the expected value.
1001+
return errReject("OptimisticLightClientUpdate: update does not match block")
1002+
1003+
ok()

beacon_chain/networking/eth2_network.nim

+26-2
Original file line numberDiff line numberDiff line change
@@ -156,6 +156,7 @@ type
156156
Success
157157
InvalidRequest
158158
ServerError
159+
ResourceUnavailable
159160

160161
PeerStateInitializer* = proc(peer: Peer): RootRef {.gcsafe, raises: [Defect].}
161162
NetworkStateInitializer* = proc(network: EthereumNode): RootRef {.gcsafe, raises: [Defect].}
@@ -204,6 +205,8 @@ type
204205

205206
InvalidInputsError* = object of CatchableError
206207

208+
ResourceUnavailableError* = object of CatchableError
209+
207210
NetRes*[T] = Result[T, Eth2NetworkingError]
208211
## This is type returned from all network requests
209212

@@ -707,6 +710,13 @@ proc handleIncomingStream(network: Eth2Node,
707710
template returnInvalidRequest(msg: string) =
708711
returnInvalidRequest(ErrorMsg msg.toBytes)
709712

713+
template returnResourceUnavailable(msg: ErrorMsg) =
714+
await sendErrorResponse(peer, conn, ResourceUnavailable, msg)
715+
return
716+
717+
template returnResourceUnavailable(msg: string) =
718+
returnResourceUnavailable(ErrorMsg msg.toBytes)
719+
710720
let s = when useNativeSnappy:
711721
let fs = libp2pInput(conn)
712722

@@ -771,8 +781,8 @@ proc handleIncomingStream(network: Eth2Node,
771781
await callUserHandler(MsgType, peer, conn, msg.get)
772782
except InvalidInputsError as err:
773783
returnInvalidRequest err.msg
774-
await sendErrorResponse(peer, conn, ServerError,
775-
ErrorMsg err.msg.toBytes)
784+
except ResourceUnavailableError as err:
785+
returnResourceUnavailable err.msg
776786
except CatchableError as err:
777787
await sendErrorResponse(peer, conn, ServerError,
778788
ErrorMsg err.msg.toBytes)
@@ -2225,3 +2235,17 @@ proc broadcastSignedContributionAndProof*(
22252235
node: Eth2Node, msg: SignedContributionAndProof) =
22262236
let topic = getSyncCommitteeContributionAndProofTopic(node.forkDigests.altair)
22272237
node.broadcast(topic, msg)
2238+
2239+
proc broadcastOptimisticLightClientUpdate*(
2240+
node: Eth2Node, msg: OptimisticLightClientUpdate) =
2241+
let
2242+
forkDigest =
2243+
if msg.fork_version == node.cfg.SHARDING_FORK_VERSION:
2244+
node.forkDigests.sharding
2245+
elif msg.fork_version == node.cfg.BELLATRIX_FORK_VERSION:
2246+
node.forkDigests.bellatrix
2247+
else:
2248+
doAssert msg.fork_version == node.cfg.ALTAIR_FORK_VERSION
2249+
node.forkDigests.altair
2250+
topic = getOptimisticLightClientUpdateTopic(forkDigest)
2251+
node.broadcast(topic, msg)

beacon_chain/networking/faststreams_backend.nim

+2-2
Original file line numberDiff line numberDiff line change
@@ -1,5 +1,5 @@
11
# beacon_chain
2-
# Copyright (c) 2018-2021 Status Research & Development GmbH
2+
# Copyright (c) 2018-2022 Status Research & Development GmbH
33
# Licensed and distributed under either of
44
# * MIT license (license terms in the root directory or at https://opensource.org/licenses/MIT).
55
# * Apache v2 license (license terms in the root directory or at https://www.apache.org/licenses/LICENSE-2.0).
@@ -112,7 +112,7 @@ proc readResponseChunk(s: AsyncInputStream,
112112

113113
let responseCode = ResponseCode responseCodeByte
114114
case responseCode:
115-
of InvalidRequest, ServerError:
115+
of InvalidRequest, ServerError, ResourceUnavailable:
116116
let errorMsgChunk = await readChunkPayload(s, noSnappy, string)
117117
let errorMsg = if errorMsgChunk.isOk: errorMsgChunk.value
118118
else: return err(errorMsgChunk.error)

beacon_chain/networking/libp2p_streams_backend.nim

+2-2
Original file line numberDiff line numberDiff line change
@@ -1,5 +1,5 @@
11
# beacon_chain
2-
# Copyright (c) 2018-2021 Status Research & Development GmbH
2+
# Copyright (c) 2018-2022 Status Research & Development GmbH
33
# Licensed and distributed under either of
44
# * MIT license (license terms in the root directory or at https://opensource.org/licenses/MIT).
55
# * Apache v2 license (license terms in the root directory or at https://www.apache.org/licenses/LICENSE-2.0).
@@ -148,7 +148,7 @@ proc readResponseChunk(conn: Connection, peer: Peer,
148148

149149
let responseCode = ResponseCode responseCodeByte
150150
case responseCode:
151-
of InvalidRequest, ServerError:
151+
of InvalidRequest, ServerError, ResourceUnavailable:
152152
let errorMsgChunk = await readChunkPayload(conn, peer, ErrorMsg)
153153
let errorMsg = if errorMsgChunk.isOk: errorMsgChunk.value
154154
else: return err(errorMsgChunk.error)

beacon_chain/nimbus_beacon_node.nim

+19
Original file line numberDiff line numberDiff line change
@@ -730,6 +730,10 @@ proc addAltairMessageHandlers(node: BeaconNode, forkDigest: ForkDigest, slot: Sl
730730
getSyncCommitteeContributionAndProofTopic(forkDigest), basicParams)
731731
node.network.updateSyncnetsMetadata(syncnets)
732732

733+
if node.config.serveLightClientData:
734+
node.network.subscribe(
735+
getOptimisticLightClientUpdateTopic(forkDigest), basicParams)
736+
733737
proc removeAltairMessageHandlers(node: BeaconNode, forkDigest: ForkDigest) =
734738
node.removePhase0MessageHandlers(forkDigest)
735739

@@ -742,6 +746,9 @@ proc removeAltairMessageHandlers(node: BeaconNode, forkDigest: ForkDigest) =
742746
node.network.unsubscribe(
743747
getSyncCommitteeContributionAndProofTopic(forkDigest))
744748

749+
if node.config.serveLightClientData:
750+
node.network.unsubscribe(getOptimisticLightClientUpdateTopic(forkDigest))
751+
745752
proc trackSyncCommitteeTopics*(node: BeaconNode) =
746753
# TODO
747754
discard
@@ -1149,6 +1156,18 @@ proc installMessageValidators(node: BeaconNode) =
11491156
installSyncCommitteeeValidators(node.dag.forkDigests.altair)
11501157
installSyncCommitteeeValidators(node.dag.forkDigests.bellatrix)
11511158

1159+
if node.config.serveLightClientData:
1160+
template installOptimisticLightClientUpdateValidator(digest: auto) =
1161+
node.network.addValidator(
1162+
getOptimisticLightClientUpdateTopic(digest),
1163+
proc(msg: OptimisticLightClientUpdate): ValidationResult =
1164+
toValidationResult(
1165+
node.processor[].optimisticLightClientUpdateValidator(
1166+
MsgSource.gossip, msg)))
1167+
1168+
installOptimisticLightClientUpdateValidator(node.dag.forkDigests.altair)
1169+
installOptimisticLightClientUpdateValidator(node.dag.forkDigests.bellatrix)
1170+
11521171
proc stop(node: BeaconNode) =
11531172
bnStatus = BeaconNodeStatus.Stopping
11541173
notice "Graceful shutdown"

beacon_chain/spec/beacon_time.nim

+6-1
Original file line numberDiff line numberDiff line change
@@ -1,5 +1,5 @@
11
# beacon_chain
2-
# Copyright (c) 2018-2021 Status Research & Development GmbH
2+
# Copyright (c) 2018-2022 Status Research & Development GmbH
33
# Licensed and distributed under either of
44
# * MIT license (license terms in the root directory or at https://opensource.org/licenses/MIT).
55
# * Apache v2 license (license terms in the root directory or at https://www.apache.org/licenses/LICENSE-2.0).
@@ -146,6 +146,9 @@ const
146146
# https://github.com/ethereum/consensus-specs/blob/v1.1.8/specs/altair/validator.md#broadcast-sync-committee-contribution
147147
syncContributionSlotOffset* = TimeDiff(nanoseconds:
148148
NANOSECONDS_PER_SLOT.int64 * 2 div INTERVALS_PER_SLOT)
149+
# https://github.com/ethereum/consensus-specs/blob/v1.1.8/specs/altair/sync-protocol.md#block-proposal
150+
optimisticLightClientUpdateSlotOffset* = TimeDiff(nanoseconds:
151+
NANOSECONDS_PER_SLOT.int64 div INTERVALS_PER_SLOT)
149152

150153
func toFloatSeconds*(t: TimeDiff): float =
151154
float(t.nanoseconds) / 1_000_000_000.0
@@ -167,6 +170,8 @@ func sync_committee_message_deadline*(s: Slot): BeaconTime =
167170
s.start_beacon_time + syncCommitteeMessageSlotOffset
168171
func sync_contribution_deadline*(s: Slot): BeaconTime =
169172
s.start_beacon_time + syncContributionSlotOffset
173+
func optimistic_light_client_update_deadline*(s: Slot): BeaconTime =
174+
s.start_beacon_time + optimisticLightClientUpdateSlotOffset
170175

171176
func slotOrZero*(time: BeaconTime): Slot =
172177
let exSlot = time.toSlot

beacon_chain/spec/datatypes/altair.nim

+16-1
Original file line numberDiff line numberDiff line change
@@ -27,7 +27,7 @@
2727
import
2828
std/[typetraits, sets, hashes],
2929
chronicles,
30-
stew/[assign2, bitops2],
30+
stew/[assign2, bitops2, objects],
3131
"."/[base, phase0]
3232

3333
export base, sets
@@ -601,9 +601,24 @@ chronicles.formatIt SyncCommitteeContribution: shortLog(it)
601601
chronicles.formatIt ContributionAndProof: shortLog(it)
602602
chronicles.formatIt SignedContributionAndProof: shortLog(it)
603603

604+
func shortLog*(v: LightClientUpdate): auto =
605+
(
606+
attested_header: shortLog(v.attested_header),
607+
finalized_header: shortLog(v.finalized_header),
608+
num_active_participants: countOnes(v.sync_aggregate.sync_committee_bits),
609+
is_signed_by_next: v.next_sync_committee.isZeroMemory
610+
)
611+
604612
template hash*(x: LightClientUpdate): Hash =
605613
hash(x.header)
606614

615+
func shortLog*(v: OptimisticLightClientUpdate): auto =
616+
(
617+
attested_header: shortLog(v.attested_header),
618+
num_active_participants: countOnes(v.sync_aggregate.sync_committee_bits),
619+
is_signed_by_next: v.is_signed_by_next_sync_committee
620+
)
621+
607622
func clear*(info: var EpochInfo) =
608623
info.validators.setLen(0)
609624
info.balances = UnslashedParticipatingBalances()

beacon_chain/spec/network.nim

+5
Original file line numberDiff line numberDiff line change
@@ -94,6 +94,11 @@ func getSyncCommitteeContributionAndProofTopic*(forkDigest: ForkDigest): string
9494
## For subscribing and unsubscribing to/from a subnet.
9595
eth2Prefix(forkDigest) & "sync_committee_contribution_and_proof/ssz_snappy"
9696

97+
# https://github.com/ethereum/consensus-specs/blob/v1.1.8/specs/altair/sync-protocol.md#optimistic_light_client_update
98+
func getOptimisticLightClientUpdateTopic*(forkDigest: ForkDigest): string =
99+
## For broadcasting the latest `OptimisticLightClientUpdate` to light clients.
100+
eth2Prefix(forkDigest) & "optimistic_light_client_update/ssz_snappy"
101+
97102
func getENRForkID*(cfg: RuntimeConfig,
98103
epoch: Epoch,
99104
genesis_validators_root: Eth2Digest): ENRForkID =

beacon_chain/spec/ssz_codec.nim

+5-2
Original file line numberDiff line numberDiff line change
@@ -1,5 +1,5 @@
11
# beacon_chain
2-
# Copyright (c) 2018-2021 Status Research & Development GmbH
2+
# Copyright (c) 2018-2022 Status Research & Development GmbH
33
# Licensed and distributed under either of
44
# * MIT license (license terms in the root directory or at https://opensource.org/licenses/MIT).
55
# * Apache v2 license (license terms in the root directory or at https://www.apache.org/licenses/LICENSE-2.0).
@@ -17,7 +17,7 @@ export codec, base, typetraits
1717

1818
# Coding and decoding of SSZ to spec-specific types
1919

20-
template toSszType*(v: Slot|Epoch): auto = uint64(v)
20+
template toSszType*(v: Slot|Epoch|SyncCommitteePeriod): auto = uint64(v)
2121
template toSszType*(v: BlsCurveType): auto = toRaw(v)
2222
template toSszType*(v: ForkDigest|GraffitiBytes): auto = distinctBase(v)
2323
template toSszType*(v: Version): auto = distinctBase(v)
@@ -34,6 +34,9 @@ template fromSszBytes*(T: type Slot, bytes: openArray[byte]): T =
3434
template fromSszBytes*(T: type Epoch, bytes: openArray[byte]): T =
3535
T fromSszBytes(uint64, bytes)
3636

37+
template fromSszBytes*(T: type SyncCommitteePeriod, bytes: openArray[byte]): T =
38+
T fromSszBytes(uint64, bytes)
39+
3740
func fromSszBytes*(T: type ForkDigest, bytes: openArray[byte]): T {.raisesssz.} =
3841
if bytes.len != sizeof(result):
3942
raiseIncorrectSize T

0 commit comments

Comments
 (0)