Improve make() capacity allocations (#6517)

* attestationutil.AttestingIndices: Minor improvement on indices array allocation
* progress
* more progress on makes
* progress
* more progress
* Merge branch 'master' of github.com:prysmaticlabs/prysm into memory1
* gaz
* fmt and imports
* Merge branch 'master' into memory2
* Min()
* remove spaces
* Merge branch 'master' of github.com:prysmaticlabs/prysm into memory2
* revert beacon-chain/operations/attestations/kv/block.go
* partially revert beacon-chain/operations/attestations/kv/aggregated.go
* Merge branch 'master' into memory2
* Merge branch 'master' into memory2
This commit is contained in:
Preston Van Loon
2020-07-09 08:50:58 -07:00
committed by GitHub
parent c804347fc4
commit fd80f73286
26 changed files with 57 additions and 42 deletions

View File

@@ -290,14 +290,17 @@ func (s *Service) cacheJustifiedStateBalances(ctx context.Context, justifiedRoot
}
epoch := helpers.CurrentEpoch(justifiedState)
validators := justifiedState.Validators()
justifiedBalances := make([]uint64, len(validators))
for i, validator := range validators {
if helpers.IsActiveValidator(validator, epoch) {
justifiedBalances[i] = validator.EffectiveBalance
justifiedBalances := make([]uint64, justifiedState.NumValidators())
if err := justifiedState.ReadFromEveryValidator(func(idx int, val *stateTrie.ReadOnlyValidator) error {
if helpers.IsActiveValidatorUsingTrie(val, epoch) {
justifiedBalances[idx] = val.EffectiveBalance()
} else {
justifiedBalances[i] = 0
justifiedBalances[idx] = 0
}
return nil
}); err != nil {
return err
}
s.justifiedBalancesLock.Lock()

View File

@@ -351,14 +351,12 @@ func unslashedAttestingIndices(state *stateTrie.BeaconState, atts []*pb.PendingA
}
attestingIndices := attestationutil.AttestingIndices(att.AggregationBits, committee)
// Create a set for attesting indices
set := make([]uint64, 0, len(attestingIndices))
for _, index := range attestingIndices {
if !seen[index] {
set = append(set, index)
setIndices = append(setIndices, index)
}
seen[index] = true
}
setIndices = append(setIndices, set...)
}
// Sort the attesting set indices by increasing order.
sort.Slice(setIndices, func(i, j int) bool { return setIndices[i] < setIndices[j] })

View File

@@ -174,7 +174,7 @@ func CommitteeAssignments(
// Some validators may need to propose multiple times per epoch, so
// we use a map of proposer idx -> []slot to keep track of this possibility.
startSlot := StartSlot(epoch)
proposerIndexToSlots := make(map[uint64][]uint64)
proposerIndexToSlots := make(map[uint64][]uint64, params.BeaconConfig().SlotsPerEpoch)
for slot := startSlot; slot < startSlot+params.BeaconConfig().SlotsPerEpoch; slot++ {
// Skip proposer assignment for genesis slot.
if slot == 0 {
@@ -197,7 +197,7 @@ func CommitteeAssignments(
// Each slot in an epoch has a different set of committees. This value is derived from the
// active validator set, which does not change.
numCommitteesPerSlot := SlotCommitteeCount(uint64(len(activeValidatorIndices)))
validatorIndexToCommittee := make(map[uint64]*CommitteeAssignmentContainer)
validatorIndexToCommittee := make(map[uint64]*CommitteeAssignmentContainer, numCommitteesPerSlot*params.BeaconConfig().SlotsPerEpoch)
// Compute all committees for all slots.
for i := uint64(0); i < params.BeaconConfig().SlotsPerEpoch; i++ {

View File

@@ -167,7 +167,7 @@ func (kv *Store) DeleteStates(ctx context.Context, blockRoots [][32]byte) error
ctx, span := trace.StartSpan(ctx, "BeaconDB.DeleteStates")
defer span.End()
rootMap := make(map[[32]byte]bool)
rootMap := make(map[[32]byte]bool, len(blockRoots))
for _, blockRoot := range blockRoots {
rootMap[blockRoot] = true
}

View File

@@ -18,11 +18,11 @@ import (
func lookupValuesForIndices(ctx context.Context, indicesByBucket map[string][]byte, tx *bolt.Tx) [][][]byte {
ctx, span := trace.StartSpan(ctx, "BeaconDB.lookupValuesForIndices")
defer span.End()
values := make([][][]byte, 0)
values := make([][][]byte, 0, len(indicesByBucket))
for k, v := range indicesByBucket {
bkt := tx.Bucket([]byte(k))
roots := bkt.Get(v)
splitRoots := make([][]byte, 0)
splitRoots := make([][]byte, 0, len(roots)/32)
for i := 0; i < len(roots); i += 32 {
splitRoots = append(splitRoots, roots[i:i+32])
}

View File

@@ -15,8 +15,8 @@ import (
// It tracks the unaggregated attestations that weren't able to aggregate to prevent
// the deletion of unaggregated attestations in the pool.
func (p *AttCaches) AggregateUnaggregatedAttestations() error {
attsByDataRoot := make(map[[32]byte][]*ethpb.Attestation)
unaggregatedAtts := p.UnaggregatedAttestations()
attsByDataRoot := make(map[[32]byte][]*ethpb.Attestation, len(unaggregatedAtts))
for _, att := range unaggregatedAtts {
attDataRoot, err := stateutil.AttestationDataRoot(att.Data)
if err != nil {
@@ -111,10 +111,11 @@ func (p *AttCaches) SaveAggregatedAttestations(atts []*ethpb.Attestation) error
// AggregatedAttestations returns the aggregated attestations in cache.
func (p *AttCaches) AggregatedAttestations() []*ethpb.Attestation {
atts := make([]*ethpb.Attestation, 0)
p.aggregatedAttLock.RLock()
defer p.aggregatedAttLock.RUnlock()
atts := make([]*ethpb.Attestation, 0)
for _, a := range p.aggregatedAtt {
atts = append(atts, a...)
}

View File

@@ -36,10 +36,10 @@ func (p *AttCaches) SaveForkchoiceAttestations(atts []*ethpb.Attestation) error
// ForkchoiceAttestations returns the forkchoice attestations in cache.
func (p *AttCaches) ForkchoiceAttestations() []*ethpb.Attestation {
atts := make([]*ethpb.Attestation, 0)
p.forkchoiceAttLock.RLock()
defer p.forkchoiceAttLock.RUnlock()
atts := make([]*ethpb.Attestation, 0, len(p.forkchoiceAtt))
for _, att := range p.forkchoiceAtt {
atts = append(atts, stateTrie.CopyAttestation(att) /* Copied */)
}

View File

@@ -41,10 +41,10 @@ func (p *AttCaches) SaveUnaggregatedAttestations(atts []*ethpb.Attestation) erro
// UnaggregatedAttestations returns all the unaggregated attestations in cache.
func (p *AttCaches) UnaggregatedAttestations() []*ethpb.Attestation {
atts := make([]*ethpb.Attestation, 0)
p.unAggregateAttLock.RLock()
defer p.unAggregateAttLock.RUnlock()
atts := make([]*ethpb.Attestation, 0, len(p.unAggregatedAtt))
for _, att := range p.unAggregatedAtt {
atts = append(atts, stateTrie.CopyAttestation(att) /* Copied */)
}

View File

@@ -44,14 +44,14 @@ func (s *Service) batchForkChoiceAtts(ctx context.Context) error {
_, span := trace.StartSpan(ctx, "Operations.attestations.batchForkChoiceAtts")
defer span.End()
attsByDataRoot := make(map[[32]byte][]*ethpb.Attestation)
if err := s.pool.AggregateUnaggregatedAttestations(); err != nil {
return err
}
atts := append(s.pool.AggregatedAttestations(), s.pool.BlockAttestations()...)
atts = append(atts, s.pool.ForkchoiceAttestations()...)
attsByDataRoot := make(map[[32]byte][]*ethpb.Attestation, len(atts))
// Consolidate attestations by aggregating them by similar data root.
for _, att := range atts {
seen, err := s.seen(att)

View File

@@ -16,6 +16,7 @@ go_library(
"//beacon-chain/core/blocks:go_default_library",
"//beacon-chain/core/helpers:go_default_library",
"//beacon-chain/state:go_default_library",
"//shared/mathutil:go_default_library",
"//shared/params:go_default_library",
"//shared/sliceutil:go_default_library",
"@com_github_pkg_errors//:go_default_library",

View File

@@ -10,6 +10,7 @@ import (
"github.com/prysmaticlabs/prysm/beacon-chain/core/blocks"
"github.com/prysmaticlabs/prysm/beacon-chain/core/helpers"
beaconstate "github.com/prysmaticlabs/prysm/beacon-chain/state"
"github.com/prysmaticlabs/prysm/shared/mathutil"
"github.com/prysmaticlabs/prysm/shared/params"
"github.com/prysmaticlabs/prysm/shared/sliceutil"
"go.opencensus.io/trace"
@@ -36,7 +37,9 @@ func (p *Pool) PendingAttesterSlashings(ctx context.Context, state *beaconstate.
numPendingAttesterSlashings.Set(float64(len(p.pendingAttesterSlashing)))
included := make(map[uint64]bool)
pending := make([]*ethpb.AttesterSlashing, 0, params.BeaconConfig().MaxAttesterSlashings)
// Allocate pending slice with a capacity of min(len(p.pendingAttesterSlashing), maxAttesterSlashings)
// since the array cannot exceed the max and is typically less than the max value.
pending := make([]*ethpb.AttesterSlashing, 0, mathutil.Min(uint64(len(p.pendingAttesterSlashing)), params.BeaconConfig().MaxAttesterSlashings))
for i := 0; i < len(p.pendingAttesterSlashing); i++ {
slashing := p.pendingAttesterSlashing[i]
if uint64(len(pending)) >= params.BeaconConfig().MaxAttesterSlashings {
@@ -75,7 +78,9 @@ func (p *Pool) PendingProposerSlashings(ctx context.Context, state *beaconstate.
// Update prom metric.
numPendingProposerSlashings.Set(float64(len(p.pendingProposerSlashing)))
pending := make([]*ethpb.ProposerSlashing, 0, params.BeaconConfig().MaxProposerSlashings)
// Allocate pending slice with a capacity of min(len(p.pendingProposerSlashing), maxProposerSlashings)
// since the array cannot exceed the max and is typically less than the max value.
pending := make([]*ethpb.ProposerSlashing, 0, mathutil.Min(uint64(len(p.pendingProposerSlashing)), params.BeaconConfig().MaxProposerSlashings))
for i := 0; i < len(p.pendingProposerSlashing); i++ {
slashing := p.pendingProposerSlashing[i]
if uint64(len(pending)) >= params.BeaconConfig().MaxProposerSlashings {

View File

@@ -12,6 +12,7 @@ go_library(
deps = [
"//beacon-chain/core/helpers:go_default_library",
"//beacon-chain/state:go_default_library",
"//shared/mathutil:go_default_library",
"//shared/params:go_default_library",
"@com_github_prysmaticlabs_ethereumapis//eth/v1alpha1:go_default_library",
"@io_opencensus_go//trace:go_default_library",

View File

@@ -8,6 +8,7 @@ import (
ethpb "github.com/prysmaticlabs/ethereumapis/eth/v1alpha1"
"github.com/prysmaticlabs/prysm/beacon-chain/core/helpers"
beaconstate "github.com/prysmaticlabs/prysm/beacon-chain/state"
"github.com/prysmaticlabs/prysm/shared/mathutil"
"github.com/prysmaticlabs/prysm/shared/params"
"go.opencensus.io/trace"
)
@@ -34,7 +35,10 @@ func NewPool() *Pool {
func (p *Pool) PendingExits(state *beaconstate.BeaconState, slot uint64) []*ethpb.SignedVoluntaryExit {
p.lock.RLock()
defer p.lock.RUnlock()
pending := make([]*ethpb.SignedVoluntaryExit, 0)
// Allocate pending slice with a capacity of min(len(p.pending), maxVoluntaryExits) since the
// array cannot exceed the max and is typically less than the max value.
pending := make([]*ethpb.SignedVoluntaryExit, 0, mathutil.Min(uint64(len(p.pending)), params.BeaconConfig().MaxVoluntaryExits))
for _, e := range p.pending {
if e.Exit.Epoch > helpers.SlotToEpoch(slot) {
continue

View File

@@ -20,7 +20,7 @@ var GossipTopicMappings = map[string]proto.Message{
// GossipTypeMapping is the inverse of GossipTopicMappings so that an arbitrary protobuf message
// can be mapped to a protocol ID string.
var GossipTypeMapping = make(map[reflect.Type]string)
var GossipTypeMapping = make(map[reflect.Type]string, len(GossipTopicMappings))
func init() {
for k, v := range GossipTopicMappings {

View File

@@ -429,7 +429,7 @@ func (p *Status) Decay() {
func (p *Status) BestFinalized(maxPeers int, ourFinalizedEpoch uint64) (uint64, []peer.ID) {
connected := p.Connected()
finalizedEpochVotes := make(map[uint64]uint64)
pidEpoch := make(map[peer.ID]uint64)
pidEpoch := make(map[peer.ID]uint64, len(connected))
potentialPIDs := make([]peer.ID, 0, len(connected))
for _, pid := range connected {
peerChainState, err := p.ChainState(pid)

View File

@@ -48,7 +48,7 @@ func (m *POWChain) DepositTrie() *trieutil.SparseMerkleTrie {
// BlockExists --
func (m *POWChain) BlockExists(_ context.Context, hash common.Hash) (bool, *big.Int, error) {
// Reverse the map of heights by hash.
heightsByHash := make(map[[32]byte]int)
heightsByHash := make(map[[32]byte]int, len(m.HashesByHeight))
for k, v := range m.HashesByHeight {
h := bytesutil.ToBytes32(v)
heightsByHash[h] = k

View File

@@ -94,9 +94,7 @@ func (bs *Server) ListValidatorAssignments(
}
// Initialize all committee related data.
committeeAssignments := map[uint64]*helpers.CommitteeAssignmentContainer{}
proposerIndexToSlots := make(map[uint64][]uint64)
committeeAssignments, proposerIndexToSlots, err = helpers.CommitteeAssignments(requestedState, requestedEpoch)
committeeAssignments, proposerIndexToSlots, err := helpers.CommitteeAssignments(requestedState, requestedEpoch)
if err != nil {
return nil, status.Errorf(codes.Internal, "Could not compute committee assignments: %v", err)
}

View File

@@ -38,7 +38,7 @@ func (s sortableAttestations) Less(i, j int) bool {
}
func mapAttestationsByTargetRoot(atts []*ethpb.Attestation) map[[32]byte][]*ethpb.Attestation {
attsMap := make(map[[32]byte][]*ethpb.Attestation)
attsMap := make(map[[32]byte][]*ethpb.Attestation, len(atts))
if len(atts) == 0 {
return attsMap
}

View File

@@ -131,7 +131,7 @@ func computeCommittees(
activeIndices []uint64,
attesterSeed [32]byte,
) (map[uint64]*ethpb.BeaconCommittees_CommitteesList, error) {
committeesListsBySlot := make(map[uint64]*ethpb.BeaconCommittees_CommitteesList)
committeesListsBySlot := make(map[uint64]*ethpb.BeaconCommittees_CommitteesList, params.BeaconConfig().SlotsPerEpoch)
for slot := startSlot; slot < startSlot+params.BeaconConfig().SlotsPerEpoch; slot++ {
var countAtSlot = uint64(len(activeIndices)) / params.BeaconConfig().SlotsPerEpoch / params.BeaconConfig().TargetCommitteeSize
if countAtSlot > params.BeaconConfig().MaxCommitteesPerSlot {

View File

@@ -28,7 +28,7 @@ func (ds *Server) GetProtoArrayForkChoice(ctx context.Context, _ *ptypes.Empty)
}
}
indices := make(map[string]uint64)
indices := make(map[string]uint64, len(store.NodeIndices))
for k, v := range store.NodeIndices {
indices[hex.EncodeToString(k[:])] = v
}

View File

@@ -164,8 +164,12 @@ func (ns *Server) GetPeer(ctx context.Context, peerReq *ethpb.PeerRequest) (*eth
// ListPeers lists the peers connected to this node.
func (ns *Server) ListPeers(ctx context.Context, _ *ptypes.Empty) (*ethpb.Peers, error) {
res := make([]*ethpb.Peer, 0)
for _, pid := range ns.PeersFetcher.Peers().Connected() {
peers := ns.PeersFetcher.Peers().Connected()
res := make([]*ethpb.Peer, 0, len(peers))
for _, pid := range peers {
if ctx.Err() != nil {
return nil, ctx.Err()
}
multiaddr, err := ns.PeersFetcher.Peers().Address(pid)
if err != nil {
continue

View File

@@ -30,7 +30,7 @@ func (s sortedObj) Len() int {
// removes duplicates from provided blocks and roots.
func (s *Service) dedupBlocksAndRoots(blks []*ethpb.SignedBeaconBlock, roots [][32]byte) ([]*ethpb.SignedBeaconBlock, [][32]byte) {
// Remove duplicate blocks received
rootMap := make(map[[32]byte]bool)
rootMap := make(map[[32]byte]bool, len(blks))
newBlks := make([]*ethpb.SignedBeaconBlock, 0, len(blks))
newRoots := make([][32]byte, 0, len(roots))
for i, r := range roots {

View File

@@ -154,7 +154,7 @@ func IsValidAttestationIndices(ctx context.Context, indexedAttestation *ethpb.In
if uint64(len(indices)) > params.BeaconConfig().MaxValidatorsPerCommittee {
return fmt.Errorf("validator indices count exceeds MAX_VALIDATORS_PER_COMMITTEE, %d > %d", len(indices), params.BeaconConfig().MaxValidatorsPerCommittee)
}
set := make(map[uint64]bool)
set := make(map[uint64]bool, len(indices))
setIndices := make([]uint64, 0, len(indices))
for _, i := range indices {
if ok := set[i]; ok {

View File

@@ -56,7 +56,7 @@ func Bytes8(x uint64) []byte {
return bytes
}
// Bytes32 returns integer x to bytes in little-endian format, x.to_bytes(8, 'little').
// Bytes32 returns integer x to bytes in little-endian format, x.to_bytes(32, 'little').
func Bytes32(x uint64) []byte {
bytes := make([]byte, 32)
binary.LittleEndian.PutUint64(bytes, x)

View File

@@ -62,7 +62,7 @@ func (s *ServiceRegistry) StopAll() {
// Statuses returns a map of Service type -> error. The map will be populated
// with the results of each service.Status() method call.
func (s *ServiceRegistry) Statuses() map[reflect.Type]error {
m := make(map[reflect.Type]error)
m := make(map[reflect.Type]error, len(s.serviceTypes))
for _, kind := range s.serviceTypes {
m[kind] = s.services[kind].Status()
}

View File

@@ -12,7 +12,7 @@ func SubsetUint64(a []uint64, b []uint64) bool {
return false
}
set := make(map[uint64]uint64)
set := make(map[uint64]uint64, len(b))
for _, v := range b {
set[v]++
}