les: implement new les fetcher (#20692)
* cmd, consensus, eth, les: implement light fetcher * les: address comment * les: address comment * les: address comments * les: check td after delivery * les: add linearExpiredValue for error counter * les: fix import * les: fix dead lock * les: order announces by td * les: encapsulate invalid counter * les: address comment * les: add more checks during the delivery * les: fix log * eth, les: fix lint * eth/fetcher: address comment
This commit is contained in:
@ -269,7 +269,7 @@ func (s *LightEthereum) EventMux() *event.TypeMux { return s.eventMux
|
||||
// network protocols to start.
|
||||
func (s *LightEthereum) Protocols() []p2p.Protocol {
|
||||
return s.makeProtocols(ClientProtocolVersions, s.handler.runPeer, func(id enode.ID) interface{} {
|
||||
if p := s.peers.peer(peerIdToString(id)); p != nil {
|
||||
if p := s.peers.peer(id.String()); p != nil {
|
||||
return p.Info()
|
||||
}
|
||||
return nil
|
||||
@ -285,6 +285,7 @@ func (s *LightEthereum) Start(srvr *p2p.Server) error {
|
||||
// Start bloom request workers.
|
||||
s.wg.Add(bloomServiceThreads)
|
||||
s.startBloomHandlers(params.BloomBitsBlocksClient)
|
||||
s.handler.start()
|
||||
|
||||
s.netRPCService = ethapi.NewPublicNetAPI(srvr, s.config.NetworkId)
|
||||
return nil
|
||||
|
@ -64,16 +64,20 @@ func newClientHandler(ulcServers []string, ulcFraction int, checkpoint *params.T
|
||||
if checkpoint != nil {
|
||||
height = (checkpoint.SectionIndex+1)*params.CHTFrequency - 1
|
||||
}
|
||||
handler.fetcher = newLightFetcher(handler, backend.serverPool.getTimeout)
|
||||
handler.fetcher = newLightFetcher(backend.blockchain, backend.engine, backend.peers, handler.ulc, backend.chainDb, backend.reqDist, handler.synchronise)
|
||||
handler.downloader = downloader.New(height, backend.chainDb, nil, backend.eventMux, nil, backend.blockchain, handler.removePeer)
|
||||
handler.backend.peers.subscribe((*downloaderPeerNotify)(handler))
|
||||
return handler
|
||||
}
|
||||
|
||||
func (h *clientHandler) start() {
|
||||
h.fetcher.start()
|
||||
}
|
||||
|
||||
func (h *clientHandler) stop() {
|
||||
close(h.closeCh)
|
||||
h.downloader.Terminate()
|
||||
h.fetcher.close()
|
||||
h.fetcher.stop()
|
||||
h.wg.Wait()
|
||||
}
|
||||
|
||||
@ -121,7 +125,6 @@ func (h *clientHandler) handle(p *serverPeer) error {
|
||||
connectionTimer.Update(time.Duration(mclock.Now() - connectedAt))
|
||||
serverConnectionGauge.Update(int64(h.backend.peers.len()))
|
||||
}()
|
||||
|
||||
h.fetcher.announce(p, &announceData{Hash: p.headInfo.Hash, Number: p.headInfo.Number, Td: p.headInfo.Td})
|
||||
|
||||
// Mark the peer starts to be served.
|
||||
@ -185,6 +188,9 @@ func (h *clientHandler) handleMsg(p *serverPeer) error {
|
||||
p.Log().Trace("Valid announcement signature")
|
||||
}
|
||||
p.Log().Trace("Announce message content", "number", req.Number, "hash", req.Hash, "td", req.Td, "reorg", req.ReorgDepth)
|
||||
|
||||
// Update peer head information first and then notify the announcement
|
||||
p.updateHead(req.Hash, req.Number, req.Td)
|
||||
h.fetcher.announce(p, &req)
|
||||
}
|
||||
case BlockHeadersMsg:
|
||||
@ -196,12 +202,17 @@ func (h *clientHandler) handleMsg(p *serverPeer) error {
|
||||
if err := msg.Decode(&resp); err != nil {
|
||||
return errResp(ErrDecode, "msg %v: %v", msg, err)
|
||||
}
|
||||
headers := resp.Headers
|
||||
p.fcServer.ReceivedReply(resp.ReqID, resp.BV)
|
||||
p.answeredRequest(resp.ReqID)
|
||||
if h.fetcher.requestedID(resp.ReqID) {
|
||||
h.fetcher.deliverHeaders(p, resp.ReqID, resp.Headers)
|
||||
} else {
|
||||
if err := h.downloader.DeliverHeaders(p.id, resp.Headers); err != nil {
|
||||
|
||||
// Filter out any explicitly requested headers, deliver the rest to the downloader
|
||||
filter := len(headers) == 1
|
||||
if filter {
|
||||
headers = h.fetcher.deliverHeaders(p, resp.ReqID, resp.Headers)
|
||||
}
|
||||
if len(headers) != 0 || !filter {
|
||||
if err := h.downloader.DeliverHeaders(p.id, headers); err != nil {
|
||||
log.Debug("Failed to deliver headers", "err", err)
|
||||
}
|
||||
}
|
||||
@ -320,8 +331,7 @@ func (h *clientHandler) handleMsg(p *serverPeer) error {
|
||||
// Deliver the received response to retriever.
|
||||
if deliverMsg != nil {
|
||||
if err := h.backend.retriever.deliver(p, deliverMsg); err != nil {
|
||||
p.errCount++
|
||||
if p.errCount > maxResponseErrors {
|
||||
if val := p.errCount.Add(1, mclock.Now()); val > maxResponseErrors {
|
||||
return err
|
||||
}
|
||||
}
|
||||
|
@ -212,7 +212,7 @@ func (f *clientPool) connect(peer clientPoolPeer, capacity uint64) bool {
|
||||
id, freeID := peer.ID(), peer.freeClientId()
|
||||
if _, ok := f.connectedMap[id]; ok {
|
||||
clientRejectedMeter.Mark(1)
|
||||
log.Debug("Client already connected", "address", freeID, "id", peerIdToString(id))
|
||||
log.Debug("Client already connected", "address", freeID, "id", id.String())
|
||||
return false
|
||||
}
|
||||
// Create a clientInfo but do not add it yet
|
||||
@ -277,7 +277,7 @@ func (f *clientPool) connect(peer clientPoolPeer, capacity uint64) bool {
|
||||
f.connectedQueue.Push(c)
|
||||
}
|
||||
clientRejectedMeter.Mark(1)
|
||||
log.Debug("Client rejected", "address", freeID, "id", peerIdToString(id))
|
||||
log.Debug("Client rejected", "address", freeID, "id", id.String())
|
||||
return false
|
||||
}
|
||||
// accept new client, drop old ones
|
||||
@ -322,7 +322,7 @@ func (f *clientPool) disconnect(p clientPoolPeer) {
|
||||
// Short circuit if the peer hasn't been registered.
|
||||
e := f.connectedMap[p.ID()]
|
||||
if e == nil {
|
||||
log.Debug("Client not connected", "address", p.freeClientId(), "id", peerIdToString(p.ID()))
|
||||
log.Debug("Client not connected", "address", p.freeClientId(), "id", p.ID().String())
|
||||
return
|
||||
}
|
||||
f.dropClient(e, f.clock.Now(), false)
|
||||
|
1301
les/fetcher.go
1301
les/fetcher.go
File diff suppressed because it is too large
Load Diff
268
les/fetcher_test.go
Normal file
268
les/fetcher_test.go
Normal file
@ -0,0 +1,268 @@
|
||||
// Copyright 2020 The go-ethereum Authors
|
||||
// This file is part of the go-ethereum library.
|
||||
//
|
||||
// The go-ethereum library is free software: you can redistribute it and/or modify
|
||||
// it under the terms of the GNU Lesser General Public License as published by
|
||||
// the Free Software Foundation, either version 3 of the License, or
|
||||
// (at your option) any later version.
|
||||
//
|
||||
// The go-ethereum library is distributed in the hope that it will be useful,
|
||||
// but WITHOUT ANY WARRANTY; without even the implied warranty of
|
||||
// MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
|
||||
// GNU Lesser General Public License for more details.
|
||||
//
|
||||
// You should have received a copy of the GNU Lesser General Public License
|
||||
// along with the go-ethereum library. If not, see <http://www.gnu.org/licenses/>.
|
||||
|
||||
package les
|
||||
|
||||
import (
|
||||
"math/big"
|
||||
"testing"
|
||||
"time"
|
||||
|
||||
"github.com/ethereum/go-ethereum/consensus/ethash"
|
||||
"github.com/ethereum/go-ethereum/core"
|
||||
"github.com/ethereum/go-ethereum/core/rawdb"
|
||||
"github.com/ethereum/go-ethereum/core/types"
|
||||
"github.com/ethereum/go-ethereum/p2p/enode"
|
||||
)
|
||||
|
||||
// verifyImportEvent verifies that one single event arrive on an import channel.
|
||||
func verifyImportEvent(t *testing.T, imported chan interface{}, arrive bool) {
|
||||
if arrive {
|
||||
select {
|
||||
case <-imported:
|
||||
case <-time.After(time.Second):
|
||||
t.Fatalf("import timeout")
|
||||
}
|
||||
} else {
|
||||
select {
|
||||
case <-imported:
|
||||
t.Fatalf("import invoked")
|
||||
case <-time.After(20 * time.Millisecond):
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
// verifyImportDone verifies that no more events are arriving on an import channel.
|
||||
func verifyImportDone(t *testing.T, imported chan interface{}) {
|
||||
select {
|
||||
case <-imported:
|
||||
t.Fatalf("extra block imported")
|
||||
case <-time.After(50 * time.Millisecond):
|
||||
}
|
||||
}
|
||||
|
||||
// verifyChainHeight verifies the chain height is as expected.
|
||||
func verifyChainHeight(t *testing.T, fetcher *lightFetcher, height uint64) {
|
||||
local := fetcher.chain.CurrentHeader().Number.Uint64()
|
||||
if local != height {
|
||||
t.Fatalf("chain height mismatch, got %d, want %d", local, height)
|
||||
}
|
||||
}
|
||||
|
||||
func TestSequentialAnnouncementsLes2(t *testing.T) { testSequentialAnnouncements(t, 2) }
|
||||
func TestSequentialAnnouncementsLes3(t *testing.T) { testSequentialAnnouncements(t, 3) }
|
||||
|
||||
func testSequentialAnnouncements(t *testing.T, protocol int) {
|
||||
s, c, teardown := newClientServerEnv(t, 4, protocol, nil, nil, 0, false, false)
|
||||
defer teardown()
|
||||
|
||||
// Create connected peer pair.
|
||||
c.handler.fetcher.noAnnounce = true // Ignore the first announce from peer which can trigger a resync.
|
||||
p1, _, err := newTestPeerPair("peer", protocol, s.handler, c.handler)
|
||||
if err != nil {
|
||||
t.Fatalf("Failed to create peer pair %v", err)
|
||||
}
|
||||
c.handler.fetcher.noAnnounce = false
|
||||
|
||||
importCh := make(chan interface{})
|
||||
c.handler.fetcher.newHeadHook = func(header *types.Header) {
|
||||
importCh <- header
|
||||
}
|
||||
for i := uint64(1); i <= s.backend.Blockchain().CurrentHeader().Number.Uint64(); i++ {
|
||||
header := s.backend.Blockchain().GetHeaderByNumber(i)
|
||||
hash, number := header.Hash(), header.Number.Uint64()
|
||||
td := rawdb.ReadTd(s.db, hash, number)
|
||||
|
||||
announce := announceData{hash, number, td, 0, nil}
|
||||
if p1.cpeer.announceType == announceTypeSigned {
|
||||
announce.sign(s.handler.server.privateKey)
|
||||
}
|
||||
p1.cpeer.sendAnnounce(announce)
|
||||
verifyImportEvent(t, importCh, true)
|
||||
}
|
||||
verifyImportDone(t, importCh)
|
||||
verifyChainHeight(t, c.handler.fetcher, 4)
|
||||
}
|
||||
|
||||
func TestGappedAnnouncementsLes2(t *testing.T) { testGappedAnnouncements(t, 2) }
|
||||
func TestGappedAnnouncementsLes3(t *testing.T) { testGappedAnnouncements(t, 3) }
|
||||
|
||||
func testGappedAnnouncements(t *testing.T, protocol int) {
|
||||
s, c, teardown := newClientServerEnv(t, 4, protocol, nil, nil, 0, false, false)
|
||||
defer teardown()
|
||||
|
||||
// Create connected peer pair.
|
||||
c.handler.fetcher.noAnnounce = true // Ignore the first announce from peer which can trigger a resync.
|
||||
peer, _, err := newTestPeerPair("peer", protocol, s.handler, c.handler)
|
||||
if err != nil {
|
||||
t.Fatalf("Failed to create peer pair %v", err)
|
||||
}
|
||||
c.handler.fetcher.noAnnounce = false
|
||||
|
||||
done := make(chan *types.Header, 1)
|
||||
c.handler.fetcher.newHeadHook = func(header *types.Header) { done <- header }
|
||||
|
||||
// Prepare announcement by latest header.
|
||||
latest := s.backend.Blockchain().CurrentHeader()
|
||||
hash, number := latest.Hash(), latest.Number.Uint64()
|
||||
td := rawdb.ReadTd(s.db, hash, number)
|
||||
|
||||
// Sign the announcement if necessary.
|
||||
announce := announceData{hash, number, td, 0, nil}
|
||||
if peer.cpeer.announceType == announceTypeSigned {
|
||||
announce.sign(s.handler.server.privateKey)
|
||||
}
|
||||
peer.cpeer.sendAnnounce(announce)
|
||||
|
||||
<-done // Wait syncing
|
||||
verifyChainHeight(t, c.handler.fetcher, 4)
|
||||
|
||||
// Send a reorged announcement
|
||||
var newAnno = make(chan struct{}, 1)
|
||||
c.handler.fetcher.noAnnounce = true
|
||||
c.handler.fetcher.newAnnounce = func(*serverPeer, *announceData) {
|
||||
newAnno <- struct{}{}
|
||||
}
|
||||
blocks, _ := core.GenerateChain(rawdb.ReadChainConfig(s.db, s.backend.Blockchain().Genesis().Hash()), s.backend.Blockchain().GetBlockByNumber(3),
|
||||
ethash.NewFaker(), s.db, 2, func(i int, gen *core.BlockGen) {
|
||||
gen.OffsetTime(-9) // higher block difficulty
|
||||
})
|
||||
s.backend.Blockchain().InsertChain(blocks)
|
||||
<-newAnno
|
||||
c.handler.fetcher.noAnnounce = false
|
||||
c.handler.fetcher.newAnnounce = nil
|
||||
|
||||
latest = blocks[len(blocks)-1].Header()
|
||||
hash, number = latest.Hash(), latest.Number.Uint64()
|
||||
td = rawdb.ReadTd(s.db, hash, number)
|
||||
|
||||
announce = announceData{hash, number, td, 1, nil}
|
||||
if peer.cpeer.announceType == announceTypeSigned {
|
||||
announce.sign(s.handler.server.privateKey)
|
||||
}
|
||||
peer.cpeer.sendAnnounce(announce)
|
||||
|
||||
<-done // Wait syncing
|
||||
verifyChainHeight(t, c.handler.fetcher, 5)
|
||||
}
|
||||
|
||||
func TestTrustedAnnouncementsLes2(t *testing.T) { testTrustedAnnouncement(t, 2) }
|
||||
func TestTrustedAnnouncementsLes3(t *testing.T) { testTrustedAnnouncement(t, 3) }
|
||||
|
||||
func testTrustedAnnouncement(t *testing.T, protocol int) {
|
||||
var (
|
||||
servers []*testServer
|
||||
teardowns []func()
|
||||
nodes []*enode.Node
|
||||
ids []string
|
||||
cpeers []*clientPeer
|
||||
speers []*serverPeer
|
||||
)
|
||||
for i := 0; i < 10; i++ {
|
||||
s, n, teardown := newTestServerPeer(t, 10, protocol)
|
||||
|
||||
servers = append(servers, s)
|
||||
nodes = append(nodes, n)
|
||||
teardowns = append(teardowns, teardown)
|
||||
|
||||
// A half of them are trusted servers.
|
||||
if i < 5 {
|
||||
ids = append(ids, n.String())
|
||||
}
|
||||
}
|
||||
_, c, teardown := newClientServerEnv(t, 0, protocol, nil, ids, 60, false, false)
|
||||
defer teardown()
|
||||
defer func() {
|
||||
for i := 0; i < len(teardowns); i++ {
|
||||
teardowns[i]()
|
||||
}
|
||||
}()
|
||||
|
||||
c.handler.fetcher.noAnnounce = true // Ignore the first announce from peer which can trigger a resync.
|
||||
|
||||
// Connect all server instances.
|
||||
for i := 0; i < len(servers); i++ {
|
||||
sp, cp, err := connect(servers[i].handler, nodes[i].ID(), c.handler, protocol)
|
||||
if err != nil {
|
||||
t.Fatalf("connect server and client failed, err %s", err)
|
||||
}
|
||||
cpeers = append(cpeers, cp)
|
||||
speers = append(speers, sp)
|
||||
}
|
||||
c.handler.fetcher.noAnnounce = false
|
||||
|
||||
newHead := make(chan *types.Header, 1)
|
||||
c.handler.fetcher.newHeadHook = func(header *types.Header) { newHead <- header }
|
||||
|
||||
check := func(height []uint64, expected uint64, callback func()) {
|
||||
for i := 0; i < len(height); i++ {
|
||||
for j := 0; j < len(servers); j++ {
|
||||
h := servers[j].backend.Blockchain().GetHeaderByNumber(height[i])
|
||||
hash, number := h.Hash(), h.Number.Uint64()
|
||||
td := rawdb.ReadTd(servers[j].db, hash, number)
|
||||
|
||||
// Sign the announcement if necessary.
|
||||
announce := announceData{hash, number, td, 0, nil}
|
||||
p := cpeers[j]
|
||||
if p.announceType == announceTypeSigned {
|
||||
announce.sign(servers[j].handler.server.privateKey)
|
||||
}
|
||||
p.sendAnnounce(announce)
|
||||
}
|
||||
}
|
||||
if callback != nil {
|
||||
callback()
|
||||
}
|
||||
verifyChainHeight(t, c.handler.fetcher, expected)
|
||||
}
|
||||
check([]uint64{1}, 1, func() { <-newHead }) // Sequential announcements
|
||||
check([]uint64{4}, 4, func() { <-newHead }) // ULC-style light syncing, rollback untrusted headers
|
||||
check([]uint64{10}, 10, func() { <-newHead }) // Sync the whole chain.
|
||||
}
|
||||
|
||||
func TestInvalidAnnounces(t *testing.T) {
|
||||
s, c, teardown := newClientServerEnv(t, 4, lpv3, nil, nil, 0, false, false)
|
||||
defer teardown()
|
||||
|
||||
// Create connected peer pair.
|
||||
c.handler.fetcher.noAnnounce = true // Ignore the first announce from peer which can trigger a resync.
|
||||
peer, _, err := newTestPeerPair("peer", lpv3, s.handler, c.handler)
|
||||
if err != nil {
|
||||
t.Fatalf("Failed to create peer pair %v", err)
|
||||
}
|
||||
c.handler.fetcher.noAnnounce = false
|
||||
|
||||
done := make(chan *types.Header, 1)
|
||||
c.handler.fetcher.newHeadHook = func(header *types.Header) { done <- header }
|
||||
|
||||
// Prepare announcement by latest header.
|
||||
headerOne := s.backend.Blockchain().GetHeaderByNumber(1)
|
||||
hash, number := headerOne.Hash(), headerOne.Number.Uint64()
|
||||
td := big.NewInt(200) // bad td
|
||||
|
||||
// Sign the announcement if necessary.
|
||||
announce := announceData{hash, number, td, 0, nil}
|
||||
if peer.cpeer.announceType == announceTypeSigned {
|
||||
announce.sign(s.handler.server.privateKey)
|
||||
}
|
||||
peer.cpeer.sendAnnounce(announce)
|
||||
<-done // Wait syncing
|
||||
|
||||
// Ensure the bad peer is evicited
|
||||
if c.handler.backend.peers.len() != 0 {
|
||||
t.Fatalf("Failed to evict invalid peer")
|
||||
}
|
||||
}
|
@ -222,13 +222,13 @@ func testOdr(t *testing.T, protocol int, expFail uint64, checkCached bool, fn od
|
||||
|
||||
// expect retrievals to fail (except genesis block) without a les peer
|
||||
client.handler.backend.peers.lock.Lock()
|
||||
client.peer.speer.hasBlock = func(common.Hash, uint64, bool) bool { return false }
|
||||
client.peer.speer.hasBlockHook = func(common.Hash, uint64, bool) bool { return false }
|
||||
client.handler.backend.peers.lock.Unlock()
|
||||
test(expFail)
|
||||
|
||||
// expect all retrievals to pass
|
||||
client.handler.backend.peers.lock.Lock()
|
||||
client.peer.speer.hasBlock = func(common.Hash, uint64, bool) bool { return true }
|
||||
client.peer.speer.hasBlockHook = func(common.Hash, uint64, bool) bool { return true }
|
||||
client.handler.backend.peers.lock.Unlock()
|
||||
test(5)
|
||||
|
||||
|
64
les/peer.go
64
les/peer.go
@ -36,7 +36,6 @@ import (
|
||||
"github.com/ethereum/go-ethereum/les/utils"
|
||||
"github.com/ethereum/go-ethereum/light"
|
||||
"github.com/ethereum/go-ethereum/p2p"
|
||||
"github.com/ethereum/go-ethereum/p2p/enode"
|
||||
"github.com/ethereum/go-ethereum/params"
|
||||
"github.com/ethereum/go-ethereum/rlp"
|
||||
)
|
||||
@ -115,11 +114,6 @@ func (m keyValueMap) get(key string, val interface{}) error {
|
||||
return rlp.DecodeBytes(enc, val)
|
||||
}
|
||||
|
||||
// peerIdToString converts enode.ID to a string form
|
||||
func peerIdToString(id enode.ID) string {
|
||||
return fmt.Sprintf("%x", id.Bytes())
|
||||
}
|
||||
|
||||
// peerCommons contains fields needed by both server peer and client peer.
|
||||
type peerCommons struct {
|
||||
*p2p.Peer
|
||||
@ -343,12 +337,12 @@ type serverPeer struct {
|
||||
sentReqs map[uint64]sentReqEntry
|
||||
|
||||
// Statistics
|
||||
errCount int // Counter the invalid responses server has replied
|
||||
errCount utils.LinearExpiredValue // Counter the invalid responses server has replied
|
||||
updateCount uint64
|
||||
updateTime mclock.AbsTime
|
||||
|
||||
// Callbacks
|
||||
hasBlock func(common.Hash, uint64, bool) bool // Used to determine whether the server has the specified block.
|
||||
// Test callback hooks
|
||||
hasBlockHook func(common.Hash, uint64, bool) bool // Used to determine whether the server has the specified block.
|
||||
}
|
||||
|
||||
func newServerPeer(version int, network uint64, trusted bool, p *p2p.Peer, rw p2p.MsgReadWriter) *serverPeer {
|
||||
@ -356,13 +350,14 @@ func newServerPeer(version int, network uint64, trusted bool, p *p2p.Peer, rw p2
|
||||
peerCommons: peerCommons{
|
||||
Peer: p,
|
||||
rw: rw,
|
||||
id: peerIdToString(p.ID()),
|
||||
id: p.ID().String(),
|
||||
version: version,
|
||||
network: network,
|
||||
sendQueue: utils.NewExecQueue(100),
|
||||
closeCh: make(chan struct{}),
|
||||
},
|
||||
trusted: trusted,
|
||||
trusted: trusted,
|
||||
errCount: utils.LinearExpiredValue{Rate: mclock.AbsTime(time.Hour)},
|
||||
}
|
||||
}
|
||||
|
||||
@ -524,7 +519,11 @@ func (p *serverPeer) getTxRelayCost(amount, size int) uint64 {
|
||||
// HasBlock checks if the peer has a given block
|
||||
func (p *serverPeer) HasBlock(hash common.Hash, number uint64, hasState bool) bool {
|
||||
p.lock.RLock()
|
||||
defer p.lock.RUnlock()
|
||||
|
||||
if p.hasBlockHook != nil {
|
||||
return p.hasBlockHook(hash, number, hasState)
|
||||
}
|
||||
head := p.headInfo.Number
|
||||
var since, recent uint64
|
||||
if hasState {
|
||||
@ -534,10 +533,7 @@ func (p *serverPeer) HasBlock(hash common.Hash, number uint64, hasState bool) bo
|
||||
since = p.chainSince
|
||||
recent = p.chainRecent
|
||||
}
|
||||
hasBlock := p.hasBlock
|
||||
p.lock.RUnlock()
|
||||
|
||||
return head >= number && number >= since && (recent == 0 || number+recent+4 > head) && hasBlock != nil && hasBlock(hash, number, hasState)
|
||||
return head >= number && number >= since && (recent == 0 || number+recent+4 > head)
|
||||
}
|
||||
|
||||
// updateFlowControl updates the flow control parameters belonging to the server
|
||||
@ -562,6 +558,15 @@ func (p *serverPeer) updateFlowControl(update keyValueMap) {
|
||||
}
|
||||
}
|
||||
|
||||
// updateHead updates the head information based on the announcement from
|
||||
// the peer.
|
||||
func (p *serverPeer) updateHead(hash common.Hash, number uint64, td *big.Int) {
|
||||
p.lock.Lock()
|
||||
defer p.lock.Unlock()
|
||||
|
||||
p.headInfo = blockInfo{Hash: hash, Number: number, Td: td}
|
||||
}
|
||||
|
||||
// Handshake executes the les protocol handshake, negotiating version number,
|
||||
// network IDs, difficulties, head and genesis blocks.
|
||||
func (p *serverPeer) Handshake(td *big.Int, head common.Hash, headNum uint64, genesis common.Hash, server *LesServer) error {
|
||||
@ -712,11 +717,15 @@ type clientPeer struct {
|
||||
// responseLock ensures that responses are queued in the same order as
|
||||
// RequestProcessed is called
|
||||
responseLock sync.Mutex
|
||||
server bool
|
||||
invalidCount uint32 // Counter the invalid request the client peer has made.
|
||||
responseCount uint64 // Counter to generate an unique id for request processing.
|
||||
errCh chan error
|
||||
fcClient *flowcontrol.ClientNode // Server side mirror token bucket.
|
||||
|
||||
// invalidLock is used for protecting invalidCount.
|
||||
invalidLock sync.RWMutex
|
||||
invalidCount utils.LinearExpiredValue // Counter the invalid request the client peer has made.
|
||||
|
||||
server bool
|
||||
errCh chan error
|
||||
fcClient *flowcontrol.ClientNode // Server side mirror token bucket.
|
||||
}
|
||||
|
||||
func newClientPeer(version int, network uint64, p *p2p.Peer, rw p2p.MsgReadWriter) *clientPeer {
|
||||
@ -724,13 +733,14 @@ func newClientPeer(version int, network uint64, p *p2p.Peer, rw p2p.MsgReadWrite
|
||||
peerCommons: peerCommons{
|
||||
Peer: p,
|
||||
rw: rw,
|
||||
id: peerIdToString(p.ID()),
|
||||
id: p.ID().String(),
|
||||
version: version,
|
||||
network: network,
|
||||
sendQueue: utils.NewExecQueue(100),
|
||||
closeCh: make(chan struct{}),
|
||||
},
|
||||
errCh: make(chan error, 1),
|
||||
invalidCount: utils.LinearExpiredValue{Rate: mclock.AbsTime(time.Hour)},
|
||||
errCh: make(chan error, 1),
|
||||
}
|
||||
}
|
||||
|
||||
@ -970,6 +980,18 @@ func (p *clientPeer) Handshake(td *big.Int, head common.Hash, headNum uint64, ge
|
||||
})
|
||||
}
|
||||
|
||||
func (p *clientPeer) bumpInvalid() {
|
||||
p.invalidLock.Lock()
|
||||
p.invalidCount.Add(1, mclock.Now())
|
||||
p.invalidLock.Unlock()
|
||||
}
|
||||
|
||||
func (p *clientPeer) getInvalid() uint64 {
|
||||
p.invalidLock.RLock()
|
||||
defer p.invalidLock.RUnlock()
|
||||
return p.invalidCount.Value(mclock.Now())
|
||||
}
|
||||
|
||||
// serverPeerSubscriber is an interface to notify services about added or
|
||||
// removed server peers
|
||||
type serverPeerSubscriber interface {
|
||||
|
@ -116,7 +116,7 @@ func NewLesServer(e *eth.Ethereum, config *eth.Config) (*LesServer, error) {
|
||||
srv.maxCapacity = totalRecharge
|
||||
}
|
||||
srv.fcManager.SetCapacityLimits(srv.freeCapacity, srv.maxCapacity, srv.freeCapacity*2)
|
||||
srv.clientPool = newClientPool(srv.chainDb, srv.freeCapacity, mclock.System{}, func(id enode.ID) { go srv.peers.unregister(peerIdToString(id)) })
|
||||
srv.clientPool = newClientPool(srv.chainDb, srv.freeCapacity, mclock.System{}, func(id enode.ID) { go srv.peers.unregister(id.String()) })
|
||||
srv.clientPool.setDefaultFactors(priceFactors{0, 1, 1}, priceFactors{0, 1, 1})
|
||||
|
||||
checkpoint := srv.latestLocalCheckpoint()
|
||||
@ -153,7 +153,7 @@ func (s *LesServer) APIs() []rpc.API {
|
||||
|
||||
func (s *LesServer) Protocols() []p2p.Protocol {
|
||||
ps := s.makeProtocols(ServerProtocolVersions, s.handler.runPeer, func(id enode.ID) interface{} {
|
||||
if p := s.peers.peer(peerIdToString(id)); p != nil {
|
||||
if p := s.peers.peer(id.String()); p != nil {
|
||||
return p.Info()
|
||||
}
|
||||
return nil
|
||||
|
@ -322,7 +322,7 @@ func (h *serverHandler) handleMsg(p *clientPeer, wg *sync.WaitGroup) error {
|
||||
origin = h.blockchain.GetHeaderByNumber(query.Origin.Number)
|
||||
}
|
||||
if origin == nil {
|
||||
atomic.AddUint32(&p.invalidCount, 1)
|
||||
p.bumpInvalid()
|
||||
break
|
||||
}
|
||||
headers = append(headers, origin)
|
||||
@ -419,7 +419,7 @@ func (h *serverHandler) handleMsg(p *clientPeer, wg *sync.WaitGroup) error {
|
||||
}
|
||||
body := h.blockchain.GetBodyRLP(hash)
|
||||
if body == nil {
|
||||
atomic.AddUint32(&p.invalidCount, 1)
|
||||
p.bumpInvalid()
|
||||
continue
|
||||
}
|
||||
bodies = append(bodies, body)
|
||||
@ -467,7 +467,7 @@ func (h *serverHandler) handleMsg(p *clientPeer, wg *sync.WaitGroup) error {
|
||||
header := h.blockchain.GetHeaderByHash(request.BHash)
|
||||
if header == nil {
|
||||
p.Log().Warn("Failed to retrieve associate header for code", "hash", request.BHash)
|
||||
atomic.AddUint32(&p.invalidCount, 1)
|
||||
p.bumpInvalid()
|
||||
continue
|
||||
}
|
||||
// Refuse to search stale state data in the database since looking for
|
||||
@ -475,7 +475,7 @@ func (h *serverHandler) handleMsg(p *clientPeer, wg *sync.WaitGroup) error {
|
||||
local := h.blockchain.CurrentHeader().Number.Uint64()
|
||||
if !h.server.archiveMode && header.Number.Uint64()+core.TriesInMemory <= local {
|
||||
p.Log().Debug("Reject stale code request", "number", header.Number.Uint64(), "head", local)
|
||||
atomic.AddUint32(&p.invalidCount, 1)
|
||||
p.bumpInvalid()
|
||||
continue
|
||||
}
|
||||
triedb := h.blockchain.StateCache().TrieDB()
|
||||
@ -483,7 +483,7 @@ func (h *serverHandler) handleMsg(p *clientPeer, wg *sync.WaitGroup) error {
|
||||
account, err := h.getAccount(triedb, header.Root, common.BytesToHash(request.AccKey))
|
||||
if err != nil {
|
||||
p.Log().Warn("Failed to retrieve account for code", "block", header.Number, "hash", header.Hash(), "account", common.BytesToHash(request.AccKey), "err", err)
|
||||
atomic.AddUint32(&p.invalidCount, 1)
|
||||
p.bumpInvalid()
|
||||
continue
|
||||
}
|
||||
code, err := triedb.Node(common.BytesToHash(account.CodeHash))
|
||||
@ -542,7 +542,7 @@ func (h *serverHandler) handleMsg(p *clientPeer, wg *sync.WaitGroup) error {
|
||||
results := h.blockchain.GetReceiptsByHash(hash)
|
||||
if results == nil {
|
||||
if header := h.blockchain.GetHeaderByHash(hash); header == nil || header.ReceiptHash != types.EmptyRootHash {
|
||||
atomic.AddUint32(&p.invalidCount, 1)
|
||||
p.bumpInvalid()
|
||||
continue
|
||||
}
|
||||
}
|
||||
@ -605,7 +605,7 @@ func (h *serverHandler) handleMsg(p *clientPeer, wg *sync.WaitGroup) error {
|
||||
|
||||
if header = h.blockchain.GetHeaderByHash(request.BHash); header == nil {
|
||||
p.Log().Warn("Failed to retrieve header for proof", "hash", request.BHash)
|
||||
atomic.AddUint32(&p.invalidCount, 1)
|
||||
p.bumpInvalid()
|
||||
continue
|
||||
}
|
||||
// Refuse to search stale state data in the database since looking for
|
||||
@ -613,14 +613,14 @@ func (h *serverHandler) handleMsg(p *clientPeer, wg *sync.WaitGroup) error {
|
||||
local := h.blockchain.CurrentHeader().Number.Uint64()
|
||||
if !h.server.archiveMode && header.Number.Uint64()+core.TriesInMemory <= local {
|
||||
p.Log().Debug("Reject stale trie request", "number", header.Number.Uint64(), "head", local)
|
||||
atomic.AddUint32(&p.invalidCount, 1)
|
||||
p.bumpInvalid()
|
||||
continue
|
||||
}
|
||||
root = header.Root
|
||||
}
|
||||
// If a header lookup failed (non existent), ignore subsequent requests for the same header
|
||||
if root == (common.Hash{}) {
|
||||
atomic.AddUint32(&p.invalidCount, 1)
|
||||
p.bumpInvalid()
|
||||
continue
|
||||
}
|
||||
// Open the account or storage trie for the request
|
||||
@ -639,7 +639,7 @@ func (h *serverHandler) handleMsg(p *clientPeer, wg *sync.WaitGroup) error {
|
||||
account, err := h.getAccount(statedb.TrieDB(), root, common.BytesToHash(request.AccKey))
|
||||
if err != nil {
|
||||
p.Log().Warn("Failed to retrieve account for proof", "block", header.Number, "hash", header.Hash(), "account", common.BytesToHash(request.AccKey), "err", err)
|
||||
atomic.AddUint32(&p.invalidCount, 1)
|
||||
p.bumpInvalid()
|
||||
continue
|
||||
}
|
||||
trie, err = statedb.OpenStorageTrie(common.BytesToHash(request.AccKey), account.Root)
|
||||
@ -833,9 +833,9 @@ func (h *serverHandler) handleMsg(p *clientPeer, wg *sync.WaitGroup) error {
|
||||
clientErrorMeter.Mark(1)
|
||||
return errResp(ErrInvalidMsgCode, "%v", msg.Code)
|
||||
}
|
||||
// If the client has made too much invalid request(e.g. request a non-exist data),
|
||||
// If the client has made too much invalid request(e.g. request a non-existent data),
|
||||
// reject them to prevent SPAM attack.
|
||||
if atomic.LoadUint32(&p.invalidCount) > maxRequestErrors {
|
||||
if p.getInvalid() > maxRequestErrors {
|
||||
clientErrorMeter.Mark(1)
|
||||
return errTooManyInvalidRequest
|
||||
}
|
||||
|
@ -223,6 +223,7 @@ func newTestClientHandler(backend *backends.SimulatedBackend, odr *LesOdr, index
|
||||
if client.oracle != nil {
|
||||
client.oracle.Start(backend)
|
||||
}
|
||||
client.handler.start()
|
||||
return client.handler
|
||||
}
|
||||
|
||||
|
@ -124,6 +124,50 @@ func (e *ExpiredValue) SubExp(a ExpiredValue) {
|
||||
}
|
||||
}
|
||||
|
||||
// LinearExpiredValue is very similar with the expiredValue which the value
|
||||
// will continuously expired. But the different part is it's expired linearly.
|
||||
type LinearExpiredValue struct {
|
||||
Offset uint64 // The latest time offset
|
||||
Val uint64 // The remaining value, can never be negative
|
||||
Rate mclock.AbsTime `rlp:"-"` // Expiration rate(by nanosecond), will ignored by RLP
|
||||
}
|
||||
|
||||
// value calculates the value at the given moment. This function always has the
|
||||
// assumption that the given timestamp shouldn't less than the recorded one.
|
||||
func (e LinearExpiredValue) Value(now mclock.AbsTime) uint64 {
|
||||
offset := uint64(now / e.Rate)
|
||||
if e.Offset < offset {
|
||||
diff := offset - e.Offset
|
||||
if e.Val >= diff {
|
||||
e.Val -= diff
|
||||
} else {
|
||||
e.Val = 0
|
||||
}
|
||||
}
|
||||
return e.Val
|
||||
}
|
||||
|
||||
// add adds a signed value at the given moment. This function always has the
|
||||
// assumption that the given timestamp shouldn't less than the recorded one.
|
||||
func (e *LinearExpiredValue) Add(amount int64, now mclock.AbsTime) uint64 {
|
||||
offset := uint64(now / e.Rate)
|
||||
if e.Offset < offset {
|
||||
diff := offset - e.Offset
|
||||
if e.Val >= diff {
|
||||
e.Val -= diff
|
||||
} else {
|
||||
e.Val = 0
|
||||
}
|
||||
e.Offset = offset
|
||||
}
|
||||
if amount < 0 && uint64(-amount) > e.Val {
|
||||
e.Val = 0
|
||||
} else {
|
||||
e.Val = uint64(int64(e.Val) + amount)
|
||||
}
|
||||
return e.Val
|
||||
}
|
||||
|
||||
// Expirer changes logOffset with a linear rate which can be changed during operation.
|
||||
// It is not thread safe, if access by multiple goroutines is needed then it should be
|
||||
// encapsulated into a locked structure.
|
||||
|
@ -18,6 +18,8 @@ package utils
|
||||
|
||||
import (
|
||||
"testing"
|
||||
|
||||
"github.com/ethereum/go-ethereum/common/mclock"
|
||||
)
|
||||
|
||||
func TestValueExpiration(t *testing.T) {
|
||||
@ -116,3 +118,78 @@ func TestExpiredValueSubtraction(t *testing.T) {
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
func TestLinearExpiredValue(t *testing.T) {
|
||||
var cases = []struct {
|
||||
value LinearExpiredValue
|
||||
now mclock.AbsTime
|
||||
expect uint64
|
||||
}{
|
||||
{LinearExpiredValue{
|
||||
Offset: 0,
|
||||
Val: 0,
|
||||
Rate: mclock.AbsTime(1),
|
||||
}, 0, 0},
|
||||
|
||||
{LinearExpiredValue{
|
||||
Offset: 1,
|
||||
Val: 1,
|
||||
Rate: mclock.AbsTime(1),
|
||||
}, 0, 1},
|
||||
|
||||
{LinearExpiredValue{
|
||||
Offset: 1,
|
||||
Val: 1,
|
||||
Rate: mclock.AbsTime(1),
|
||||
}, mclock.AbsTime(2), 0},
|
||||
|
||||
{LinearExpiredValue{
|
||||
Offset: 1,
|
||||
Val: 1,
|
||||
Rate: mclock.AbsTime(1),
|
||||
}, mclock.AbsTime(3), 0},
|
||||
}
|
||||
for _, c := range cases {
|
||||
if value := c.value.Value(c.now); value != c.expect {
|
||||
t.Fatalf("Value mismatch, want=%d, got=%d", c.expect, value)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
func TestLinearExpiredAddition(t *testing.T) {
|
||||
var cases = []struct {
|
||||
value LinearExpiredValue
|
||||
amount int64
|
||||
now mclock.AbsTime
|
||||
expect uint64
|
||||
}{
|
||||
{LinearExpiredValue{
|
||||
Offset: 0,
|
||||
Val: 0,
|
||||
Rate: mclock.AbsTime(1),
|
||||
}, -1, 0, 0},
|
||||
|
||||
{LinearExpiredValue{
|
||||
Offset: 1,
|
||||
Val: 1,
|
||||
Rate: mclock.AbsTime(1),
|
||||
}, -1, 0, 0},
|
||||
|
||||
{LinearExpiredValue{
|
||||
Offset: 1,
|
||||
Val: 2,
|
||||
Rate: mclock.AbsTime(1),
|
||||
}, -1, mclock.AbsTime(2), 0},
|
||||
|
||||
{LinearExpiredValue{
|
||||
Offset: 1,
|
||||
Val: 2,
|
||||
Rate: mclock.AbsTime(1),
|
||||
}, -2, mclock.AbsTime(2), 0},
|
||||
}
|
||||
for _, c := range cases {
|
||||
if value := c.value.Add(c.amount, c.now); value != c.expect {
|
||||
t.Fatalf("Value mismatch, want=%d, got=%d", c.expect, value)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
Reference in New Issue
Block a user